smengcl commented on code in PR #4621:
URL: https://github.com/apache/ozone/pull/4621#discussion_r1185744899


##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerDoubleBuffer.java:
##########
@@ -611,7 +630,12 @@ public synchronized CompletableFuture<Void> 
add(OMClientResponse response,
   private synchronized boolean canFlush() {
     try {
       while (currentBuffer.size() == 0) {
-        wait(Long.MAX_VALUE);
+        wait(1000L);
+        if  (currentBuffer.size() == 0) {
+          // Both buffers are empty, so notify twice
+          flushNotifier.notifyFlush();
+          flushNotifier.notifyFlush();
+        }
       }
       return true;
     }  catch (InterruptedException ex) {

Review Comment:
   ```suggestion
       } catch (InterruptedException ex) {
   ```



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerDoubleBuffer.java:
##########
@@ -649,4 +673,45 @@ private synchronized void swapCurrentAndReadyBuffer() {
   public OzoneManagerDoubleBufferMetrics getOzoneManagerDoubleBufferMetrics() {
     return ozoneManagerDoubleBufferMetrics;
   }
+
+  @VisibleForTesting
+  int getCurrentBufferSize() {
+    return currentBuffer.size();
+  }
+
+  @VisibleForTesting
+  int getReadyBufferSize() {
+    return readyBuffer.size();
+  }
+
+  @VisibleForTesting
+  void resume() {
+    isRunning.set(true);
+  }
+
+  void awaitFlush() throws InterruptedException {
+    flushNotifier.await();
+  }
+
+  static class FlushNotifier {
+    private final Set<CountDownLatch> flushLatches =
+        ConcurrentHashMap.newKeySet();
+
+    void await() throws InterruptedException {
+
+      // Wait until both the current and ready buffers are flushed.
+      CountDownLatch latch = new CountDownLatch(2);
+      flushLatches.add(latch);
+      latch.await();
+      flushLatches.remove(latch);
+    }
+
+    int notifyFlush() {
+      int retval = flushLatches.size();
+      for (CountDownLatch l: flushLatches) {

Review Comment:
   ```suggestion
         for (CountDownLatch l : flushLatches) {
   ```



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerDoubleBuffer.java:
##########
@@ -611,7 +630,12 @@ public synchronized CompletableFuture<Void> 
add(OMClientResponse response,
   private synchronized boolean canFlush() {
     try {
       while (currentBuffer.size() == 0) {
-        wait(Long.MAX_VALUE);
+        wait(1000L);
+        if  (currentBuffer.size() == 0) {

Review Comment:
   ```suggestion
           if (currentBuffer.size() == 0) {
   ```



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerDoubleBuffer.java:
##########
@@ -611,7 +630,12 @@ public synchronized CompletableFuture<Void> 
add(OMClientResponse response,
   private synchronized boolean canFlush() {
     try {
       while (currentBuffer.size() == 0) {
-        wait(Long.MAX_VALUE);
+        wait(1000L);
+        if  (currentBuffer.size() == 0) {
+          // Both buffers are empty, so notify twice
+          flushNotifier.notifyFlush();
+          flushNotifier.notifyFlush();
+        }

Review Comment:
   How is this snippet confirming both buffers are empty? Would you need to 
check `readyBuffer` size as well?



##########
hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerStateMachine.java:
##########
@@ -718,4 +718,7 @@ public long getTermForIndex(long transactionIndex) {
     return applyTransactionMap.get(transactionIndex);
   }
 
+  public void awaitDoubleBufferFlush() throws InterruptedException {

Review Comment:
   nit: javadoc to explain its intended usage



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


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to