|
@@ -30,7 +30,6 @@ import java.util.NavigableSet;
|
|
import java.util.PriorityQueue;
|
|
import java.util.PriorityQueue;
|
|
import java.util.Set;
|
|
import java.util.Set;
|
|
|
|
|
|
-import org.apache.commons.collections.map.HashedMap;
|
|
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
@@ -38,19 +37,18 @@ import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
|
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
import org.apache.hadoop.yarn.api.records.Priority;
|
|
import org.apache.hadoop.yarn.api.records.Priority;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
-import org.apache.hadoop.yarn.event.EventHandler;
|
|
|
|
import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
|
|
import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.monitor.SchedulingEditPolicy;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.monitor.SchedulingEditPolicy;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.nodelabels.RMNodeLabelsManager;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.nodelabels.RMNodeLabelsManager;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ContainerPreemptEvent;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ContainerPreemptEvent;
|
|
-import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ContainerPreemptEventType;
|
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.PreemptableResourceScheduler;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.PreemptableResourceScheduler;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CSQueue;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CSQueue;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacityScheduler;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacityScheduler;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.LeafQueue;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.LeafQueue;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaSchedulerApp;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaSchedulerApp;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
|
import org.apache.hadoop.yarn.util.Clock;
|
|
import org.apache.hadoop.yarn.util.Clock;
|
|
import org.apache.hadoop.yarn.util.SystemClock;
|
|
import org.apache.hadoop.yarn.util.SystemClock;
|
|
import org.apache.hadoop.yarn.util.resource.ResourceCalculator;
|
|
import org.apache.hadoop.yarn.util.resource.ResourceCalculator;
|
|
@@ -116,8 +114,7 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
public static final String NATURAL_TERMINATION_FACTOR =
|
|
public static final String NATURAL_TERMINATION_FACTOR =
|
|
"yarn.resourcemanager.monitor.capacity.preemption.natural_termination_factor";
|
|
"yarn.resourcemanager.monitor.capacity.preemption.natural_termination_factor";
|
|
|
|
|
|
- // the dispatcher to send preempt and kill events
|
|
|
|
- public EventHandler<ContainerPreemptEvent> dispatcher;
|
|
|
|
|
|
+ private RMContext rmContext;
|
|
|
|
|
|
private final Clock clock;
|
|
private final Clock clock;
|
|
private double maxIgnoredOverCapacity;
|
|
private double maxIgnoredOverCapacity;
|
|
@@ -137,20 +134,17 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
}
|
|
}
|
|
|
|
|
|
public ProportionalCapacityPreemptionPolicy(Configuration config,
|
|
public ProportionalCapacityPreemptionPolicy(Configuration config,
|
|
- EventHandler<ContainerPreemptEvent> dispatcher,
|
|
|
|
- CapacityScheduler scheduler) {
|
|
|
|
- this(config, dispatcher, scheduler, new SystemClock());
|
|
|
|
|
|
+ RMContext context, CapacityScheduler scheduler) {
|
|
|
|
+ this(config, context, scheduler, new SystemClock());
|
|
}
|
|
}
|
|
|
|
|
|
public ProportionalCapacityPreemptionPolicy(Configuration config,
|
|
public ProportionalCapacityPreemptionPolicy(Configuration config,
|
|
- EventHandler<ContainerPreemptEvent> dispatcher,
|
|
|
|
- CapacityScheduler scheduler, Clock clock) {
|
|
|
|
- init(config, dispatcher, scheduler);
|
|
|
|
|
|
+ RMContext context, CapacityScheduler scheduler, Clock clock) {
|
|
|
|
+ init(config, context, scheduler);
|
|
this.clock = clock;
|
|
this.clock = clock;
|
|
}
|
|
}
|
|
|
|
|
|
- public void init(Configuration config,
|
|
|
|
- EventHandler<ContainerPreemptEvent> disp,
|
|
|
|
|
|
+ public void init(Configuration config, RMContext context,
|
|
PreemptableResourceScheduler sched) {
|
|
PreemptableResourceScheduler sched) {
|
|
LOG.info("Preemption monitor:" + this.getClass().getCanonicalName());
|
|
LOG.info("Preemption monitor:" + this.getClass().getCanonicalName());
|
|
assert null == scheduler : "Unexpected duplicate call to init";
|
|
assert null == scheduler : "Unexpected duplicate call to init";
|
|
@@ -159,7 +153,7 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
sched.getClass().getCanonicalName() + " not instance of " +
|
|
sched.getClass().getCanonicalName() + " not instance of " +
|
|
CapacityScheduler.class.getCanonicalName());
|
|
CapacityScheduler.class.getCanonicalName());
|
|
}
|
|
}
|
|
- dispatcher = disp;
|
|
|
|
|
|
+ rmContext = context;
|
|
scheduler = (CapacityScheduler) sched;
|
|
scheduler = (CapacityScheduler) sched;
|
|
maxIgnoredOverCapacity = config.getDouble(MAX_IGNORED_OVER_CAPACITY, 0.1);
|
|
maxIgnoredOverCapacity = config.getDouble(MAX_IGNORED_OVER_CAPACITY, 0.1);
|
|
naturalTerminationFactor =
|
|
naturalTerminationFactor =
|
|
@@ -218,6 +212,7 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
* @param root the root of the CapacityScheduler queue hierarchy
|
|
* @param root the root of the CapacityScheduler queue hierarchy
|
|
* @param clusterResources the total amount of resources in the cluster
|
|
* @param clusterResources the total amount of resources in the cluster
|
|
*/
|
|
*/
|
|
|
|
+ @SuppressWarnings("unchecked")
|
|
private void containerBasedPreemptOrKill(CSQueue root,
|
|
private void containerBasedPreemptOrKill(CSQueue root,
|
|
Resource clusterResources) {
|
|
Resource clusterResources) {
|
|
|
|
|
|
@@ -252,18 +247,21 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
// preempt (or kill) the selected containers
|
|
// preempt (or kill) the selected containers
|
|
for (Map.Entry<ApplicationAttemptId,Set<RMContainer>> e
|
|
for (Map.Entry<ApplicationAttemptId,Set<RMContainer>> e
|
|
: toPreempt.entrySet()) {
|
|
: toPreempt.entrySet()) {
|
|
|
|
+ ApplicationAttemptId appAttemptId = e.getKey();
|
|
for (RMContainer container : e.getValue()) {
|
|
for (RMContainer container : e.getValue()) {
|
|
// if we tried to preempt this for more than maxWaitTime
|
|
// if we tried to preempt this for more than maxWaitTime
|
|
if (preempted.get(container) != null &&
|
|
if (preempted.get(container) != null &&
|
|
preempted.get(container) + maxWaitTime < clock.getTime()) {
|
|
preempted.get(container) + maxWaitTime < clock.getTime()) {
|
|
// kill it
|
|
// kill it
|
|
- dispatcher.handle(new ContainerPreemptEvent(e.getKey(), container,
|
|
|
|
- ContainerPreemptEventType.KILL_CONTAINER));
|
|
|
|
|
|
+ rmContext.getDispatcher().getEventHandler().handle(
|
|
|
|
+ new ContainerPreemptEvent(appAttemptId, container,
|
|
|
|
+ SchedulerEventType.KILL_CONTAINER));
|
|
preempted.remove(container);
|
|
preempted.remove(container);
|
|
} else {
|
|
} else {
|
|
//otherwise just send preemption events
|
|
//otherwise just send preemption events
|
|
- dispatcher.handle(new ContainerPreemptEvent(e.getKey(), container,
|
|
|
|
- ContainerPreemptEventType.PREEMPT_CONTAINER));
|
|
|
|
|
|
+ rmContext.getDispatcher().getEventHandler().handle(
|
|
|
|
+ new ContainerPreemptEvent(appAttemptId, container,
|
|
|
|
+ SchedulerEventType.PREEMPT_CONTAINER));
|
|
if (preempted.get(container) == null) {
|
|
if (preempted.get(container) == null) {
|
|
preempted.put(container, clock.getTime());
|
|
preempted.put(container, clock.getTime());
|
|
}
|
|
}
|
|
@@ -634,6 +632,7 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
* @param rsrcPreempt
|
|
* @param rsrcPreempt
|
|
* @return Set<RMContainer> Set of RMContainers
|
|
* @return Set<RMContainer> Set of RMContainers
|
|
*/
|
|
*/
|
|
|
|
+ @SuppressWarnings("unchecked")
|
|
private Set<RMContainer> preemptFrom(FiCaSchedulerApp app,
|
|
private Set<RMContainer> preemptFrom(FiCaSchedulerApp app,
|
|
Resource clusterResource, Resource rsrcPreempt,
|
|
Resource clusterResource, Resource rsrcPreempt,
|
|
List<RMContainer> skippedAMContainerlist, Resource skippedAMSize) {
|
|
List<RMContainer> skippedAMContainerlist, Resource skippedAMSize) {
|
|
@@ -649,8 +648,9 @@ public class ProportionalCapacityPreemptionPolicy implements SchedulingEditPolic
|
|
return ret;
|
|
return ret;
|
|
}
|
|
}
|
|
if (!observeOnly) {
|
|
if (!observeOnly) {
|
|
- dispatcher.handle(new ContainerPreemptEvent(appId, c,
|
|
|
|
- ContainerPreemptEventType.DROP_RESERVATION));
|
|
|
|
|
|
+ rmContext.getDispatcher().getEventHandler().handle(
|
|
|
|
+ new ContainerPreemptEvent(
|
|
|
|
+ appId, c, SchedulerEventType.DROP_RESERVATION));
|
|
}
|
|
}
|
|
Resources.subtractFrom(rsrcPreempt, c.getContainer().getResource());
|
|
Resources.subtractFrom(rsrcPreempt, c.getContainer().getResource());
|
|
}
|
|
}
|