|
@@ -21,11 +21,14 @@ package org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity;
|
|
import java.io.IOException;
|
|
import java.io.IOException;
|
|
import java.io.InputStream;
|
|
import java.io.InputStream;
|
|
import java.util.ArrayList;
|
|
import java.util.ArrayList;
|
|
|
|
+import java.util.Collection;
|
|
import java.util.Comparator;
|
|
import java.util.Comparator;
|
|
import java.util.HashMap;
|
|
import java.util.HashMap;
|
|
import java.util.List;
|
|
import java.util.List;
|
|
import java.util.Map;
|
|
import java.util.Map;
|
|
|
|
+import java.util.Random;
|
|
import java.util.concurrent.ConcurrentHashMap;
|
|
import java.util.concurrent.ConcurrentHashMap;
|
|
|
|
+import java.util.concurrent.atomic.AtomicBoolean;
|
|
|
|
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.Log;
|
|
import org.apache.commons.logging.LogFactory;
|
|
import org.apache.commons.logging.LogFactory;
|
|
@@ -194,6 +197,18 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
private ResourceCalculator calculator;
|
|
private ResourceCalculator calculator;
|
|
private boolean usePortForNodeName;
|
|
private boolean usePortForNodeName;
|
|
|
|
|
|
|
|
+ private boolean scheduleAsynchronously;
|
|
|
|
+ private AsyncScheduleThread asyncSchedulerThread;
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * EXPERT
|
|
|
|
+ */
|
|
|
|
+ private long asyncScheduleInterval;
|
|
|
|
+ private static final String ASYNC_SCHEDULER_INTERVAL =
|
|
|
|
+ CapacitySchedulerConfiguration.SCHEDULE_ASYNCHRONOUSLY_PREFIX
|
|
|
|
+ + ".scheduling-interval-ms";
|
|
|
|
+ private static final long DEFAULT_ASYNC_SCHEDULER_INTERVAL = 5;
|
|
|
|
+
|
|
public CapacityScheduler() {}
|
|
public CapacityScheduler() {}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
@@ -272,11 +287,23 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
|
|
|
|
initializeQueues(this.conf);
|
|
initializeQueues(this.conf);
|
|
|
|
|
|
|
|
+ scheduleAsynchronously = this.conf.getScheduleAynschronously();
|
|
|
|
+ asyncScheduleInterval =
|
|
|
|
+ this.conf.getLong(ASYNC_SCHEDULER_INTERVAL,
|
|
|
|
+ DEFAULT_ASYNC_SCHEDULER_INTERVAL);
|
|
|
|
+ if (scheduleAsynchronously) {
|
|
|
|
+ asyncSchedulerThread = new AsyncScheduleThread(this);
|
|
|
|
+ asyncSchedulerThread.start();
|
|
|
|
+ }
|
|
|
|
+
|
|
initialized = true;
|
|
initialized = true;
|
|
LOG.info("Initialized CapacityScheduler with " +
|
|
LOG.info("Initialized CapacityScheduler with " +
|
|
"calculator=" + getResourceCalculator().getClass() + ", " +
|
|
"calculator=" + getResourceCalculator().getClass() + ", " +
|
|
"minimumAllocation=<" + getMinimumResourceCapability() + ">, " +
|
|
"minimumAllocation=<" + getMinimumResourceCapability() + ">, " +
|
|
- "maximumAllocation=<" + getMaximumResourceCapability() + ">");
|
|
|
|
|
|
+ "maximumAllocation=<" + getMaximumResourceCapability() + ">, " +
|
|
|
|
+ "asynchronousScheduling=" + scheduleAsynchronously + ", " +
|
|
|
|
+ "asyncScheduleInterval=" + asyncScheduleInterval + "ms");
|
|
|
|
+
|
|
} else {
|
|
} else {
|
|
CapacitySchedulerConfiguration oldConf = this.conf;
|
|
CapacitySchedulerConfiguration oldConf = this.conf;
|
|
this.conf = loadCapacitySchedulerConfiguration(configuration);
|
|
this.conf = loadCapacitySchedulerConfiguration(configuration);
|
|
@@ -290,7 +317,69 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
+
|
|
|
|
+ long getAsyncScheduleInterval() {
|
|
|
|
+ return asyncScheduleInterval;
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private final static Random random = new Random(System.currentTimeMillis());
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Schedule on all nodes by starting at a random point.
|
|
|
|
+ * @param cs
|
|
|
|
+ */
|
|
|
|
+ static void schedule(CapacityScheduler cs) {
|
|
|
|
+ // First randomize the start point
|
|
|
|
+ int current = 0;
|
|
|
|
+ Collection<FiCaSchedulerNode> nodes = cs.getAllNodes().values();
|
|
|
|
+ int start = random.nextInt(nodes.size());
|
|
|
|
+ for (FiCaSchedulerNode node : nodes) {
|
|
|
|
+ if (current++ >= start) {
|
|
|
|
+ cs.allocateContainersToNode(node);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ // Now, just get everyone to be safe
|
|
|
|
+ for (FiCaSchedulerNode node : nodes) {
|
|
|
|
+ cs.allocateContainersToNode(node);
|
|
|
|
+ }
|
|
|
|
+ try {
|
|
|
|
+ Thread.sleep(cs.getAsyncScheduleInterval());
|
|
|
|
+ } catch (InterruptedException e) {}
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ static class AsyncScheduleThread extends Thread {
|
|
|
|
+
|
|
|
|
+ private final CapacityScheduler cs;
|
|
|
|
+ private AtomicBoolean runSchedules = new AtomicBoolean(false);
|
|
|
|
+
|
|
|
|
+ public AsyncScheduleThread(CapacityScheduler cs) {
|
|
|
|
+ this.cs = cs;
|
|
|
|
+ setDaemon(true);
|
|
|
|
+ }
|
|
|
|
|
|
|
|
+ @Override
|
|
|
|
+ public void run() {
|
|
|
|
+ while (true) {
|
|
|
|
+ if (!runSchedules.get()) {
|
|
|
|
+ try {
|
|
|
|
+ Thread.sleep(100);
|
|
|
|
+ } catch (InterruptedException ie) {}
|
|
|
|
+ } else {
|
|
|
|
+ schedule(cs);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public void beginSchedule() {
|
|
|
|
+ runSchedules.set(true);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public void suspendSchedule() {
|
|
|
|
+ runSchedules.set(false);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ }
|
|
|
|
+
|
|
@Private
|
|
@Private
|
|
public static final String ROOT_QUEUE =
|
|
public static final String ROOT_QUEUE =
|
|
CapacitySchedulerConfiguration.PREFIX + CapacitySchedulerConfiguration.ROOT;
|
|
CapacitySchedulerConfiguration.PREFIX + CapacitySchedulerConfiguration.ROOT;
|
|
@@ -696,6 +785,9 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
LOG.debug("Node being looked for scheduling " + nm
|
|
LOG.debug("Node being looked for scheduling " + nm
|
|
+ " availableResource: " + node.getAvailableResource());
|
|
+ " availableResource: " + node.getAvailableResource());
|
|
}
|
|
}
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private synchronized void allocateContainersToNode(FiCaSchedulerNode node) {
|
|
|
|
|
|
// Assign new containers...
|
|
// Assign new containers...
|
|
// 1. Check for reserved applications
|
|
// 1. Check for reserved applications
|
|
@@ -708,7 +800,8 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
|
|
|
|
// Try to fulfill the reservation
|
|
// Try to fulfill the reservation
|
|
LOG.info("Trying to fulfill reservation for application " +
|
|
LOG.info("Trying to fulfill reservation for application " +
|
|
- reservedApplication.getApplicationId() + " on node: " + nm);
|
|
|
|
|
|
+ reservedApplication.getApplicationId() + " on node: " +
|
|
|
|
+ node.getNodeID());
|
|
|
|
|
|
LeafQueue queue = ((LeafQueue)reservedApplication.getQueue());
|
|
LeafQueue queue = ((LeafQueue)reservedApplication.getQueue());
|
|
CSAssignment assignment = queue.assignContainers(clusterResource, node);
|
|
CSAssignment assignment = queue.assignContainers(clusterResource, node);
|
|
@@ -729,9 +822,16 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
|
|
|
|
// Try to schedule more if there are no reservations to fulfill
|
|
// Try to schedule more if there are no reservations to fulfill
|
|
if (node.getReservedContainer() == null) {
|
|
if (node.getReservedContainer() == null) {
|
|
- root.assignContainers(clusterResource, node);
|
|
|
|
|
|
+ if (Resources.greaterThanOrEqual(calculator, getClusterResources(),
|
|
|
|
+ node.getAvailableResource(), minimumAllocation)) {
|
|
|
|
+ if (LOG.isDebugEnabled()) {
|
|
|
|
+ LOG.debug("Trying to schedule on node: " + node.getNodeName() +
|
|
|
|
+ ", available: " + node.getAvailableResource());
|
|
|
|
+ }
|
|
|
|
+ root.assignContainers(clusterResource, node);
|
|
|
|
+ }
|
|
} else {
|
|
} else {
|
|
- LOG.info("Skipping scheduling since node " + nm +
|
|
|
|
|
|
+ LOG.info("Skipping scheduling since node " + node.getNodeID() +
|
|
" is reserved by application " +
|
|
" is reserved by application " +
|
|
node.getReservedContainer().getContainerId().getApplicationAttemptId()
|
|
node.getReservedContainer().getContainerId().getApplicationAttemptId()
|
|
);
|
|
);
|
|
@@ -772,7 +872,11 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
case NODE_UPDATE:
|
|
case NODE_UPDATE:
|
|
{
|
|
{
|
|
NodeUpdateSchedulerEvent nodeUpdatedEvent = (NodeUpdateSchedulerEvent)event;
|
|
NodeUpdateSchedulerEvent nodeUpdatedEvent = (NodeUpdateSchedulerEvent)event;
|
|
- nodeUpdate(nodeUpdatedEvent.getRMNode());
|
|
|
|
|
|
+ RMNode node = nodeUpdatedEvent.getRMNode();
|
|
|
|
+ nodeUpdate(node);
|
|
|
|
+ if (!scheduleAsynchronously) {
|
|
|
|
+ allocateContainersToNode(getNode(node.getNodeID()));
|
|
|
|
+ }
|
|
}
|
|
}
|
|
break;
|
|
break;
|
|
case APP_ADDED:
|
|
case APP_ADDED:
|
|
@@ -831,6 +935,10 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
++numNodeManagers;
|
|
++numNodeManagers;
|
|
LOG.info("Added node " + nodeManager.getNodeAddress() +
|
|
LOG.info("Added node " + nodeManager.getNodeAddress() +
|
|
" clusterResource: " + clusterResource);
|
|
" clusterResource: " + clusterResource);
|
|
|
|
+
|
|
|
|
+ if (scheduleAsynchronously && numNodeManagers == 1) {
|
|
|
|
+ asyncSchedulerThread.beginSchedule();
|
|
|
|
+ }
|
|
}
|
|
}
|
|
|
|
|
|
private synchronized void removeNode(RMNode nodeInfo) {
|
|
private synchronized void removeNode(RMNode nodeInfo) {
|
|
@@ -842,6 +950,10 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
root.updateClusterResource(clusterResource);
|
|
root.updateClusterResource(clusterResource);
|
|
--numNodeManagers;
|
|
--numNodeManagers;
|
|
|
|
|
|
|
|
+ if (scheduleAsynchronously && numNodeManagers == 0) {
|
|
|
|
+ asyncSchedulerThread.suspendSchedule();
|
|
|
|
+ }
|
|
|
|
+
|
|
// Remove running containers
|
|
// Remove running containers
|
|
List<RMContainer> runningContainers = node.getRunningContainers();
|
|
List<RMContainer> runningContainers = node.getRunningContainers();
|
|
for (RMContainer container : runningContainers) {
|
|
for (RMContainer container : runningContainers) {
|
|
@@ -931,7 +1043,12 @@ public class CapacityScheduler extends AbstractYarnScheduler
|
|
FiCaSchedulerNode getNode(NodeId nodeId) {
|
|
FiCaSchedulerNode getNode(NodeId nodeId) {
|
|
return nodes.get(nodeId);
|
|
return nodes.get(nodeId);
|
|
}
|
|
}
|
|
-
|
|
|
|
|
|
+
|
|
|
|
+ @Lock(Lock.NoLock.class)
|
|
|
|
+ Map<NodeId, FiCaSchedulerNode> getAllNodes() {
|
|
|
|
+ return nodes;
|
|
|
|
+ }
|
|
|
|
+
|
|
@Override
|
|
@Override
|
|
public RMContainer getRMContainer(ContainerId containerId) {
|
|
public RMContainer getRMContainer(ContainerId containerId) {
|
|
FiCaSchedulerApp attempt = getCurrentAttemptForContainer(containerId);
|
|
FiCaSchedulerApp attempt = getCurrentAttemptForContainer(containerId);
|