You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ratis.apache.org by GitBox <gi...@apache.org> on 2022/03/03 11:48:12 UTC

[GitHub] [ratis] szetszwo commented on a change in pull request #616: RATIS-1545. RaftLogOutputStream support async flush.

szetszwo commented on a change in pull request #616:
URL: https://github.com/apache/ratis/pull/616#discussion_r818574232



##########
File path: ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
##########
@@ -144,6 +144,7 @@ synchronized void updateIndex(long i) {
   private final WriteLogTasks writeTasks = new WriteLogTasks();
   private volatile boolean running = true;
   private final Thread workerThread;
+  private final Thread outStreamFlushThread;

Review comment:
       Let's use an ExecutorService.

##########
File path: ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
##########
@@ -176,7 +177,7 @@ synchronized void updateIndex(long i) {
   private final RaftServer.Division server;
   private int flushBatchSize;
 
-  private Timestamp lastFlush;
+  private Timestamp outStreamLastFlush;

Review comment:
       State machine and out stream should either flush together or not flush at all.  We should not try to separate them.  So let's don't rename this field.

##########
File path: ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
##########
@@ -349,13 +354,25 @@ private void run() {
     }
   }
 
+  private void checkAndFlush() {
+    while (running) {
+      try {
+        outStreamFlushIfNecessary();
+        flushIntervalMin.sleep();
+      } catch (Exception e) {
+        LOG.error("{} hit exception", Thread.currentThread().getName(), e);
+        Optional.ofNullable(server).ifPresent(RaftServer.Division::close);
+      }
+    }
+  }
+
   private boolean shouldFlush() {
     if (out == null) {
       return false;
     } else if (pendingFlushNum >= forceSyncNum) {
       return true;
     }
-    return pendingFlushNum > 0 && queue.isEmpty() && lastFlush.elapsedTime().compareTo(flushIntervalMin) > 0;
+    return pendingFlushNum > 0 && queue.isEmpty();

Review comment:
       Let's change flush to async but not the min time logic.




-- 
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: issues-unsubscribe@ratis.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org