You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by sb...@apache.org on 2015/06/10 22:00:06 UTC

[25/50] [abbrv] incubator-ignite git commit: ignite-1002 Inject Ignite before applying predicate

ignite-1002 Inject Ignite before applying predicate


Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/91104a29
Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/91104a29
Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/91104a29

Branch: refs/heads/ignite-471-2
Commit: 91104a29bc32b6d73e7bf4c70eafa9f777b62352
Parents: e3fe8ce
Author: agura <ag...@gridgain.com>
Authored: Mon Jun 8 18:41:43 2015 +0300
Committer: agura <ag...@gridgain.com>
Committed: Tue Jun 9 20:17:33 2015 +0300

----------------------------------------------------------------------
 .../continuous/GridContinuousProcessor.java     |   2 +
 .../util/nio/GridNioDelimitedBuffer.java        |   2 +-
 .../nio/GridNioDelimitedBufferSelfTest.java     | 112 +++++++++++++++++++
 .../util/nio/GridNioDelimitedBufferTest.java    | 112 -------------------
 .../stream/socket/SocketStreamerSelfTest.java   |  29 ++---
 .../ignite/testsuites/IgniteBasicTestSuite.java |   1 +
 .../testsuites/IgniteStreamSelfTestSuite.java   |  39 +++++++
 .../testsuites/IgniteStreamTestSuite.java       |  39 -------
 .../testsuites/IgniteUtilSelfTestSuite.java     |   2 +-
 9 files changed, 168 insertions(+), 170 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/main/java/org/apache/ignite/internal/processors/continuous/GridContinuousProcessor.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/continuous/GridContinuousProcessor.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/continuous/GridContinuousProcessor.java
