|
@@ -17,10 +17,17 @@
|
|
|
*/
|
|
|
package org.apache.hadoop.yarn.server.resourcemanager.monitor;
|
|
|
|
|
|
+import java.util.concurrent.Executors;
|
|
|
+import java.util.concurrent.ScheduledExecutorService;
|
|
|
+import java.util.concurrent.ScheduledFuture;
|
|
|
+import java.util.concurrent.ThreadFactory;
|
|
|
+import java.util.concurrent.TimeUnit;
|
|
|
+
|
|
|
import org.apache.commons.logging.Log;
|
|
|
import org.apache.commons.logging.LogFactory;
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.service.AbstractService;
|
|
|
+import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.PreemptableResourceScheduler;
|
|
|
|
|
@@ -31,9 +38,10 @@ public class SchedulingMonitor extends AbstractService {
|
|
|
private final SchedulingEditPolicy scheduleEditPolicy;
|
|
|
private static final Log LOG = LogFactory.getLog(SchedulingMonitor.class);
|
|
|
|
|
|
- //thread which runs periodically to see the last time since a heartbeat is
|
|
|
- //received.
|
|
|
- private Thread checkerThread;
|
|
|
+ // ScheduledExecutorService which schedules the PreemptionChecker to run
|
|
|
+ // periodically.
|
|
|
+ private ScheduledExecutorService ses;
|
|
|
+ private ScheduledFuture<?> handler;
|
|
|
private volatile boolean stopped;
|
|
|
private long monitorInterval;
|
|
|
private RMContext rmContext;
|
|
@@ -64,17 +72,25 @@ public class SchedulingMonitor extends AbstractService {
|
|
|
@Override
|
|
|
public void serviceStart() throws Exception {
|
|
|
assert !stopped : "starting when already stopped";
|
|
|
- checkerThread = new Thread(new PreemptionChecker());
|
|
|
- checkerThread.setName(getName());
|
|
|
- checkerThread.start();
|
|
|
+ ses = Executors.newSingleThreadScheduledExecutor(new ThreadFactory() {
|
|
|
+ public Thread newThread(Runnable r) {
|
|
|
+ Thread t = new Thread(r);
|
|
|
+ t.setName(getName());
|
|
|
+ return t;
|
|
|
+ }
|
|
|
+ });
|
|
|
+ handler = ses.scheduleAtFixedRate(new PreemptionChecker(),
|
|
|
+ 0, monitorInterval, TimeUnit.MILLISECONDS);
|
|
|
super.serviceStart();
|
|
|
}
|
|
|
|
|
|
@Override
|
|
|
public void serviceStop() throws Exception {
|
|
|
stopped = true;
|
|
|
- if (checkerThread != null) {
|
|
|
- checkerThread.interrupt();
|
|
|
+ if (handler != null) {
|
|
|
+ LOG.info("Stop " + getName());
|
|
|
+ handler.cancel(true);
|
|
|
+ ses.shutdown();
|
|
|
}
|
|
|
super.serviceStop();
|
|
|
}
|
|
@@ -87,17 +103,12 @@ public class SchedulingMonitor extends AbstractService {
|
|
|
private class PreemptionChecker implements Runnable {
|
|
|
@Override
|
|
|
public void run() {
|
|
|
- while (!stopped && !Thread.currentThread().isInterrupted()) {
|
|
|
- //invoke the preemption policy at a regular pace
|
|
|
- //the policy will generate preemption or kill events
|
|
|
- //managed by the dispatcher
|
|
|
+ try {
|
|
|
+ //invoke the preemption policy
|
|
|
invokePolicy();
|
|
|
- try {
|
|
|
- Thread.sleep(monitorInterval);
|
|
|
- } catch (InterruptedException e) {
|
|
|
- LOG.info(getName() + " thread interrupted");
|
|
|
- break;
|
|
|
- }
|
|
|
+ } catch (YarnRuntimeException e) {
|
|
|
+ LOG.error("YarnRuntimeException raised while executing preemption"
|
|
|
+ + " checker, skip this run..., exception=", e);
|
|
|
}
|
|
|
}
|
|
|
}
|