|
@@ -24,6 +24,7 @@ import java.util.List;
|
|
|
import java.util.Map;
|
|
|
import java.util.concurrent.BlockingQueue;
|
|
|
import java.util.concurrent.LinkedBlockingQueue;
|
|
|
+import java.util.stream.Collectors;
|
|
|
|
|
|
import org.slf4j.Logger;
|
|
|
import org.slf4j.LoggerFactory;
|
|
@@ -31,6 +32,7 @@ import org.slf4j.Marker;
|
|
|
import org.slf4j.MarkerFactory;
|
|
|
import org.apache.hadoop.classification.InterfaceAudience.Public;
|
|
|
import org.apache.hadoop.classification.InterfaceStability.Evolving;
|
|
|
+import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.service.AbstractService;
|
|
|
import org.apache.hadoop.util.ShutdownHookManager;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
@@ -55,8 +57,13 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
|
|
|
|
|
|
private final BlockingQueue<Event> eventQueue;
|
|
|
private volatile int lastEventQueueSizeLogged = 0;
|
|
|
+ private volatile int lastEventDetailsQueueSizeLogged = 0;
|
|
|
private volatile boolean stopped = false;
|
|
|
|
|
|
+ //Configuration for control the details queue event printing.
|
|
|
+ private int detailsInterval;
|
|
|
+ private boolean printTrigger = false;
|
|
|
+
|
|
|
// Configuration flag for enabling/disabling draining dispatcher's events on
|
|
|
// stop functionality.
|
|
|
private volatile boolean drainEventsOnStop = false;
|
|
@@ -129,6 +136,12 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
|
|
|
}
|
|
|
if (event != null) {
|
|
|
dispatch(event);
|
|
|
+ if (printTrigger) {
|
|
|
+ //Log the latest dispatch event type
|
|
|
+ // may cause the too many events queued
|
|
|
+ LOG.info("Latest dispatch event type: " + event.getType());
|
|
|
+ printTrigger = false;
|
|
|
+ }
|
|
|
}
|
|
|
}
|
|
|
}
|
|
@@ -140,6 +153,15 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
|
|
|
exitOnDispatchException = false;
|
|
|
}
|
|
|
|
|
|
+ @Override
|
|
|
+ protected void serviceInit(Configuration conf) throws Exception{
|
|
|
+ super.serviceInit(conf);
|
|
|
+ this.detailsInterval = getConfig().getInt(YarnConfiguration.
|
|
|
+ YARN_DISPATCHER_PRINT_EVENTS_INFO_THRESHOLD,
|
|
|
+ YarnConfiguration.
|
|
|
+ DEFAULT_YARN_DISPATCHER_PRINT_EVENTS_INFO_THRESHOLD);
|
|
|
+ }
|
|
|
+
|
|
|
@Override
|
|
|
protected void serviceStart() throws Exception {
|
|
|
//start all the components
|
|
@@ -246,6 +268,17 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
|
|
|
}
|
|
|
|
|
|
class GenericEventHandler implements EventHandler<Event> {
|
|
|
+ private void printEventQueueDetails(BlockingQueue<Event> queue) {
|
|
|
+ Map<Enum, Long> counterMap = eventQueue.stream().
|
|
|
+ collect(Collectors.
|
|
|
+ groupingBy(e -> e.getType(), Collectors.counting())
|
|
|
+ );
|
|
|
+ for (Map.Entry<Enum, Long> entry : counterMap.entrySet()) {
|
|
|
+ long num = entry.getValue();
|
|
|
+ LOG.info("Event type: " + entry.getKey()
|
|
|
+ + ", Event record counter: " + num);
|
|
|
+ }
|
|
|
+ }
|
|
|
public void handle(Event event) {
|
|
|
if (blockNewEvents) {
|
|
|
return;
|
|
@@ -259,6 +292,12 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
|
|
|
lastEventQueueSizeLogged = qSize;
|
|
|
LOG.info("Size of event-queue is " + qSize);
|
|
|
}
|
|
|
+ if (qSize != 0 && qSize % detailsInterval == 0
|
|
|
+ && lastEventDetailsQueueSizeLogged != qSize) {
|
|
|
+ lastEventDetailsQueueSizeLogged = qSize;
|
|
|
+ printEventQueueDetails(eventQueue);
|
|
|
+ printTrigger = true;
|
|
|
+ }
|
|
|
int remCapacity = eventQueue.remainingCapacity();
|
|
|
if (remCapacity < 1000) {
|
|
|
LOG.warn("Very low remaining capacity in the event-queue: "
|