kowshik commented on a change in pull request #11345:
URL: https://github.com/apache/kafka/pull/11345#discussion_r760752732



##########
File path: core/src/main/scala/kafka/log/UnifiedLog.scala
##########
@@ -1498,28 +1498,44 @@ class UnifiedLog(@volatile var logStartOffset: Long,
     producerStateManager.takeSnapshot()
     updateHighWatermarkWithLogEndOffset()
     // Schedule an asynchronous flush of the old segment
-    scheduler.schedule("flush-log", () => flush(newSegment.baseOffset))
+    scheduler.schedule("flush-log", () => 
flushUptoOffsetExclusive(newSegment.baseOffset))
     newSegment
   }
 
   /**
    * Flush all local log segments
+   *
+   * @param forceFlushActiveSegment should be true during a clean shutdown, 
and false otherwise. The reason is that
+   * we have to pass logEndOffset + 1 to the `localLog.flush(offset: Long): 
Unit` function to flush empty
+   * active segments, which is important to make sure we persist the active 
segment file during shutdown, particularly
+   * when it's empty.
    */
-  def flush(): Unit = flush(logEndOffset)
+  def flush(forceFlushActiveSegment: Boolean): Unit = flush(logEndOffset, 
forceFlushActiveSegment)

Review comment:
       @hachikuji The PR description needs to be updated, but the main take 
away from this PR is that it helps fix a durability corner case during clean 
shutdown. Before this PR, an empty active segment doesn't get flushed to disk 
during clean shutdown. This PR fixes it, and after this PR, it gets flushed. 
Without this fix, the loss of an empty active segment file on disk can cause 
the replica to lose track of logEndOffset.
   
   Alongside the above fix, the PR also prevents some unwanted logging side 
effect in the recovery code path in LogLoader.
   
   I agree that the noise to the flush API signature is unfortunate. I'm not 
sure if we can refactor to introduce a better API that can avoid the noise. But 
given that this PR fixes a durability case, the PR seems useful overall.




-- 
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: jira-unsubscr...@kafka.apache.org

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


Reply via email to