133tosakarin commented on PR #1168:
URL: https://github.com/apache/ratis/pull/1168#issuecomment-2417045970

   @szetszwo @OneSizeFitsQuorum 
   
   Please check the comments below. could you help me refer to which one to use 
as the final solution?
   There are currently two main options:
   1. Move future.join outsize the lock (current commit)
   2. Use wait/notify (Code Block below)
   
   ```
   Subject: [PATCH] use wait/notify
   ---
   Index: 
ratis-server/src/main/java/org/apache/ratis/server/leader/LogAppenderDaemon.java
   IDEA additional info:
   Subsystem: com.intellij.openapi.diff.impl.patch.CharsetEP
   <+>UTF-8
   ===================================================================
   diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/leader/LogAppenderDaemon.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/leader/LogAppenderDaemon.java
   --- 
a/ratis-server/src/main/java/org/apache/ratis/server/leader/LogAppenderDaemon.java
       (revision 62ae6d9f068ce01dd467a6fa83fb74dad9c2c8d8)
   +++ 
b/ratis-server/src/main/java/org/apache/ratis/server/leader/LogAppenderDaemon.java
       (date 1729085480985)
   @@ -94,6 +94,9 @@
            logAppender.restart();
          }
          closeFuture.complete(finalState);
   +      synchronized (logAppender.getServer()) {
   +        logAppender.getServer().notifyAll();
   +      }
        }
      }
    
   Index: 
ratis-server/src/main/java/org/apache/ratis/server/impl/FollowerState.java
   IDEA additional info:
   Subsystem: com.intellij.openapi.diff.impl.patch.CharsetEP
   <+>UTF-8
   ===================================================================
   diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/FollowerState.java 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/FollowerState.java
   --- 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/FollowerState.java    
 (revision 62ae6d9f068ce01dd467a6fa83fb74dad9c2c8d8)
   +++ 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/FollowerState.java    
 (date 1729086757702)
   @@ -130,6 +130,9 @@
          runImpl();
        } finally {
          stopped.complete(null);
   +      synchronized (server) {
   +        server.notifyAll();
   +      }
        }
      }
    
   Index: 
ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java
   IDEA additional info:
   Subsystem: com.intellij.openapi.diff.impl.patch.CharsetEP
   <+>UTF-8
   ===================================================================
   diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java
   --- 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java   
 (revision 62ae6d9f068ce01dd467a6fa83fb74dad9c2c8d8)
   +++ 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java   
 (date 1729087936568)
   @@ -565,16 +565,38 @@
        }
      }
    
   +   static class Pair<U, V> {
   +    private final U first;
   +    private final V second;
   +
   +    private Pair(U first, V second) {
   +      this.first = first;
   +      this.second = second;
   +    }
   +
   +    static <U,V> Pair<U, V> makePair(U u,  V v) {
   +      return new Pair<>(u, v);
   +    }
   +
   +    U first() {
   +      return first;
   +    }
   +
   +    V second() {
   +      return second;
   +    }
   +  }
   +
      /**
       * Change the server state to Follower if this server is in a different 
role or force is true.
       * @param newTerm The new term.
       * @param force Force to start a new {@link FollowerState} even if this 
server is already a follower.
       * @return if the term/votedFor should be updated to the new term
       */
   -  private boolean changeToFollower(long newTerm, boolean force, boolean 
allowListener, Object reason) {
   +  private Pair<Boolean, CompletableFuture<Void>> changeToFollower(long 
newTerm, boolean force, boolean allowListener, Object reason) {
        final AtomicReference<Boolean> metadataUpdated = new 
AtomicReference<>();
   -    changeToFollowerAsync(newTerm, force, allowListener, reason, 
metadataUpdated).join();
   -    return metadataUpdated.get();
   +    CompletableFuture<Void> future = changeToFollowerAsync(newTerm, force, 
allowListener, reason, metadataUpdated);
   +    return Pair.makePair(metadataUpdated.get(), future);
      }
    
      private synchronized CompletableFuture<Void> changeToFollowerAsync(
   @@ -617,20 +639,37 @@
          long newTerm,
          boolean allowListener,
          Object reason) throws IOException {
   -    if (changeToFollower(newTerm, false, allowListener, reason)) {
   -      state.persistMetadata();
   +    Pair<Boolean, CompletableFuture<Void>> pair = changeToFollower(newTerm, 
false, allowListener, reason);
   +    try {
   +      if (pair.first()) {
   +        state.persistMetadata();
   +      }
   +    } finally {
   +      waitFutureAndJoin(pair.second());
        }
      }
    
   -  synchronized void changeToLeader() {
   +  synchronized void waitFutureAndJoin(CompletableFuture<?> future) {
   +    while (!future.isDone()) {
   +      try {
   +        this.wait();
   +      } catch (InterruptedException e) {
   +        // ignore
   +      }
   +    }
   +    future.join();
   +  }
   +  
   +  synchronized CompletableFuture<Void> changeToLeader() {
        Preconditions.assertTrue(getInfo().isCandidate());
   -    role.shutdownLeaderElection();
   +    CompletableFuture<Void> future = role.shutdownLeaderElection();
        setRole(RaftPeerRole.LEADER, "changeToLeader");
        final LeaderStateImpl leader = role.updateLeaderState(this);
        state.becomeLeader();
    
        // start sending AppendEntries RPC to followers
        leader.start();
   +    return future;
      }
    
      @Override
   @@ -1460,8 +1499,9 @@
          final boolean voteGranted = context.decideVote(candidate, 
candidateLastEntry);
          if (candidate != null && phase == Phase.ELECTION) {
            // change server state in the ELECTION phase
   -        final boolean termUpdated =
   -            changeToFollower(candidateTerm, true, false, "candidate:" + 
candidateId);
   +        Pair<Boolean, CompletableFuture<Void>> pair = 
changeToFollower(candidateTerm, true, false, "candidate:" + candidateId);
   +        final boolean termUpdated = pair.first;
   +        waitFutureAndJoin(pair.second);
            if (voteGranted) {
              state.grantVote(candidate.getId());
            }
   Index: 
ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderElection.java
   IDEA additional info:
   Subsystem: com.intellij.openapi.diff.impl.patch.CharsetEP
   <+>UTF-8
   ===================================================================
   diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderElection.java 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderElection.java
   --- 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderElection.java   
 (revision 62ae6d9f068ce01dd467a6fa83fb74dad9c2c8d8)
   +++ 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/LeaderElection.java   
 (date 1729087936578)
   @@ -242,6 +242,9 @@
          runImpl();
        } finally {
          stopped.complete(null);
   +      synchronized (server) {
   +        server.notifyAll();
   +      }
        }
      }
    
   @@ -256,7 +259,7 @@
          for (int round = 0; shouldRun(); round++) {
            if (skipPreVote || askForVotes(Phase.PRE_VOTE, round)) {
              if (askForVotes(Phase.ELECTION, round)) {
   -            server.changeToLeader();
   +            server.changeToLeader().join();
              }
            }
          }
   
   ```


-- 
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