@@ -71,6 +71,8 @@ class SnapshotInstallationHandler {
7171 private final AtomicBoolean isSnapshotNull = new AtomicBoolean ();
7272 private final AtomicLong installedIndex = new AtomicLong (INVALID_LOG_INDEX );
7373 private final AtomicInteger nextChunkIndex = new AtomicInteger (-1 );
74+ /** The callId of the chunk with index 0. */
75+ private final AtomicLong chunk0CallId = new AtomicLong (-1 );
7476
7577 SnapshotInstallationHandler (RaftServerImpl server , RaftProperties properties ) {
7678 this .server = server ;
@@ -176,8 +178,22 @@ private CompletableFuture<InstallSnapshotReplyProto> checkAndInstallSnapshot(Ins
176178 state .setLeader (leaderId , "installSnapshot" );
177179
178180 server .updateLastRpcTime (FollowerState .UpdateType .INSTALL_SNAPSHOT_START );
181+ long callId = chunk0CallId .get ();
182+ // 1. leaderTerm < currentTerm will never come here
183+ // 2. leaderTerm == currentTerm && callId == request.getCallId()
184+ // means the snapshotRequest is staled with the same leader
185+ // 3. leaderTerm > currentTerm means this is a new snapshot request from a new leader,
186+ // chunk0CallId will be reset when a snapshot request with requestIndex == 0 is received .
187+ if (callId > request .getServerRequest ().getCallId () && currentTerm == leaderTerm ) {
188+ LOG .warn ("{}: Snapshot Request Staled: chunk 0 callId is {} but {}" , getMemberId (), callId ,
189+ ServerStringUtils .toInstallSnapshotRequestString (request ));
190+ InstallSnapshotReplyProto reply = toInstallSnapshotReplyProto (leaderId , getMemberId (),
191+ currentTerm , snapshotChunkRequest .getRequestIndex (), InstallSnapshotResult .SNAPSHOT_EXPIRED );
192+ return future .thenApply (dummy -> reply );
193+ }
179194 if (snapshotChunkRequest .getRequestIndex () == 0 ) {
180195 nextChunkIndex .set (0 );
196+ chunk0CallId .set (request .getServerRequest ().getCallId ());
181197 } else if (nextChunkIndex .get () != snapshotChunkRequest .getRequestIndex ()) {
182198 throw new IOException ("Snapshot request already failed at chunk index " + nextChunkIndex .get ()
183199 + "; ignoring request with chunk index " + snapshotChunkRequest .getRequestIndex ());
@@ -205,6 +221,7 @@ private CompletableFuture<InstallSnapshotReplyProto> checkAndInstallSnapshot(Ins
205221 // re-load the state machine if this is the last chunk
206222 if (snapshotChunkRequest .getDone ()) {
207223 state .reloadStateMachine (lastIncluded );
224+ chunk0CallId .set (-1 );
208225 }
209226 } finally {
210227 server .updateLastRpcTime (FollowerState .UpdateType .INSTALL_SNAPSHOT_COMPLETE );
0 commit comments