Skip to content

Commit 71cf6fa

Browse files
authored
Only turn to follower when term bumping on follower check (#36449)
Deals with a situation where a follower becomes disconnected from the leader, but only for such a short time where it becomes candidate and puts up a NO_MASTER_BLOCK, but then receives a follower check from the leader. If the leader does not notice the node disconnecting, it is important for the node not to be turned back into a follower but try and join the leader again. We still should prefer the node into a follower on a follower check when this follower check triggers a term bump as this can help during a leader election to quickly have a leader turn all other nodes into followers, even before the leader has had the chance to transfer a possibly very large cluster state. Closes #36428
1 parent d8e3d97 commit 71cf6fa

File tree

3 files changed

+39
-3
lines changed

3 files changed

+39
-3
lines changed

server/src/main/java/org/elasticsearch/cluster/coordination/Coordinator.java

Lines changed: 5 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -206,8 +206,9 @@ private void removeNode(DiscoveryNode discoveryNode, String reason) {
206206
}
207207
}
208208

209-
private void onFollowerCheckRequest(FollowerCheckRequest followerCheckRequest) {
209+
void onFollowerCheckRequest(FollowerCheckRequest followerCheckRequest) {
210210
synchronized (mutex) {
211+
final long previousTerm = getCurrentTerm();
211212
ensureTermAtLeast(followerCheckRequest.getSender(), followerCheckRequest.getTerm());
212213

213214
if (getCurrentTerm() != followerCheckRequest.getTerm()) {
@@ -216,7 +217,9 @@ private void onFollowerCheckRequest(FollowerCheckRequest followerCheckRequest) {
216217
+ getCurrentTerm() + "], rejecting " + followerCheckRequest);
217218
}
218219

219-
becomeFollower("onFollowerCheckRequest", followerCheckRequest.getSender());
220+
if (previousTerm != getCurrentTerm()) {
221+
becomeFollower("onFollowerCheckRequest", followerCheckRequest.getSender());
222+
}
220223
}
221224
}
222225

server/src/test/java/org/elasticsearch/cluster/coordination/CoordinatorTests.java

Lines changed: 34 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -95,6 +95,7 @@
9595
import static org.elasticsearch.cluster.coordination.LeaderChecker.LEADER_CHECK_RETRY_COUNT_SETTING;
9696
import static org.elasticsearch.cluster.coordination.LeaderChecker.LEADER_CHECK_TIMEOUT_SETTING;
9797
import static org.elasticsearch.cluster.coordination.Reconfigurator.CLUSTER_AUTO_SHRINK_VOTING_CONFIGURATION;
98+
import static org.elasticsearch.discovery.DiscoverySettings.NO_MASTER_BLOCK_ID;
9899
import static org.elasticsearch.discovery.PeerFinder.DISCOVERY_FIND_PEERS_INTERVAL_SETTING;
99100
import static org.elasticsearch.node.Node.NODE_NAME_SETTING;
100101
import static org.elasticsearch.transport.TransportService.HANDSHAKE_ACTION_NAME;
@@ -110,6 +111,7 @@
110111
import static org.hamcrest.Matchers.lessThan;
111112
import static org.hamcrest.Matchers.lessThanOrEqualTo;
112113
import static org.hamcrest.Matchers.not;
114+
import static org.hamcrest.Matchers.nullValue;
113115
import static org.hamcrest.Matchers.sameInstance;
114116
import static org.hamcrest.Matchers.startsWith;
115117

@@ -887,6 +889,31 @@ public void testIncompatibleDiffResendsFullState() {
887889
postPublishStats.getIncompatibleClusterStateDiffReceivedCount());
888890
}
889891

892+
/**
893+
* Simulates a situation where a follower becomes disconnected from the leader, but only for such a short time where
894+
* it becomes candidate and puts up a NO_MASTER_BLOCK, but then receives a follower check from the leader. If the leader
895+
* does not notice the node disconnecting, it is important for the node not to be turned back into a follower but try
896+
* and join the leader again.
897+
*/
898+
public void testStayCandidateAfterReceivingFollowerCheckFromKnownMaster() {
899+
final Cluster cluster = new Cluster(2, false);
900+
cluster.runRandomly();
901+
cluster.stabilise();
902+
903+
final ClusterNode leader = cluster.getAnyLeader();
904+
final ClusterNode nonLeader = cluster.getAnyNodeExcept(leader);
905+
onNode(nonLeader.getLocalNode(), () -> {
906+
logger.debug("forcing {} to become candidate", nonLeader.getId());
907+
synchronized (nonLeader.coordinator.mutex) {
908+
nonLeader.coordinator.becomeCandidate("forced");
909+
}
910+
logger.debug("simulate follower check coming through from {} to {}", leader.getId(), nonLeader.getId());
911+
nonLeader.coordinator.onFollowerCheckRequest(new FollowersChecker.FollowerCheckRequest(leader.coordinator.getCurrentTerm(),
912+
leader.getLocalNode()));
913+
}).run();
914+
cluster.stabilise();
915+
}
916+
890917
private static long defaultMillis(Setting<TimeValue> setting) {
891918
return setting.get(Settings.EMPTY).millis() + Cluster.DEFAULT_DELAY_VARIABILITY;
892919
}
@@ -1176,8 +1203,15 @@ void stabilise(long stabilisationDurationMillis) {
11761203
assertTrue(nodeId + " is in the latest applied state on " + leaderId,
11771204
leader.getLastAppliedClusterState().getNodes().nodeExists(nodeId));
11781205
assertTrue(nodeId + " has been bootstrapped", clusterNode.coordinator.isInitialConfigurationSet());
1206+
assertThat(nodeId + " has correct master", clusterNode.getLastAppliedClusterState().nodes().getMasterNode(),
1207+
equalTo(leader.getLocalNode()));
1208+
assertThat(nodeId + " has no NO_MASTER_BLOCK",
1209+
clusterNode.getLastAppliedClusterState().blocks().hasGlobalBlock(NO_MASTER_BLOCK_ID), equalTo(false));
11791210
} else {
11801211
assertThat(nodeId + " is not following " + leaderId, clusterNode.coordinator.getMode(), is(CANDIDATE));
1212+
assertThat(nodeId + " has no master", clusterNode.getLastAppliedClusterState().nodes().getMasterNode(), nullValue());
1213+
assertThat(nodeId + " has NO_MASTER_BLOCK",
1214+
clusterNode.getLastAppliedClusterState().blocks().hasGlobalBlock(NO_MASTER_BLOCK_ID), equalTo(true));
11811215
assertFalse(nodeId + " is not in the applied state on " + leaderId,
11821216
leader.getLastAppliedClusterState().getNodes().nodeExists(nodeId));
11831217
}

server/src/test/java/org/elasticsearch/discovery/ClusterDisruptionIT.java

Lines changed: 0 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -286,7 +286,6 @@ public void testRejoinDocumentExistsInAllShardCopies() throws Exception {
286286
}
287287

288288
// simulate handling of sending shard failure during an isolation
289-
@AwaitsFix(bugUrl = "https://github.com/elastic/elasticsearch/issues/36428")
290289
public void testSendingShardFailure() throws Exception {
291290
List<String> nodes = startCluster(3);
292291
String masterNode = internalCluster().getMasterName();

0 commit comments

Comments
 (0)