szetszwo commented on code in PR #898:
URL: https://github.com/apache/ratis/pull/898#discussion_r1284785622


##########
ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderStateImpl.java:
##########
@@ -322,6 +323,9 @@ private List<FollowerInfo> 
getFollowerInfos(PeerConfiguration peers) {
   private final PendingStepDown pendingStepDown;
 
   private final ReadIndexHeartbeats readIndexHeartbeats;
+  private final long leaseTimeoutMs;
+  // TODO invalidate leader lease when stepDown / transferLeader
+  private final AtomicReference<Timestamp> lease = new 
AtomicReference<>(Timestamp.currentTime());

Review Comment:
   Similar to ReadIndexHeartbeats, how about we move all the leader-lease logic 
to a new class?



##########
ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderStateImpl.java:
##########
@@ -1127,6 +1139,47 @@ public void onAppendEntriesReply(LogAppender appender, 
RaftProtos.AppendEntriesR
     readIndexHeartbeats.onAppendEntriesReply(appender, reply, 
this::hasMajority);
   }
 
+  boolean hasValidLeaderLease() {
+    if (checkLeaderLease()) {
+      return true;
+    }
+
+    extendLeaderLease();
+    return checkLeaderLease();
+  }
+
+  private boolean checkLeaderLease() {
+    return isReady() && lease.get().elapsedTimeMs() < leaseTimeoutMs;

Review Comment:
   Check for single node.
   ```java
     private boolean checkLeaderLease() {
       if (!isReady()) {
         return false;
       }
       final RaftConfigurationImpl conf = server.getRaftConf();
       if (conf.getCurrentPeers().size() == 1 && conf.getPreviousPeers().size() 
<= 1) {
         return true;
       }
       return lease.get().elapsedTimeMs() < leaseTimeoutMs;
     }
   ```
   
   BTW, we should also check previous peers for read index; see below.
   ```java
   @@ -1094,7 +1106,8 @@ class LeaderStateImpl implements LeaderState {
        final long readIndex = server.getRaftLog().getLastCommittedIndex();
    
        // if group contains only one member, fast path
   -    if (server.getRaftConf().getCurrentPeers().size() == 1) {
   +    final RaftConfigurationImpl conf = server.getRaftConf();
   +    if (conf.getCurrentPeers().size() == 1 && 
conf.getPreviousPeers().size() <= 1) {
          return CompletableFuture.completedFuture(readIndex);
        }
   ```



##########
ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderStateImpl.java:
##########
@@ -1127,6 +1139,47 @@ public void onAppendEntriesReply(LogAppender appender, 
RaftProtos.AppendEntriesR
     readIndexHeartbeats.onAppendEntriesReply(appender, reply, 
this::hasMajority);
   }
 
+  boolean hasValidLeaderLease() {
+    if (checkLeaderLease()) {
+      return true;
+    }
+
+    extendLeaderLease();
+    return checkLeaderLease();
+  }
+
+  private boolean checkLeaderLease() {
+    return isReady() && lease.get().elapsedTimeMs() < leaseTimeoutMs;
+  }
+
+  private void extendLeaderLease() {

Review Comment:
   This is tricky since we also have to take care about previous peers, if they 
exists.  How to extend leader lease when the conf is changing?
   



##########
ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java:
##########
@@ -747,6 +746,8 @@ static class AppendEntriesRequest {
 
     private final TermIndex lastEntry;
 
+    private Timestamp sendTime;

Review Comment:
   It should be `volatile`.



##########
ratis-grpc/src/main/java/org/apache/ratis/grpc/server/GrpcLogAppender.java:
##########
@@ -368,6 +368,7 @@ public void onNext(AppendEntriesReplyProto reply) {
       AppendEntriesRequest request = pendingRequests.remove(reply);
       if (request != null) {
         request.stopRequestTimer(); // Update completion time
+        
getFollower().updateLastAppendEntriesResponseTime(request.getSendTime()); // 
Update the last rpc time
       }

Review Comment:
   We should updateLastRpcResponseTime() when request == null.
   ```jav
         if (request != null) {
           request.stopRequestTimer(); // Update completion time
           
getFollower().updateLastAppendEntriesResponseTime(request.getSendTime());
         } else {
           getFollower().updateLastRpcResponseTime();
         }
   ```



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to