|  | 
| 18 | 18 | package org.apache.spark.scheduler | 
| 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 | 23 | import scala.util.DynamicVariable | 
| 24 | 24 | 
 | 
| @@ -57,6 +57,12 @@ private[spark] class LiveListenerBus(val sparkContext: SparkContext) extends Spa | 
| 57 | 57 |   // Indicate if `stop()` is called | 
| 58 | 58 |   private val stopped = new AtomicBoolean(false) | 
| 59 | 59 | 
 | 
|  | 60 | +  /** A counter for dropped events. It will be reset every time we log it. */ | 
|  | 61 | +  private val droppedEventsCounter = new AtomicLong(0L) | 
|  | 62 | + | 
|  | 63 | +  /** When `droppedEventsCounter` was logged last time. */ | 
|  | 64 | +  @volatile private var lastReportTimestamp = 0L | 
|  | 65 | + | 
| 60 | 66 |   // Indicate if we are processing some event | 
| 61 | 67 |   // Guarded by `self` | 
| 62 | 68 |   private var processingEvent = false | 
| @@ -123,6 +129,23 @@ private[spark] class LiveListenerBus(val sparkContext: SparkContext) extends Spa | 
| 123 | 129 |       eventLock.release() | 
| 124 | 130 |     } else { | 
| 125 | 131 |       onDropEvent(event) | 
|  | 132 | +      droppedEventsCounter.incrementAndGet() | 
|  | 133 | +    } | 
|  | 134 | +    // Don't log too frequently | 
|  | 135 | +    if (System.currentTimeMillis() - lastReportTimestamp >= 60 * 1000) { | 
|  | 136 | +      var droppedEvents = droppedEventsCounter.get | 
|  | 137 | +      while (droppedEvents > 0) { | 
|  | 138 | +        // There may be multiple threads trying to decrease droppedEventsCounter. | 
|  | 139 | +        // Use "compareAndSet" to make sure only one thread can win. | 
|  | 140 | +        // And if another thread is increasing droppedEventsCounter, "compareAndSet" will fail | 
|  | 141 | +        // and we will try again. | 
|  | 142 | +        if (droppedEventsCounter.compareAndSet(droppedEvents, 0)) { | 
|  | 143 | +          lastReportTimestamp = System.currentTimeMillis() | 
|  | 144 | +          logWarning(s"Dropped $droppedEvents SparkListenerEvents") | 
|  | 145 | +          return | 
|  | 146 | +        } | 
|  | 147 | +        droppedEvents = droppedEventsCounter.get | 
|  | 148 | +      } | 
| 126 | 149 |     } | 
| 127 | 150 |   } | 
| 128 | 151 | 
 | 
|  | 
0 commit comments