You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kafka.apache.org by jg...@apache.org on 2020/06/19 03:33:33 UTC

[kafka] branch trunk updated: KAFKA-10141; Add more detail to log segment delete messages (#8850)

This is an automated email from the ASF dual-hosted git repository.

jgus pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/kafka.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 1e5fa9e  KAFKA-10141; Add more detail to log segment delete messages (#8850)
1e5fa9e is described below

commit 1e5fa9e082fcd5875222f9d7458ffa1fa7b1587b
Author: Sanjana Kaundinya <sk...@gmail.com>
AuthorDate: Thu Jun 18 20:32:55 2020 -0700

    KAFKA-10141; Add more detail to log segment delete messages (#8850)
    
    It is helpful to include as much information as possible when deleting log segments. This patch introduces log messages that give more specific details as to why the log segment was deleted and the specific metadata regarding that log segment.
    
    Reviewers: Jason Gustafson <ja...@confluent.io>
---
 core/src/main/scala/kafka/log/Log.scala        | 43 +++++++++++++++++++++-----
 core/src/main/scala/kafka/log/LogSegment.scala |  5 +++
 2 files changed, 40 insertions(+), 8 deletions(-)

diff --git a/core/src/main/scala/kafka/log/Log.scala b/core/src/main/scala/kafka/log/Log.scala
index d38daf8a..decff4d 100644
--- a/core/src/main/scala/kafka/log/Log.scala
+++ b/core/src/main/scala/kafka/log/Log.scala
@@ -1702,11 +1702,10 @@ class Log(@volatile private var _dir: File,
    *                  (if there is one) and returns true iff it is deletable
    * @return The number of segments deleted
    */
-  private def deleteOldSegments(predicate: (LogSegment, Option[LogSegment]) => Boolean, reason: String): Int = {
+  private def deleteOldSegments(predicate: (LogSegment, Option[LogSegment]) => Boolean) = {
     lock synchronized {
       val deletable = deletableSegments(predicate)
       if (deletable.nonEmpty) {
-        info(s"Found deletable segments with base offsets [${deletable.map(_.baseOffset).mkString(",")}] due to $reason")
         deleteSegments(deletable)
       } else 0
     }
@@ -1784,8 +1783,26 @@ class Log(@volatile private var _dir: File,
   private def deleteRetentionMsBreachedSegments(): Int = {
     if (config.retentionMs < 0) return 0
     val startMs = time.milliseconds
-    deleteOldSegments((segment, _) => startMs - segment.largestTimestamp > config.retentionMs,
-      reason = s"retention time ${config.retentionMs}ms breach")
+
+    def shouldDelete(segment: LogSegment, nextSegmentOpt: Option[LogSegment]) = {
+      if (startMs - segment.largestTimestamp > config.retentionMs) {
+        segment.largestRecordTimestamp match {
+          case Some(ts) =>
+            info(s"Segment with base offset ${segment.baseOffset} will be deleted due to" +
+              s" retention time ${config.retentionMs}ms breach based on the largest record timestamp from the" +
+              s" segment, which is $ts")
+          case None =>
+            info(s"Segment with base offset ${segment.baseOffset} will be deleted due to" +
+              s" retention time ${config.retentionMs}ms breach based on the last modified timestamp from the" +
+              s" segment, which is ${segment.lastModified}")
+        }
+        true
+      } else {
+        false
+      }
+    }
+
+    deleteOldSegments(shouldDelete)
   }
 
   private def deleteRetentionSizeBreachedSegments(): Int = {
@@ -1794,20 +1811,30 @@ class Log(@volatile private var _dir: File,
     def shouldDelete(segment: LogSegment, nextSegmentOpt: Option[LogSegment]) = {
       if (diff - segment.size >= 0) {
         diff -= segment.size
+        info(s"Segment with base offset ${segment.baseOffset} will be deleted due to" +
+          s" retention size ${config.retentionSize} bytes breach. Segment size is" +
+          s" ${segment.size} and total log size after deletion will be ${size - diff}")
         true
       } else {
         false
       }
     }
 
-    deleteOldSegments(shouldDelete, reason = s"retention size in bytes ${config.retentionSize} breach")
+    deleteOldSegments(shouldDelete)
   }
 
   private def deleteLogStartOffsetBreachedSegments(): Int = {
-    def shouldDelete(segment: LogSegment, nextSegmentOpt: Option[LogSegment]) =
-      nextSegmentOpt.exists(_.baseOffset <= logStartOffset)
+    def shouldDelete(segment: LogSegment, nextSegmentOpt: Option[LogSegment]) = {
+      if (nextSegmentOpt.exists(_.baseOffset <= logStartOffset)) {
+        info(s"Segment with base offset ${segment.baseOffset} will be deleted due to" +
+          s" startOffset breach. logStartOffset is $logStartOffset")
+        true
+      } else {
+        false
+      }
+    }
 
-    deleteOldSegments(shouldDelete, reason = s"log start offset $logStartOffset breach")
+    deleteOldSegments(shouldDelete)
   }
 
   def isFuture: Boolean = dir.getName.endsWith(Log.FutureDirSuffix)
diff --git a/core/src/main/scala/kafka/log/LogSegment.scala b/core/src/main/scala/kafka/log/LogSegment.scala
index 61a6e59..ce5d48c 100755
--- a/core/src/main/scala/kafka/log/LogSegment.scala
+++ b/core/src/main/scala/kafka/log/LogSegment.scala
@@ -634,6 +634,11 @@ class LogSegment private[log] (val log: FileRecords,
   def lastModified = log.file.lastModified
 
   /**
+   * The largest timestamp this segment contains, if maxTimestampSoFar >= 0, otherwise None.
+   */
+  def largestRecordTimestamp: Option[Long] = if (maxTimestampSoFar >= 0) Some(maxTimestampSoFar) else None
+
+  /**
    * The largest timestamp this segment contains.
    */
   def largestTimestamp = if (maxTimestampSoFar >= 0) maxTimestampSoFar else lastModified