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

szetszwo pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-ratis.git


The following commit(s) were added to refs/heads/master by this push:
     new de557a7  RATIS-558. GrpcLogAppender doesn't reset the log entries to 
follower on reply to inconsistent entries.
de557a7 is described below

commit de557a713d4834cc6343d474ac64e122d9c8267f
Author: Tsz Wo Nicholas Sze <[email protected]>
AuthorDate: Thu May 23 13:59:25 2019 +0800

    RATIS-558. GrpcLogAppender doesn't reset the log entries to follower on 
reply to inconsistent entries.
---
 .../java/org/apache/ratis/grpc/server/GrpcLogAppender.java  | 13 ++++---------
 .../src/test/java/org/apache/ratis/RaftAsyncTests.java      |  4 ++--
 2 files changed, 6 insertions(+), 11 deletions(-)

diff --git 
a/ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java 
b/ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java
index 0403ae9..aa719bb 100644
--- a/ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java
+++ b/ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java
@@ -264,7 +264,7 @@ public class GrpcLogAppender extends LogAppender {
           }
           break;
         case INCONSISTENCY:
-          checkAndUpdateNextIndex(request, reply.getNextIndex());
+          updateNextIndex(reply.getNextIndex());
           break;
         default:
           throw new IllegalStateException("Unexpected reply result: " + 
reply.getResult());
@@ -300,14 +300,9 @@ public class GrpcLogAppender extends LogAppender {
     return follower.updateMatchIndex(newMatchIndex);
   }
 
-  private void checkAndUpdateNextIndex(AppendEntriesRequestProto request, long 
replyNextIndex) {
-    Preconditions.assertTrue(request.hasPreviousLog());
-    if (request.getPreviousLog().getIndex() >= replyNextIndex) {
-      synchronized (this) {
-        pendingRequests.clear();
-        follower.updateNextIndex(replyNextIndex);
-      }
-    }
+  private synchronized void updateNextIndex(long replyNextIndex) {
+    pendingRequests.clear();
+    follower.updateNextIndex(replyNextIndex);
   }
 
   private class InstallSnapshotResponseHandler
diff --git a/ratis-server/src/test/java/org/apache/ratis/RaftAsyncTests.java 
b/ratis-server/src/test/java/org/apache/ratis/RaftAsyncTests.java
index aa9893a..7e10441 100644
--- a/ratis-server/src/test/java/org/apache/ratis/RaftAsyncTests.java
+++ b/ratis-server/src/test/java/org/apache/ratis/RaftAsyncTests.java
@@ -376,14 +376,14 @@ public abstract class RaftAsyncTests<CLUSTER extends 
MiniRaftCluster> extends Ba
       CompletableFuture<RaftClientReply> replyFuture = client.sendAsync(new 
SimpleMessage("abc"));
       Thread.sleep(waitTime);
       // replyFuture should not be completed until append request is unblocked.
-      Assert.assertTrue(!replyFuture.isDone());
+      Assert.assertFalse(replyFuture.isDone());
       // unblock append request.
       cluster.getServerAliveStream()
           .filter(impl -> !impl.isLeader())
           .map(SimpleStateMachine4Testing::get)
           .forEach(SimpleStateMachine4Testing::unblockWriteStateMachineData);
 
-      replyFuture.get();
+      Assert.assertTrue(replyFuture.get().isSuccess());
       Assert.assertTrue(System.currentTimeMillis() - time > waitTime);
     }
 

Reply via email to