|
18 | 18 | package org.apache.spark.util
|
19 | 19 |
|
20 | 20 | import java.util.concurrent._
|
21 |
| -import java.util.concurrent.atomic.AtomicBoolean |
| 21 | +import java.util.concurrent.atomic.{AtomicBoolean, AtomicLong} |
| 22 | + |
22 | 23 | import scala.util.DynamicVariable
|
23 | 24 |
|
24 | 25 | import org.apache.spark.SparkContext
|
@@ -51,6 +52,12 @@ private[spark] abstract class AsynchronousListenerBus[L <: AnyRef, E](name: Stri
|
51 | 52 | // Indicate if `stop()` is called
|
52 | 53 | private val stopped = new AtomicBoolean(false)
|
53 | 54 |
|
| 55 | + /** A counter for dropped events. It will be reset every time we log it. */ |
| 56 | + private val droppedEventsCounter = new AtomicLong(0L) |
| 57 | + |
| 58 | + /** When `droppedEventsCounter` was logged last time in milliseconds. */ |
| 59 | + @volatile private var lastReportTimestamp = 0L |
| 60 | + |
54 | 61 | // Indicate if we are processing some event
|
55 | 62 | // Guarded by `self`
|
56 | 63 | private var processingEvent = false
|
@@ -117,6 +124,24 @@ private[spark] abstract class AsynchronousListenerBus[L <: AnyRef, E](name: Stri
|
117 | 124 | eventLock.release()
|
118 | 125 | } else {
|
119 | 126 | onDropEvent(event)
|
| 127 | + droppedEventsCounter.incrementAndGet() |
| 128 | + } |
| 129 | + |
| 130 | + val droppedEvents = droppedEventsCounter.get |
| 131 | + if (droppedEvents > 0) { |
| 132 | + // Don't log too frequently |
| 133 | + if (System.currentTimeMillis() - lastReportTimestamp >= 60 * 1000) { |
| 134 | + // There may be multiple threads trying to decrease droppedEventsCounter. |
| 135 | + // Use "compareAndSet" to make sure only one thread can win. |
| 136 | + // And if another thread is increasing droppedEventsCounter, "compareAndSet" will fail and |
| 137 | + // then that thread will update it. |
| 138 | + if (droppedEventsCounter.compareAndSet(droppedEvents, 0)) { |
| 139 | + val prevLastReportTimestamp = lastReportTimestamp |
| 140 | + lastReportTimestamp = System.currentTimeMillis() |
| 141 | + logWarning(s"Dropped $droppedEvents SparkListenerEvents since " + |
| 142 | + new java.util.Date(prevLastReportTimestamp)) |
| 143 | + } |
| 144 | + } |
120 | 145 | }
|
121 | 146 | }
|
122 | 147 |
|
|
0 commit comments