YARN-4722. AsyncDispatcher logs redundant event queue sizes (Jason Lowe via sjlee)

This commit is contained in:
Sangjin Lee 2016-02-24 09:29:41 -08:00
parent 3369a4f691
commit 553b591ba0
3 changed files with 11 additions and 2 deletions

View File

@ -2396,6 +2396,9 @@ Release 2.6.5 - UNRELEASED
YARN-2046. Out of band heartbeats are sent only on container kill and
possibly too early (Ming Ma via jlowe)
YARN-4722. AsyncDispatcher logs redundant event queue sizes (Jason Lowe via
sjlee)
Release 2.6.4 - 2016-02-11
INCOMPATIBLE CHANGES

View File

@ -50,6 +50,7 @@ public class AsyncDispatcher extends AbstractService implements Dispatcher {
private static final Log LOG = LogFactory.getLog(AsyncDispatcher.class);
private final BlockingQueue<Event> eventQueue;
private volatile int lastEventQueueSizeLogged = 0;
private volatile boolean stopped = false;
// Configuration flag for enabling/disabling draining dispatcher's events on
@ -236,7 +237,9 @@ public void handle(Event event) {
/* all this method does is enqueue all the events onto the queue */
int qSize = eventQueue.size();
if (qSize !=0 && qSize %1000 == 0) {
if (qSize != 0 && qSize % 1000 == 0
&& lastEventQueueSizeLogged != qSize) {
lastEventQueueSizeLogged = qSize;
LOG.info("Size of event-queue is " + qSize);
}
int remCapacity = eventQueue.remainingCapacity();

View File

@ -727,6 +727,7 @@ public static class SchedulerEventDispatcher extends AbstractService
private final ResourceScheduler scheduler;
private final BlockingQueue<SchedulerEvent> eventQueue =
new LinkedBlockingQueue<SchedulerEvent>();
private volatile int lastEventQueueSizeLogged = 0;
private final Thread eventProcessor;
private volatile boolean stopped = false;
private boolean shouldExitOnError = false;
@ -804,7 +805,9 @@ protected void serviceStop() throws Exception {
public void handle(SchedulerEvent event) {
try {
int qSize = eventQueue.size();
if (qSize !=0 && qSize %1000 == 0) {
if (qSize != 0 && qSize % 1000 == 0
&& lastEventQueueSizeLogged != qSize) {
lastEventQueueSizeLogged = qSize;
LOG.info("Size of scheduler event-queue is " + qSize);
}
int remCapacity = eventQueue.remainingCapacity();