You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ignite.apache.org by gv...@apache.org on 2019/02/04 09:13:15 UTC

[ignite] branch master updated: IGNITE-11148: PartitionCountersNeighborcastFuture blocks partition map exchange. This closes #5998.

This is an automated email from the ASF dual-hosted git repository.

gvvinblade pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/ignite.git


The following commit(s) were added to refs/heads/master by this push:
     new 63b9e16  IGNITE-11148: PartitionCountersNeighborcastFuture blocks partition map exchange. This closes #5998.
63b9e16 is described below

commit 63b9e1653d67e72d06365a7054cc0398c2d28cda
Author: ipavlukhin <vo...@gmail.com>
AuthorDate: Mon Feb 4 12:13:06 2019 +0300

    IGNITE-11148: PartitionCountersNeighborcastFuture blocks partition map exchange. This closes #5998.
---
 .../PartitionCountersNeighborcastFuture.java       | 29 +++++++++++++++++++---
 ...ContinuousQueryAsyncFailoverMvccTxSelfTest.java |  7 ------
 ...CacheContinuousQueryFailoverMvccTxSelfTest.java |  7 ------
 3 files changed, 25 insertions(+), 18 deletions(-)

diff --git a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/PartitionCountersNeighborcastFuture.java b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/PartitionCountersNeighborcastFuture.java
index f7fe9cb..9e65b56 100644
--- a/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/PartitionCountersNeighborcastFuture.java
+++ b/modules/core/src/main/java/org/apache/ignite/internal/processors/cache/transactions/PartitionCountersNeighborcastFuture.java
@@ -31,8 +31,10 @@ import org.apache.ignite.internal.processors.cache.GridCacheSharedContext;
 import org.apache.ignite.internal.processors.cache.distributed.dht.PartitionUpdateCountersMessage;
 import org.apache.ignite.internal.processors.cache.mvcc.msg.PartitionCountersNeighborcastRequest;
 import org.apache.ignite.internal.util.future.GridFutureAdapter;
+import org.apache.ignite.internal.util.tostring.GridToStringExclude;
 import org.apache.ignite.internal.util.typedef.F;
 import org.apache.ignite.internal.util.typedef.internal.CU;
+import org.apache.ignite.internal.util.typedef.internal.S;
 import org.apache.ignite.lang.IgniteUuid;
 import org.jetbrains.annotations.Nullable;
 
@@ -45,6 +47,7 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
     /** */
     private final IgniteUuid futId = IgniteUuid.randomUuid();
     /** */
+    @GridToStringExclude
     private boolean trackable = true;
     /** */
     private final GridCacheSharedContext<?, ?> cctx;
@@ -88,9 +91,10 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
             MiniFuture miniFut = new MiniFuture(peer);
 
             try {
-                cctx.io().send(peer, new PartitionCountersNeighborcastRequest(cntrs, futId), SYSTEM_POOL);
-
+                // we must add mini future before sending a message, otherwise mini future must miss completion
                 add(miniFut);
+
+                cctx.io().send(peer, new PartitionCountersNeighborcastRequest(cntrs, futId), SYSTEM_POOL);
             }
             catch (IgniteCheckedException e) {
                 if (!(e instanceof ClusterTopologyCheckedException))
@@ -140,9 +144,10 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
      * @param nodeId Remote peer node id.
      */
     public void onResult(UUID nodeId) {
-        if (log.isInfoEnabled())
+        if (log.isInfoEnabled()) {
             log.info("Remote peer acked partition counters delivery [futId=" + futId +
                 ", node=" + nodeId + ']');
+        }
 
         completeMini(nodeId);
     }
@@ -167,9 +172,15 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
             if (mini.nodeId.equals(nodeId)) {
                 cctx.kernalContext().closure().runLocalSafe(mini::onDone);
 
-                break;
+                return;
             }
         }
+
+        if (log.isInfoEnabled()) {
+            log.info("Failed to find mini future corresponding to node, can prevent parent future completion [" +
+                "futId=" + futId +
+                ", nodeId=" + nodeId + ']');
+        }
     }
 
     /** */
@@ -196,6 +207,11 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
         trackable = false;
     }
 
+    /** {@inheritDoc} */
+    @Override public String toString() {
+        return S.toString(PartitionCountersNeighborcastFuture.class, this, "innerFuts", futures());
+    }
+
     /**
      * Component of compound parent future. Represents interaction with one of remote peers.
      */
@@ -207,5 +223,10 @@ public class PartitionCountersNeighborcastFuture extends GridCacheCompoundIdenti
         private MiniFuture(UUID nodeId) {
             this.nodeId = nodeId;
         }
+
+        /** {@inheritDoc} */
+        @Override public String toString() {
+            return S.toString(MiniFuture.class, this, "done", isDone());
+        }
     }
 }
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryAsyncFailoverMvccTxSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryAsyncFailoverMvccTxSelfTest.java
index a9ca77c..b3d0b7e 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryAsyncFailoverMvccTxSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryAsyncFailoverMvccTxSelfTest.java
@@ -49,11 +49,4 @@ public class CacheContinuousQueryAsyncFailoverMvccTxSelfTest  extends CacheConti
     @Override public void testBackupQueueEvict() throws Exception {
         // No-op.
     }
-
-    /** {@inheritDoc} */
-    @Ignore("https://issues.apache.org/jira/browse/IGNITE-11148")
-    @Test
-    @Override public void testMultiThreadedFailover() throws Exception {
-        // No-op.
-    }
 }
diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryFailoverMvccTxSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryFailoverMvccTxSelfTest.java
index a54f270..a7c35cf 100644
--- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryFailoverMvccTxSelfTest.java
+++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/query/continuous/CacheContinuousQueryFailoverMvccTxSelfTest.java
@@ -44,11 +44,4 @@ public class CacheContinuousQueryFailoverMvccTxSelfTest extends CacheContinuousQ
     @Override public void testBackupQueueEvict() throws Exception {
         // No-op.
     }
-
-    /** {@inheritDoc} */
-    @Ignore("https://issues.apache.org/jira/browse/IGNITE-11148")
-    @Test
-    @Override public void testMultiThreadedFailover() throws Exception {
-        // No-op.
-    }
 }