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/11 14:18:04 UTC

[1/4] incubator-ignite git commit: ignite-998

Repository: incubator-ignite
Updated Branches:
  refs/heads/ignite-sprint-5 b52e47ae5 -> 4375529fa


ignite-998


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

Branch: refs/heads/ignite-sprint-5
Commit: c47438e8d2ea734218bcd9c14945ce43456269ce
Parents: af120a7
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 10 14:24:24 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 10 15:23:37 2015 +0300

----------------------------------------------------------------------
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  38 +------
 .../tcp/TcpClientDiscoverySpiSelfTest.java      | 102 ++++++++++++++++++-
 2 files changed, 103 insertions(+), 37 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c47438e8/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
----------------------------------------------------------------------
diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
index 5aceaae..44374db 100644
--- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
+++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/ServerImpl.java
@@ -1752,6 +1752,9 @@ class ServerImpl extends TcpDiscoveryImpl {
         @Nullable Collection<TcpDiscoveryAbstractMessage> messages(IgniteUuid lastMsgId) {
             assert lastMsgId != null;
 
+            if (msgs.isEmpty())
+                return Collections.emptyList();
+
             Collection<TcpDiscoveryAbstractMessage> cp = new ArrayList<>(msgs.size());
 
             boolean skip = true;
@@ -1769,30 +1772,6 @@ class ServerImpl extends TcpDiscoveryImpl {
         }
 
         /**
-         * Resets pending messages.
-         *
-         * @param msgs Message.
-         * @param discardId Discarded message ID.
-         */
-        void reset(@Nullable Collection<TcpDiscoveryAbstractMessage> msgs, @Nullable IgniteUuid discardId) {
-            this.msgs.clear();
-
-            if (msgs != null)
-                this.msgs.addAll(msgs);
-
-            this.discardId = discardId;
-        }
-
-        /**
-         * Clears pending messages.
-         */
-        void clear() {
-            msgs.clear();
-
-            discardId = null;
-        }
-
-        /**
          * Discards message with provided ID and all before it.
          *
          * @param id Discarded message ID.
@@ -2921,8 +2900,7 @@ class ServerImpl extends TcpDiscoveryImpl {
                             topHist.clear();
                             topHist.putAll(msg.topologyHistory());
 
-                            // Restore pending messages.
-                            pendingMsgs.reset(msg.messages(), msg.discardedMessageId());
+                            pendingMsgs.discard(msg.discardedMessageId());
 
                             // Clear data to minimize message size.
                             msg.messages(null, null);
@@ -3180,10 +3158,6 @@ class ServerImpl extends TcpDiscoveryImpl {
                 if (log.isDebugEnabled())
                     log.debug("Removed node from topology: " + leftNode);
 
-                // Clear pending messages map.
-                if (!ring.hasRemoteNodes())
-                    pendingMsgs.clear();
-
                 long topVer;
 
                 if (locNodeCoord) {
@@ -3347,10 +3321,6 @@ class ServerImpl extends TcpDiscoveryImpl {
 
                 assert node != null;
 
-                // Clear pending messages map.
-                if (!ring.hasRemoteNodes())
-                    pendingMsgs.clear();
-
                 long topVer;
 
                 if (locNodeCoord) {

http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/c47438e8/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
index 7333020..ece898d 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
@@ -103,12 +103,17 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     /** */
     private boolean longSockTimeouts;
 
+    /** */
+    private int maxMissedClientHbs = TcpDiscoverySpi.DFLT_MAX_MISSED_CLIENT_HEARTBEATS;
+
     /** {@inheritDoc} */
     @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception {
         IgniteConfiguration cfg = super.getConfiguration(gridName);
 
         TcpDiscoverySpi disco = new TestTcpDiscoverySpi();
 
+        disco.setMaxMissedClientHeartbeats(maxMissedClientHbs);
+
         if (gridName.startsWith("server"))
             disco.setIpFinder(IP_FINDER);
         else if (gridName.startsWith("client")) {
@@ -494,6 +499,96 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     /**
      * @throws Exception If failed.
      */
+    public void testClientReconnectTopologyChange1() throws Exception {
+        maxMissedClientHbs = 100;
+
+        clientsPerSrv = 1;
+
+        startServerNodes(2);
+        startClientNodes(1);
+
+        checkNodes(2, 1);
+
+        srvLeftLatch = new CountDownLatch(3);
+        srvFailedLatch = new CountDownLatch(1);
+
+        attachListeners(2, 0);
+
+        Ignite ignite = G.ignite("client-0");
+
+        TestTcpDiscoverySpi spi = ((TestTcpDiscoverySpi)ignite.configuration().getDiscoverySpi());
+
+        spi.pauseAll();
+
+        try {
+            spi.brakeConnection();
+
+            Ignite g = startGrid("server-" + srvIdx.getAndIncrement());
+
+            g.close();
+
+            spi.resumeAll();
+
+            assertFalse(srvFailedLatch.await(2000, TimeUnit.MILLISECONDS));
+
+            assertEquals(1L, srvLeftLatch.getCount());
+
+            checkNodes(2, 1);
+        }
+        finally {
+            spi.resumeAll();
+        }
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
+    public void testClientReconnectTopologyChange2() throws Exception {
+        fail("https://issues.apache.org/jira/browse/IGNITE-998");
+
+        maxMissedClientHbs = 100;
+
+        clientsPerSrv = 1;
+
+        startServerNodes(1);
+        startClientNodes(1);
+
+        checkNodes(1, 1);
+
+        srvLeftLatch = new CountDownLatch(2);
+        srvFailedLatch = new CountDownLatch(1);
+
+        attachListeners(1, 0);
+
+        Ignite ignite = G.ignite("client-0");
+
+        TestTcpDiscoverySpi spi = ((TestTcpDiscoverySpi)ignite.configuration().getDiscoverySpi());
+
+        spi.pauseAll();
+
+        try {
+            spi.brakeConnection();
+
+            Ignite g = startGrid("server-" + srvIdx.getAndIncrement());
+
+            g.close();
+
+            spi.resumeAll();
+
+            assertFalse(srvFailedLatch.await(2000, TimeUnit.MILLISECONDS));
+
+            assertEquals(1L, srvLeftLatch.getCount());
+
+            checkNodes(1, 1);
+        }
+        finally {
+            spi.resumeAll();
+        }
+    }
+
+    /**
+     * @throws Exception If failed.
+     */
     public void testGetMissedMessagesOnReconnect() throws Exception {
         clientsPerSrv = 1;
 
@@ -731,8 +826,6 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     }
 
     /**
-     * TODO: IGNITE-587.
-     *
      * @throws Exception If failed.
      */
     public void testDataExchangeFromClient() throws Exception {
@@ -740,6 +833,7 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     }
 
     /**
+     * @param masterName Node name
      * @throws Exception If failed.
      */
     private void testDataExchange(String masterName) throws Exception {
@@ -890,7 +984,8 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     }
 
     /**
-     * @param clientIdx Index.
+     * @param clientIdx Client index.
+     * @param srvIdx Server index.
      * @throws Exception In case of error.
      */
     private void setClientRouter(int clientIdx, int srvIdx) throws Exception {
@@ -948,6 +1043,7 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
     /**
      * @param srvCnt Number of server nodes.
      * @param clientCnt Number of client nodes.
+     * @throws Exception If failed.
      */
     private void attachListeners(int srvCnt, int clientCnt) throws Exception {
         if (srvJoinedLatch != null) {


[4/4] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-998' into ignite-sprint-5

Posted by sb...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-998' into ignite-sprint-5


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

Branch: refs/heads/ignite-sprint-5
Commit: 4375529fa929e650f7b68d750318d67a8609ee10
Parents: b52e47a 4ab301d
Author: sboikov <sb...@gridgain.com>
Authored: Thu Jun 11 16:06:03 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Thu Jun 11 16:06:03 2015 +0300

----------------------------------------------------------------------
 .../ignite/spi/discovery/tcp/ServerImpl.java    |  38 +------
 .../tcp/TcpClientDiscoverySpiSelfTest.java      | 114 ++++++++++++++++++-
 2 files changed, 115 insertions(+), 37 deletions(-)
----------------------------------------------------------------------



[3/4] incubator-ignite git commit: Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-998

Posted by sb...@apache.org.
Merge remote-tracking branch 'remotes/origin/ignite-sprint-5' into ignite-998


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

Branch: refs/heads/ignite-sprint-5
Commit: 4ab301d2f0505ee3a67978392827dcccea60a58b
Parents: 46e6c48 89a4f7c
Author: sboikov <sb...@gridgain.com>
Authored: Thu Jun 11 10:19:02 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Thu Jun 11 10:19:02 2015 +0300

----------------------------------------------------------------------
 DEVNOTES.txt                                    |   6 +
 assembly/dependencies-fabric.xml                |   1 +
 examples/pom.xml                                |  34 ++
 .../client/memcache/MemcacheRestExample.java    |  32 +-
 modules/core/pom.xml                            |   1 -
 .../java/org/apache/ignite/IgniteCache.java     |  25 +-
 .../apache/ignite/IgniteSystemProperties.java   |   3 +
 .../apache/ignite/cache/query/ScanQuery.java    |  45 +-
 .../configuration/CacheConfiguration.java       |   1 -
 .../affinity/GridAffinityAssignmentCache.java   |   8 +-
 .../processors/cache/GridCacheAdapter.java      |  15 +-
 .../GridCachePartitionExchangeManager.java      |   2 +-
 .../processors/cache/GridCacheProcessor.java    |  30 +-
 .../processors/cache/GridCacheSwapManager.java  |  55 ++-
 .../processors/cache/IgniteCacheProxy.java      |  11 +-
 .../processors/cache/IgniteInternalCache.java   |  27 +-
 .../processors/cache/QueryCursorImpl.java       |  23 +-
 .../distributed/dht/GridDhtLocalPartition.java  |   7 +
 .../processors/cache/query/CacheQuery.java      |   2 +-
 .../query/GridCacheDistributedQueryManager.java |   3 +
 .../cache/query/GridCacheQueryAdapter.java      | 147 ++++++-
 .../cache/query/GridCacheQueryManager.java      | 209 ++++++----
 .../cache/query/GridCacheQueryRequest.java      |  47 ++-
 .../processors/cache/query/QueryCursorEx.java   |   8 +
 .../datastructures/GridCacheSetImpl.java        |   4 +-
 .../processors/query/GridQueryIndexing.java     |   4 +-
 .../processors/query/GridQueryProcessor.java    |  18 +-
 .../service/GridServiceProcessor.java           |   2 +-
 .../ignite/internal/util/GridJavaProcess.java   |   2 +-
 .../ignite/internal/util/IgniteUtils.java       |   4 +-
 .../shmem/IpcSharedMemoryClientEndpoint.java    |   2 +-
 .../ipc/shmem/IpcSharedMemoryNativeLoader.java  | 151 ++++++-
 .../shmem/IpcSharedMemoryServerEndpoint.java    |   2 +-
 .../util/nio/GridShmemCommunicationClient.java  | 146 +++++++
 .../communication/tcp/TcpCommunicationSpi.java  | 415 ++++++++++++++++++-
 .../tcp/TcpCommunicationSpiMBean.java           |   8 +
 .../spi/discovery/tcp/TcpDiscoverySpi.java      |   2 +-
 .../RoundRobinGlobalLoadBalancer.java           |   2 +-
 .../cache/GridCacheAbstractFullApiSelfTest.java |  15 +
 .../cache/IgniteDynamicCacheStartSelfTest.java  |  19 +
 .../distributed/IgniteCacheManyClientsTest.java | 169 ++++++++
 .../IgniteCacheMessageRecoveryAbstractTest.java |   1 +
 ...achePartitionedPreloadLifecycleSelfTest.java |   2 +-
 ...CacheReplicatedPreloadLifecycleSelfTest.java |   6 +-
 .../GridCacheSwapScanQueryAbstractSelfTest.java | 112 +++--
 .../ipc/shmem/IgfsSharedMemoryTestServer.java   |   2 +
 .../IpcSharedMemoryCrashDetectionSelfTest.java  |   2 +-
 .../ipc/shmem/IpcSharedMemorySpaceSelfTest.java |   2 +-
 .../ipc/shmem/IpcSharedMemoryUtilsSelfTest.java |   2 +-
 .../LoadWithCorruptedLibFileTestRunner.java     |   2 +-
 .../IpcSharedMemoryBenchmarkReader.java         |   2 +-
 .../IpcSharedMemoryBenchmarkWriter.java         |   2 +-
 .../communication/GridIoManagerBenchmark0.java  |   1 +
 .../spi/GridTcpSpiForwardingSelfTest.java       |   1 +
 .../GridTcpCommunicationSpiAbstractTest.java    |  13 +
 ...mmunicationSpiConcurrentConnectSelfTest.java |   4 +-
 ...cpCommunicationSpiMultithreadedSelfTest.java |  21 +-
 ...pCommunicationSpiMultithreadedShmemTest.java |  28 ++
 ...dTcpCommunicationSpiRecoveryAckSelfTest.java |   1 +
 ...GridTcpCommunicationSpiRecoverySelfTest.java |   1 +
 .../GridTcpCommunicationSpiShmemSelfTest.java   |  38 ++
 .../tcp/GridTcpCommunicationSpiTcpSelfTest.java |   7 +
 .../ignite/testframework/GridTestUtils.java     |   2 +-
 .../testsuites/IgniteCacheTestSuite4.java       |   2 +
 .../IgniteSpiCommunicationSelfTestSuite.java    |   2 +
 modules/hadoop/pom.xml                          |   1 +
 .../HadoopIgfs20FileSystemAbstractSelfTest.java |  13 +
 ...oopSecondaryFileSystemConfigurationTest.java |  14 +
 ...IgniteHadoopFileSystemHandshakeSelfTest.java |   7 +
 .../IgniteHadoopFileSystemIpcCacheSelfTest.java |   7 +
 .../hadoop/HadoopAbstractSelfTest.java          |   7 +
 .../processors/query/h2/IgniteH2Indexing.java   |  44 +-
 .../h2/twostep/GridReduceQueryExecutor.java     |   8 +-
 ...CacheScanPartitionQueryFallbackSelfTest.java | 408 ++++++++++++++++++
 .../cache/GridCacheCrossCacheQuerySelfTest.java |  12 +-
 .../cache/IgniteCacheAbstractQuerySelfTest.java |  77 +++-
 .../IgniteCacheQuerySelfTestSuite.java          |   2 +
 modules/scalar-2.10/README.txt                  |   4 +
 modules/scalar-2.10/licenses/apache-2.0.txt     | 202 +++++++++
 .../scalar-2.10/licenses/scala-bsd-license.txt  |  18 +
 modules/scalar-2.10/pom.xml                     | 197 +++++++++
 modules/spark-2.10/README.txt                   |   4 +
 modules/spark-2.10/licenses/apache-2.0.txt      | 202 +++++++++
 .../spark-2.10/licenses/scala-bsd-license.txt   |  18 +
 modules/spark-2.10/pom.xml                      | 120 ++++++
 modules/spark/README.txt                        |   8 +
 modules/spark/licenses/apache-2.0.txt           | 202 +++++++++
 modules/spark/licenses/scala-bsd-license.txt    |  18 +
 modules/spark/pom.xml                           | 114 +++++
 .../org/apache/ignite/spark/IgniteContext.scala | 119 ++++++
 .../org/apache/ignite/spark/IgniteRDD.scala     | 244 +++++++++++
 .../apache/ignite/spark/JavaIgniteContext.scala |  63 +++
 .../org/apache/ignite/spark/JavaIgniteRDD.scala |  99 +++++
 .../ignite/spark/impl/IgniteAbstractRDD.scala   |  39 ++
 .../ignite/spark/impl/IgnitePartition.scala     |  24 ++
 .../ignite/spark/impl/IgniteQueryIterator.scala |  27 ++
 .../apache/ignite/spark/impl/IgniteSqlRDD.scala |  41 ++
 .../spark/impl/JavaIgniteAbstractRDD.scala      |  34 ++
 .../ignite/spark/JavaIgniteRDDSelfTest.java     | 298 +++++++++++++
 .../scala/org/apache/ignite/spark/Entity.scala  |  28 ++
 .../org/apache/ignite/spark/IgniteRddSpec.scala | 231 +++++++++++
 modules/visor-console-2.10/README.txt           |   4 +
 modules/visor-console-2.10/pom.xml              | 174 ++++++++
 parent/pom.xml                                  |   4 +
 pom.xml                                         |  20 +-
 105 files changed, 4827 insertions(+), 292 deletions(-)
----------------------------------------------------------------------



[2/4] incubator-ignite git commit: ignite-998

Posted by sb...@apache.org.
ignite-998


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

Branch: refs/heads/ignite-sprint-5
Commit: 46e6c48ee59a3837c06564d2431caabc55cf9a63
Parents: c47438e
Author: sboikov <sb...@gridgain.com>
Authored: Wed Jun 10 15:33:01 2015 +0300
Committer: sboikov <sb...@gridgain.com>
Committed: Wed Jun 10 15:33:01 2015 +0300

----------------------------------------------------------------------
 .../discovery/tcp/TcpClientDiscoverySpiSelfTest.java    | 12 ++++++++++++
 1 file changed, 12 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46e6c48e/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
----------------------------------------------------------------------
diff --git a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
index ece898d..55a14e4 100644
--- a/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpiSelfTest.java
@@ -1136,6 +1136,8 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
      * @param clientCnt Number of client nodes.
      */
     private void checkNodes(int srvCnt, int clientCnt) {
+        long topVer = -1;
+
         for (int i = 0; i < srvCnt; i++) {
             Ignite g = G.ignite("server-" + i);
 
@@ -1144,6 +1146,11 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
             assertFalse(g.cluster().localNode().isClient());
 
             checkRemoteNodes(g, srvCnt + clientCnt - 1);
+
+            if (topVer < 0)
+                topVer = g.cluster().topologyVersion();
+            else
+                assertEquals(topVer, g.cluster().topologyVersion());
         }
 
         for (int i = 0; i < clientCnt; i++) {
@@ -1156,6 +1163,11 @@ public class TcpClientDiscoverySpiSelfTest extends GridCommonAbstractTest {
             assertTrue(g.cluster().localNode().isClient());
 
             checkRemoteNodes(g, srvCnt + clientCnt - 1);
+
+            if (topVer < 0)
+                topVer = g.cluster().topologyVersion();
+            else
+                assertEquals(topVer, g.cluster().topologyVersion());
         }
     }