You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by br...@apache.org on 2021/04/19 13:47:49 UTC
[cassandra] 01/07: Ignore stale ack in shadow
This is an automated email from the ASF dual-hosted git repository.
brandonwilliams pushed a commit to branch CASSANDRA-16588
in repository https://gitbox.apache.org/repos/asf/cassandra.git
commit 64a5fff0da3a389ff3ec0155b8472f919171cb20
Author: Brandon Williams <br...@apache.org>
AuthorDate: Wed Apr 14 13:49:21 2021 -0500
Ignore stale ack in shadow
---
src/java/org/apache/cassandra/gms/Gossiper.java | 15 +++++++++
.../org/apache/cassandra/gms/ShadowRoundTest.java | 37 ++++++++++++++++++++++
2 files changed, 52 insertions(+)
diff --git a/src/java/org/apache/cassandra/gms/Gossiper.java b/src/java/org/apache/cassandra/gms/Gossiper.java
index 69f7fee..5e6adbf 100644
--- a/src/java/org/apache/cassandra/gms/Gossiper.java
+++ b/src/java/org/apache/cassandra/gms/Gossiper.java
@@ -1701,12 +1701,27 @@ public class Gossiper implements IFailureDetectionEventListener, GossiperMBean
return anyNodeOn30;
}
+ public boolean isShadowRoundStateMap(Map<InetAddress, EndpointState> epStateMap)
+ {
+ // it is possible for a previously queued ack to be sent to us when we come back up in shadow
+ EndpointState localState = epStateMap.get(FBUtilities.getBroadcastAddress());
+ if (localState != null && epStateMap.size() == 1) // response only contains our IP
+ {
+ logger.debug("Not exiting shadow round because received bogus ACK {} -> {}", FBUtilities.getBroadcastAddress(), localState);
+ return false;
+ }
+ return true;
+ }
+
protected void maybeFinishShadowRound(InetAddress respondent, boolean isInShadowRound, Map<InetAddress, EndpointState> epStateMap)
{
if (inShadowRound)
{
if (!isInShadowRound)
{
+ if (!isShadowRoundStateMap(epStateMap))
+ return;
+
if (!seeds.contains(respondent))
logger.warn("Received an ack from {}, who isn't a seed. Ensure your seed list includes a live node. Exiting shadow round",
respondent);
diff --git a/test/unit/org/apache/cassandra/gms/ShadowRoundTest.java b/test/unit/org/apache/cassandra/gms/ShadowRoundTest.java
index f8cc49c..5ce72b5 100644
--- a/test/unit/org/apache/cassandra/gms/ShadowRoundTest.java
+++ b/test/unit/org/apache/cassandra/gms/ShadowRoundTest.java
@@ -28,6 +28,7 @@ import org.junit.Test;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
+import org.apache.cassandra.config.Config;
import org.apache.cassandra.config.DatabaseDescriptor;
import org.apache.cassandra.db.Keyspace;
import org.apache.cassandra.exceptions.ConfigurationException;
@@ -38,6 +39,7 @@ import org.apache.cassandra.net.MessagingService;
import org.apache.cassandra.net.MockMessagingService;
import org.apache.cassandra.net.MockMessagingSpy;
import org.apache.cassandra.service.StorageService;
+import org.apache.cassandra.utils.FBUtilities;
import static org.apache.cassandra.net.MockMessagingService.verb;
import static org.junit.Assert.assertEquals;
@@ -113,4 +115,39 @@ public class ShadowRoundTest
assertEquals(0, spyAck2.messagesIntercepted);
assertEquals(0, spyMigrationReq.messagesIntercepted);
}
+
+ @Test
+ public void testBadAckInShadow()
+ {
+ final AtomicBoolean ackSend = new AtomicBoolean(false);
+ MockMessagingSpy spySyn = MockMessagingService.when(verb(MessagingService.Verb.GOSSIP_DIGEST_SYN))
+ .respondN((msgOut, to) ->
+ {
+ // ACK with bad data in shadow round
+ if (!ackSend.compareAndSet(false, true))
+ {
+ while (!Gossiper.instance.isEnabled()) ;
+ }
+
+ HeartBeatState hb = new HeartBeatState(123, 456);
+ EndpointState state = new EndpointState(hb);
+ GossipDigestAck payload = new GossipDigestAck(
+ Collections.singletonList(new GossipDigest(FBUtilities.getBroadcastAddress(), hb.getGeneration(), hb.getHeartBeatVersion())),
+ Collections.singletonMap(FBUtilities.getBroadcastAddress(), state));
+
+ logger.warn("Simulating digest ACK reply, {}", payload);
+ return MessageIn.create(to, payload, Collections.emptyMap(), MessagingService.Verb.GOSSIP_DIGEST_ACK, MessagingService.current_version);
+ }, 1);
+
+ System.setProperty(Config.PROPERTY_PREFIX + "auto_bootstrap", "false");
+ try
+ {
+ StorageService.instance.initServer();
+ }
+ catch (Exception e)
+ {
+ assertEquals("Unable to gossip with any peers", e.getMessage());
+ }
+ }
+
}
---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@cassandra.apache.org
For additional commands, e-mail: commits-help@cassandra.apache.org