index 67b32a6..38d970b 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/continuous/GridContinuousProcessor.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/continuous/GridContinuousProcessor.java
@@ -710,6 +710,8 @@ public class GridContinuousProcessor extends GridProcessorAdapter {
             try {
                 IgnitePredicate<ClusterNode> prjPred = data.projectionPredicate();
 
+                ctx.resource().injectGeneric(prjPred);
+
                 if (prjPred == null || prjPred.apply(ctx.discovery().node(ctx.localNodeId()))) {
                     registered = registerHandler(node.id(), routineId, hnd, data.bufferSize(), data.interval(),
                         data.autoUnsubscribe(), false);

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBuffer.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBuffer.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBuffer.java
index 2b764ec..44ab4a5 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBuffer.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBuffer.java
@@ -85,7 +85,7 @@ public class GridNioDelimitedBuffer {
                         idx++;
                     }
                     else {
-                        pos = cnt - idx;
+                        pos = cnt - (i - pos) - 1;
 
                         idx = 0;
                     }

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferSelfTest.java
new file mode 100644
index 0000000..cbf7d89
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferSelfTest.java
@@ -0,0 +1,112 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.internal.util.nio;
+
+import junit.framework.TestCase;
+
+import java.nio.*;
+import java.util.*;
+
+/**
+ * Tests for {@link GridNioDelimitedBuffer}.
+ */
+public class GridNioDelimitedBufferSelfTest extends TestCase {
+    /** */
+    private static final String ASCII = "ASCII";
+
+    /**
+     * Tests simple delimiter (excluded from alphabet)
+     */
+    public void testReadZString() throws Exception {
+        Random rnd = new Random();
+
+        int buffSize = 0;
+
+        byte[] delim = new byte[] {0};
+
+        List<String> strs = new ArrayList<>(50);
+
+        for (int i = 0; i < 50; i++) {
+            int len = rnd.nextInt(128) + 1;
+
+            buffSize += len + delim.length;
+
+            StringBuilder sb = new StringBuilder(len);
+
+            for (int j = 0; j < len; j++)
+                sb.append((char)(rnd.nextInt(26) + 'a'));
+
+
+            strs.add(sb.toString());
+        }
+
+        ByteBuffer buff = ByteBuffer.allocate(buffSize);
+
+        for (String str : strs) {
+            buff.put(str.getBytes(ASCII));
+            buff.put(delim);
+        }
+
+        buff.flip();
+
+        byte[] msg;
+
+        GridNioDelimitedBuffer delimBuff = new GridNioDelimitedBuffer(delim);
+
+        List<String> res = new ArrayList<>(strs.size());
+
+        while ((msg = delimBuff.read(buff)) != null)
+            res.add(new String(msg, ASCII));
+
+        assertEquals(strs, res);
+    }
+
+    /**
+     * Tests compound delimiter (included to alphabet)
+     */
+    public void testDelim() throws Exception {
+        byte[] delim = "aabb".getBytes(ASCII);
+
+        List<String> strs = Arrays.asList("za", "zaa", "zaab", "zab", "zaabaababbbbabaab");
+
+        int buffSize = 0;
+
+        for (String str : strs)
+            buffSize += str.length() + delim.length;
+
+        ByteBuffer buff = ByteBuffer.allocate(buffSize);
+
+        for (String str : strs) {
+            buff.put(str.getBytes(ASCII));
+            buff.put(delim);
+        }
+
+        buff.flip();
+
+        byte[] msg;
+
+        GridNioDelimitedBuffer delimBuff = new GridNioDelimitedBuffer(delim);
+
+        List<String> res = new ArrayList<>(strs.size());
+
+        while ((msg = delimBuff.read(buff)) != null)
+            res.add(new String(msg, ASCII));
+
+        assertEquals(strs, res);
+    }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferTest.java b/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferTest.java
deleted file mode 100644
index a0dd2e5..0000000
--- a/modules/core/src/test/java/org/apache/ignite/internal/util/nio/GridNioDelimitedBufferTest.java
+++ /dev/null
@@ -1,112 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *      http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-package org.apache.ignite.internal.util.nio;
-
-import junit.framework.TestCase;
-
-import java.nio.*;
-import java.util.*;
-
-/**
- * Tests for {@link GridNioDelimitedBuffer}.
- */
-public class GridNioDelimitedBufferTest extends TestCase {
-    /** */
-    private static final String ASCII = "ASCII";
-
-    /**
-     * Tests simple delimiter (excluded from alphabet)
-     */
-    public void testReadZString() throws Exception {
-        Random rnd = new Random();
-
-        int buffSize = 0;
-
-        byte[] delim = new byte[] {0};
-
-        List<String> strs = new ArrayList<>(50);
-
-        for (int i = 0; i < 50; i++) {
-            int len = rnd.nextInt(128) + 1;
-
-            buffSize += len + delim.length;
-
-            StringBuilder sb = new StringBuilder(len);
-
-            for (int j = 0; j < len; j++)
-                sb.append((char)(rnd.nextInt(26) + 'a'));
-
-
-            strs.add(sb.toString());
-        }
-
-        ByteBuffer buff = ByteBuffer.allocate(buffSize);
-
-        for (String str : strs) {
-            buff.put(str.getBytes(ASCII));
-            buff.put(delim);
-        }
-
-        buff.flip();
-
-        byte[] msg;
-
-        GridNioDelimitedBuffer delimBuff = new GridNioDelimitedBuffer(delim);
-
-        List<String> res = new ArrayList<>(strs.size());
-
-        while ((msg = delimBuff.read(buff)) != null)
-            res.add(new String(msg, ASCII));
-
-        assertEquals(strs, res);
-    }
-
-    /**
-     * Tests compound delimiter (included to alphabet)
-     */
-    public void testDelim() throws Exception {
-        byte[] delim = "aabb".getBytes(ASCII);
-
-        List<String> strs = Arrays.asList("za", "zaa", "zaab", "zab", "zaabaababbbbabaab");
-
-        int buffSize = 0;
-
-        for (String str : strs)
-            buffSize += str.length() + delim.length;
-
-        ByteBuffer buff = ByteBuffer.allocate(buffSize);
-
-        for (String str : strs) {
-            buff.put(str.getBytes(ASCII));
-            buff.put(delim);
-        }
-
-        buff.flip();
-
-        byte[] msg;
-
-        GridNioDelimitedBuffer delimBuff = new GridNioDelimitedBuffer(delim);
-
-        List<String> res = new ArrayList<>(strs.size());
-
-        while ((msg = delimBuff.read(buff)) != null)
-            res.add(new String(msg, ASCII));
-
-        assertEquals(strs, res);
-    }
-}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/stream/socket/SocketStreamerSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/stream/socket/SocketStreamerSelfTest.java b/modules/core/src/test/java/org/apache/ignite/stream/socket/SocketStreamerSelfTest.java
index 752e43c..04f9b41 100644
--- a/modules/core/src/test/java/org/apache/ignite/stream/socket/SocketStreamerSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/stream/socket/SocketStreamerSelfTest.java
@@ -58,14 +58,11 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
     /** Port. */
     private static int port;
 
-    /** Ignite. */
-    private static Ignite ignite;
-
     /** {@inheritDoc} */
-    @Override protected IgniteConfiguration getConfiguration() throws Exception {
-        IgniteConfiguration cfg = super.getConfiguration();
+    @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
+        IgniteConfiguration cfg = super.getConfiguration(gridName);
 
-        CacheConfiguration ccfg = cacheConfiguration(cfg, null);
+        CacheConfiguration ccfg = defaultCacheConfiguration();
 
         cfg.setCacheConfiguration(ccfg);
 
@@ -81,8 +78,7 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
 
     /** {@inheritDoc} */
     @Override protected void beforeTestsStarted() throws Exception {
-        ignite = startGrids(GRID_CNT);
-        ignite.<Integer, String>getOrCreateCache(defaultCacheConfiguration());
+        startGrids(GRID_CNT);
 
         try (ServerSocket sock = new ServerSocket(0)) {
             port = sock.getLocalPort();
@@ -94,11 +90,6 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
         stopAllGrids();
     }
 
-    /** {@inheritDoc} */
-    @Override protected void beforeTest() throws Exception {
-        ignite.cache(null).clear();
-    }
-
     /**
      * @throws Exception If failed.
      */
@@ -235,6 +226,12 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
     {
         SocketStreamer<Tuple, Integer, String> sockStmr = null;
 
+        Ignite ignite = grid(0);
+
+        IgniteCache<Integer, String> cache = ignite.cache(null);
+
+        cache.clear();
+
         try (IgniteDataStreamer<Integer, String> stmr = ignite.dataStreamer(null)) {
 
             stmr.allowOverwrite(true);
@@ -242,8 +239,6 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
 
             sockStmr = new SocketStreamer<>();
 
-            IgniteCache<Integer, String> cache = ignite.cache(null);
-
             sockStmr.setIgnite(ignite);
 
             sockStmr.setStreamer(stmr);
@@ -279,10 +274,10 @@ public class SocketStreamerSelfTest extends GridCommonAbstractTest {
 
             latch.await();
 
-            assertEquals(CNT, cache.size(CachePeekMode.PRIMARY));
-
             for (int i = 0; i < CNT; i++)
                 assertEquals(Integer.toString(i), cache.get(i));
+
+            assertEquals(CNT, cache.size(CachePeekMode.PRIMARY));
         }
         finally {
             if (sockStmr != null)

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteBasicTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteBasicTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteBasicTestSuite.java
index e0a1e6e..cc3abb4 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteBasicTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteBasicTestSuite.java
@@ -49,6 +49,7 @@ public class IgniteBasicTestSuite extends TestSuite {
         suite.addTest(IgniteExternalizableSelfTestSuite.suite());
         suite.addTest(IgniteP2PSelfTestSuite.suite());
         suite.addTest(IgniteCacheP2pUnmarshallingErrorTestSuite.suite());
+        suite.addTest(IgniteStreamSelfTestSuite.suite());
 
         suite.addTest(new TestSuite(GridSelfTest.class));
         suite.addTest(new TestSuite(GridProjectionSelfTest.class));

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamSelfTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamSelfTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamSelfTestSuite.java
new file mode 100644
index 0000000..a277fc8
--- /dev/null
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamSelfTestSuite.java
@@ -0,0 +1,39 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ignite.testsuites;
+
+import org.apache.ignite.stream.socket.*;
+
+import junit.framework.*;
+
+/**
+ * Stream test suite.
+ */
+public class IgniteStreamSelfTestSuite extends TestSuite {
+    /**
+     * @return Stream tests suite.
+     * @throws Exception If failed.
+     */
+    public static TestSuite suite() throws Exception {
+        TestSuite suite = new TestSuite("Ignite Stream Test Suite");
+
+        suite.addTest(new TestSuite(SocketStreamerSelfTest.class));
+
+        return suite;
+    }
+}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamTestSuite.java
deleted file mode 100644
index 61be976..0000000
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteStreamTestSuite.java
+++ /dev/null
@@ -1,39 +0,0 @@
-/*
- * Licensed to the Apache Software Foundation (ASF) under one or more
- * contributor license agreements.  See the NOTICE file distributed with
- * this work for additional information regarding copyright ownership.
- * The ASF licenses this file to You under the Apache License, Version 2.0
- * (the "License"); you may not use this file except in compliance with
- * the License.  You may obtain a copy of the License at
- *
- *      http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-
-package org.apache.ignite.testsuites;
-
-import org.apache.ignite.stream.socket.*;
-
-import junit.framework.*;
-
-/**
- * Stream test suite.
- */
-public class IgniteStreamTestSuite extends TestSuite {
-    /**
-     * @return Stream tests suite.
-     * @throws Exception If failed.
-     */
-    public static TestSuite suite() throws Exception {
-        TestSuite suite = new TestSuite("Ignite Stream Test Suite");
-
-        suite.addTest(new TestSuite(SocketStreamerSelfTest.class));
-
-        return suite;
-    }
-}

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/91104a29/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteUtilSelfTestSuite.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteUtilSelfTestSuite.java b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteUtilSelfTestSuite.java
index 32cd038..1c75a7f 100644
--- a/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteUtilSelfTestSuite.java
+++ b/modules/core/src/test/java/org/apache/ignite/testsuites/IgniteUtilSelfTestSuite.java
@@ -67,7 +67,7 @@ public class IgniteUtilSelfTestSuite extends TestSuite {
         suite.addTestSuite(GridNioSelfTest.class);
         suite.addTestSuite(GridNioFilterChainSelfTest.class);
         suite.addTestSuite(GridNioSslSelfTest.class);
-        suite.addTestSuite(GridNioDelimitedBufferTest.class);
+        suite.addTestSuite(GridNioDelimitedBufferSelfTest.class);
 
         return suite;
     }