|
@@ -23,12 +23,14 @@ import java.util.ArrayList;
|
|
import java.util.Collection;
|
|
import java.util.Collection;
|
|
import java.util.Comparator;
|
|
import java.util.Comparator;
|
|
import java.util.HashMap;
|
|
import java.util.HashMap;
|
|
|
|
+import java.util.HashSet;
|
|
import java.util.Iterator;
|
|
import java.util.Iterator;
|
|
import java.util.List;
|
|
import java.util.List;
|
|
import java.util.Map;
|
|
import java.util.Map;
|
|
import java.util.Set;
|
|
import java.util.Set;
|
|
import java.util.TreeSet;
|
|
import java.util.TreeSet;
|
|
|
|
|
|
|
|
+import org.apache.commons.lang.StringUtils;
|
|
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.classification.InterfaceAudience.Private;
|
|
import org.apache.hadoop.classification.InterfaceAudience.Private;
|
|
@@ -46,77 +48,42 @@ import org.apache.hadoop.yarn.api.records.QueueUserACLInfo;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
import org.apache.hadoop.yarn.factories.RecordFactory;
|
|
import org.apache.hadoop.yarn.factories.RecordFactory;
|
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
|
|
|
+import org.apache.hadoop.yarn.nodelabels.CommonNodeLabelsManager;
|
|
|
|
+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.rmcontainer.RMContainerEventType;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainerEventType;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainerState;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainerState;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ActiveUsersManager;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ActiveUsersManager;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.NodeType;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.NodeType;
|
|
-import org.apache.hadoop.yarn.server.resourcemanager.scheduler.QueueMetrics;
|
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerApplicationAttempt;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerApplicationAttempt;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerUtils;
|
|
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.common.fica.FiCaSchedulerNode;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaSchedulerNode;
|
|
-import org.apache.hadoop.yarn.util.resource.ResourceCalculator;
|
|
|
|
import org.apache.hadoop.yarn.util.resource.Resources;
|
|
import org.apache.hadoop.yarn.util.resource.Resources;
|
|
|
|
|
|
|
|
+import com.google.common.collect.Sets;
|
|
|
|
+
|
|
@Private
|
|
@Private
|
|
@Evolving
|
|
@Evolving
|
|
-public class ParentQueue implements CSQueue {
|
|
|
|
|
|
+public class ParentQueue extends AbstractCSQueue {
|
|
|
|
|
|
private static final Log LOG = LogFactory.getLog(ParentQueue.class);
|
|
private static final Log LOG = LogFactory.getLog(ParentQueue.class);
|
|
|
|
|
|
- private CSQueue parent;
|
|
|
|
- private final String queueName;
|
|
|
|
-
|
|
|
|
- private float capacity;
|
|
|
|
- private float maximumCapacity;
|
|
|
|
- private float absoluteCapacity;
|
|
|
|
- private float absoluteMaxCapacity;
|
|
|
|
- private float absoluteUsedCapacity = 0.0f;
|
|
|
|
-
|
|
|
|
- private float usedCapacity = 0.0f;
|
|
|
|
-
|
|
|
|
- protected final Set<CSQueue> childQueues;
|
|
|
|
- private final Comparator<CSQueue> queueComparator;
|
|
|
|
-
|
|
|
|
- private Resource usedResources = Resources.createResource(0, 0);
|
|
|
|
-
|
|
|
|
|
|
+ protected final Set<CSQueue> childQueues;
|
|
private final boolean rootQueue;
|
|
private final boolean rootQueue;
|
|
-
|
|
|
|
- private final Resource minimumAllocation;
|
|
|
|
-
|
|
|
|
- private volatile int numApplications;
|
|
|
|
- private volatile int numContainers;
|
|
|
|
-
|
|
|
|
- private QueueState state;
|
|
|
|
-
|
|
|
|
- private final QueueMetrics metrics;
|
|
|
|
-
|
|
|
|
- private QueueInfo queueInfo;
|
|
|
|
-
|
|
|
|
- private Map<QueueACL, AccessControlList> acls =
|
|
|
|
- new HashMap<QueueACL, AccessControlList>();
|
|
|
|
|
|
+ final Comparator<CSQueue> queueComparator;
|
|
|
|
+ volatile int numApplications;
|
|
|
|
|
|
private final RecordFactory recordFactory =
|
|
private final RecordFactory recordFactory =
|
|
RecordFactoryProvider.getRecordFactory(null);
|
|
RecordFactoryProvider.getRecordFactory(null);
|
|
|
|
|
|
- private final ResourceCalculator resourceCalculator;
|
|
|
|
-
|
|
|
|
- private boolean reservationsContinueLooking;
|
|
|
|
-
|
|
|
|
public ParentQueue(CapacitySchedulerContext cs,
|
|
public ParentQueue(CapacitySchedulerContext cs,
|
|
- String queueName, CSQueue parent, CSQueue old) {
|
|
|
|
- minimumAllocation = cs.getMinimumResourceCapability();
|
|
|
|
|
|
+ String queueName, CSQueue parent, CSQueue old) throws IOException {
|
|
|
|
+ super(cs, queueName, parent, old);
|
|
|
|
|
|
- this.parent = parent;
|
|
|
|
- this.queueName = queueName;
|
|
|
|
- this.rootQueue = (parent == null);
|
|
|
|
- this.resourceCalculator = cs.getResourceCalculator();
|
|
|
|
|
|
+ this.queueComparator = cs.getQueueComparator();
|
|
|
|
|
|
- // must be called after parent and queueName is set
|
|
|
|
- this.metrics = old != null ? old.getMetrics() :
|
|
|
|
- QueueMetrics.forQueue(getQueuePath(), parent,
|
|
|
|
- cs.getConfiguration().getEnableUserMetrics(),
|
|
|
|
- cs.getConf());
|
|
|
|
|
|
+ this.rootQueue = (parent == null);
|
|
|
|
|
|
float rawCapacity = cs.getConfiguration().getCapacity(getQueuePath());
|
|
float rawCapacity = cs.getConfiguration().getCapacity(getQueuePath());
|
|
|
|
|
|
@@ -141,17 +108,14 @@ public class ParentQueue implements CSQueue {
|
|
|
|
|
|
Map<QueueACL, AccessControlList> acls =
|
|
Map<QueueACL, AccessControlList> acls =
|
|
cs.getConfiguration().getAcls(getQueuePath());
|
|
cs.getConfiguration().getAcls(getQueuePath());
|
|
-
|
|
|
|
- this.queueInfo = recordFactory.newRecordInstance(QueueInfo.class);
|
|
|
|
- this.queueInfo.setQueueName(queueName);
|
|
|
|
|
|
+
|
|
this.queueInfo.setChildQueues(new ArrayList<QueueInfo>());
|
|
this.queueInfo.setChildQueues(new ArrayList<QueueInfo>());
|
|
|
|
|
|
- setupQueueConfigs(cs.getClusterResource(),
|
|
|
|
- capacity, absoluteCapacity,
|
|
|
|
- maximumCapacity, absoluteMaxCapacity, state, acls,
|
|
|
|
|
|
+ setupQueueConfigs(cs.getClusterResource(), capacity, absoluteCapacity,
|
|
|
|
+ maximumCapacity, absoluteMaxCapacity, state, acls, accessibleLabels,
|
|
|
|
+ defaultLabelExpression, capacitiyByNodeLabels, maxCapacityByNodeLabels,
|
|
cs.getConfiguration().getReservationContinueLook());
|
|
cs.getConfiguration().getReservationContinueLook());
|
|
|
|
|
|
- this.queueComparator = cs.getQueueComparator();
|
|
|
|
this.childQueues = new TreeSet<CSQueue>(queueComparator);
|
|
this.childQueues = new TreeSet<CSQueue>(queueComparator);
|
|
|
|
|
|
LOG.info("Initialized parent-queue " + queueName +
|
|
LOG.info("Initialized parent-queue " + queueName +
|
|
@@ -159,41 +123,29 @@ public class ParentQueue implements CSQueue {
|
|
", fullname=" + getQueuePath());
|
|
", fullname=" + getQueuePath());
|
|
}
|
|
}
|
|
|
|
|
|
- protected synchronized void setupQueueConfigs(
|
|
|
|
- Resource clusterResource,
|
|
|
|
- float capacity, float absoluteCapacity,
|
|
|
|
- float maximumCapacity, float absoluteMaxCapacity,
|
|
|
|
|
|
+ synchronized void setupQueueConfigs(Resource clusterResource, float capacity,
|
|
|
|
+ float absoluteCapacity, float maximumCapacity, float absoluteMaxCapacity,
|
|
QueueState state, Map<QueueACL, AccessControlList> acls,
|
|
QueueState state, Map<QueueACL, AccessControlList> acls,
|
|
- boolean continueLooking
|
|
|
|
- ) {
|
|
|
|
- // Sanity check
|
|
|
|
- CSQueueUtils.checkMaxCapacity(getQueueName(), capacity, maximumCapacity);
|
|
|
|
- CSQueueUtils.checkAbsoluteCapacities(getQueueName(), absoluteCapacity, absoluteMaxCapacity);
|
|
|
|
-
|
|
|
|
- this.capacity = capacity;
|
|
|
|
- this.absoluteCapacity = absoluteCapacity;
|
|
|
|
-
|
|
|
|
- this.maximumCapacity = maximumCapacity;
|
|
|
|
- this.absoluteMaxCapacity = absoluteMaxCapacity;
|
|
|
|
-
|
|
|
|
- this.state = state;
|
|
|
|
-
|
|
|
|
- this.acls = acls;
|
|
|
|
-
|
|
|
|
- this.queueInfo.setCapacity(this.capacity);
|
|
|
|
- this.queueInfo.setMaximumCapacity(this.maximumCapacity);
|
|
|
|
- this.queueInfo.setQueueState(this.state);
|
|
|
|
-
|
|
|
|
- this.reservationsContinueLooking = continueLooking;
|
|
|
|
-
|
|
|
|
- StringBuilder aclsString = new StringBuilder();
|
|
|
|
|
|
+ Set<String> accessibleLabels, String defaultLabelExpression,
|
|
|
|
+ Map<String, Float> nodeLabelCapacities,
|
|
|
|
+ Map<String, Float> maximumCapacitiesByLabel,
|
|
|
|
+ boolean reservationContinueLooking) throws IOException {
|
|
|
|
+ super.setupQueueConfigs(clusterResource, capacity, absoluteCapacity,
|
|
|
|
+ maximumCapacity, absoluteMaxCapacity, state, acls, accessibleLabels,
|
|
|
|
+ defaultLabelExpression, nodeLabelCapacities, maximumCapacitiesByLabel,
|
|
|
|
+ reservationContinueLooking);
|
|
|
|
+ StringBuilder aclsString = new StringBuilder();
|
|
for (Map.Entry<QueueACL, AccessControlList> e : acls.entrySet()) {
|
|
for (Map.Entry<QueueACL, AccessControlList> e : acls.entrySet()) {
|
|
aclsString.append(e.getKey() + ":" + e.getValue().getAclString());
|
|
aclsString.append(e.getKey() + ":" + e.getValue().getAclString());
|
|
}
|
|
}
|
|
|
|
|
|
- // Update metrics
|
|
|
|
- CSQueueUtils.updateQueueStatistics(
|
|
|
|
- resourceCalculator, this, parent, clusterResource, minimumAllocation);
|
|
|
|
|
|
+ StringBuilder labelStrBuilder = new StringBuilder();
|
|
|
|
+ if (accessibleLabels != null) {
|
|
|
|
+ for (String s : accessibleLabels) {
|
|
|
|
+ labelStrBuilder.append(s);
|
|
|
|
+ labelStrBuilder.append(",");
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
|
|
LOG.info(queueName +
|
|
LOG.info(queueName +
|
|
", capacity=" + capacity +
|
|
", capacity=" + capacity +
|
|
@@ -201,13 +153,13 @@ public class ParentQueue implements CSQueue {
|
|
", maxCapacity=" + maximumCapacity +
|
|
", maxCapacity=" + maximumCapacity +
|
|
", asboluteMaxCapacity=" + absoluteMaxCapacity +
|
|
", asboluteMaxCapacity=" + absoluteMaxCapacity +
|
|
", state=" + state +
|
|
", state=" + state +
|
|
- ", acls=" + aclsString +
|
|
|
|
|
|
+ ", acls=" + aclsString +
|
|
|
|
+ ", labels=" + labelStrBuilder.toString() + "\n" +
|
|
", reservationsContinueLooking=" + reservationsContinueLooking);
|
|
", reservationsContinueLooking=" + reservationsContinueLooking);
|
|
}
|
|
}
|
|
|
|
|
|
private static float PRECISION = 0.0005f; // 0.05% precision
|
|
private static float PRECISION = 0.0005f; // 0.05% precision
|
|
void setChildQueues(Collection<CSQueue> childQueues) {
|
|
void setChildQueues(Collection<CSQueue> childQueues) {
|
|
-
|
|
|
|
// Validate
|
|
// Validate
|
|
float childCapacities = 0;
|
|
float childCapacities = 0;
|
|
for (CSQueue queue : childQueues) {
|
|
for (CSQueue queue : childQueues) {
|
|
@@ -221,6 +173,21 @@ public class ParentQueue implements CSQueue {
|
|
" capacity of " + childCapacities +
|
|
" capacity of " + childCapacities +
|
|
" for children of queue " + queueName);
|
|
" for children of queue " + queueName);
|
|
}
|
|
}
|
|
|
|
+ // check label capacities
|
|
|
|
+ for (String nodeLabel : labelManager.getClusterNodeLabels()) {
|
|
|
|
+ float capacityByLabel = getCapacityByNodeLabel(nodeLabel);
|
|
|
|
+ // check children's labels
|
|
|
|
+ float sum = 0;
|
|
|
|
+ for (CSQueue queue : childQueues) {
|
|
|
|
+ sum += queue.getCapacityByNodeLabel(nodeLabel);
|
|
|
|
+ }
|
|
|
|
+ if ((capacityByLabel > 0 && Math.abs(1.0f - sum) > PRECISION)
|
|
|
|
+ || (capacityByLabel == 0) && (sum > 0)) {
|
|
|
|
+ throw new IllegalArgumentException("Illegal" + " capacity of "
|
|
|
|
+ + sum + " for children of queue " + queueName
|
|
|
|
+ + " for label=" + nodeLabel);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
|
|
this.childQueues.clear();
|
|
this.childQueues.clear();
|
|
this.childQueues.addAll(childQueues);
|
|
this.childQueues.addAll(childQueues);
|
|
@@ -228,21 +195,6 @@ public class ParentQueue implements CSQueue {
|
|
LOG.debug("setChildQueues: " + getChildQueuesToPrint());
|
|
LOG.debug("setChildQueues: " + getChildQueuesToPrint());
|
|
}
|
|
}
|
|
}
|
|
}
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized CSQueue getParent() {
|
|
|
|
- return parent;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized void setParent(CSQueue newParentQueue) {
|
|
|
|
- this.parent = (ParentQueue)newParentQueue;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public String getQueueName() {
|
|
|
|
- return queueName;
|
|
|
|
- }
|
|
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public String getQueuePath() {
|
|
public String getQueuePath() {
|
|
@@ -250,65 +202,6 @@ public class ParentQueue implements CSQueue {
|
|
return parentPath + getQueueName();
|
|
return parentPath + getQueueName();
|
|
}
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
|
- public synchronized float getCapacity() {
|
|
|
|
- return capacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized float getAbsoluteCapacity() {
|
|
|
|
- return absoluteCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public float getAbsoluteMaximumCapacity() {
|
|
|
|
- return absoluteMaxCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized float getAbsoluteUsedCapacity() {
|
|
|
|
- return absoluteUsedCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public float getMaximumCapacity() {
|
|
|
|
- return maximumCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public ActiveUsersManager getActiveUsersManager() {
|
|
|
|
- // Should never be called since all applications are submitted to LeafQueues
|
|
|
|
- return null;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized float getUsedCapacity() {
|
|
|
|
- return usedCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized Resource getUsedResources() {
|
|
|
|
- return usedResources;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized List<CSQueue> getChildQueues() {
|
|
|
|
- return new ArrayList<CSQueue>(childQueues);
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- public synchronized int getNumContainers() {
|
|
|
|
- return numContainers;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- public synchronized int getNumApplications() {
|
|
|
|
- return numApplications;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized QueueState getState() {
|
|
|
|
- return state;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
@Override
|
|
@Override
|
|
public synchronized QueueInfo getQueueInfo(
|
|
public synchronized QueueInfo getQueueInfo(
|
|
boolean includeChildQueues, boolean recursive) {
|
|
boolean includeChildQueues, boolean recursive) {
|
|
@@ -391,6 +284,10 @@ public class ParentQueue implements CSQueue {
|
|
newlyParsedParentQueue.absoluteMaxCapacity,
|
|
newlyParsedParentQueue.absoluteMaxCapacity,
|
|
newlyParsedParentQueue.state,
|
|
newlyParsedParentQueue.state,
|
|
newlyParsedParentQueue.acls,
|
|
newlyParsedParentQueue.acls,
|
|
|
|
+ newlyParsedParentQueue.accessibleLabels,
|
|
|
|
+ newlyParsedParentQueue.defaultLabelExpression,
|
|
|
|
+ newlyParsedParentQueue.capacitiyByNodeLabels,
|
|
|
|
+ newlyParsedParentQueue.maxCapacityByNodeLabels,
|
|
newlyParsedParentQueue.reservationsContinueLooking);
|
|
newlyParsedParentQueue.reservationsContinueLooking);
|
|
|
|
|
|
// Re-configure existing child queues and add new ones
|
|
// Re-configure existing child queues and add new ones
|
|
@@ -434,21 +331,6 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
return queuesMap;
|
|
return queuesMap;
|
|
}
|
|
}
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public boolean hasAccess(QueueACL acl, UserGroupInformation user) {
|
|
|
|
- synchronized (this) {
|
|
|
|
- if (acls.get(acl).isUserAllowed(user)) {
|
|
|
|
- return true;
|
|
|
|
- }
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- if (parent != null) {
|
|
|
|
- return parent.hasAccess(acl, user);
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- return false;
|
|
|
|
- }
|
|
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public void submitApplication(ApplicationId applicationId, String user,
|
|
public void submitApplication(ApplicationId applicationId, String user,
|
|
@@ -521,7 +403,7 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- public synchronized void removeApplication(ApplicationId applicationId,
|
|
|
|
|
|
+ private synchronized void removeApplication(ApplicationId applicationId,
|
|
String user) {
|
|
String user) {
|
|
|
|
|
|
--numApplications;
|
|
--numApplications;
|
|
@@ -532,30 +414,6 @@ public class ParentQueue implements CSQueue {
|
|
" leaf-queue of parent: " + getQueueName() +
|
|
" leaf-queue of parent: " + getQueueName() +
|
|
" #applications: " + getNumApplications());
|
|
" #applications: " + getNumApplications());
|
|
}
|
|
}
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized void setUsedCapacity(float usedCapacity) {
|
|
|
|
- this.usedCapacity = usedCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- @Override
|
|
|
|
- public synchronized void setAbsoluteUsedCapacity(float absUsedCapacity) {
|
|
|
|
- this.absoluteUsedCapacity = absUsedCapacity;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- /**
|
|
|
|
- * Set maximum capacity - used only for testing.
|
|
|
|
- * @param maximumCapacity new max capacity
|
|
|
|
- */
|
|
|
|
- synchronized void setMaxCapacity(float maximumCapacity) {
|
|
|
|
- // Sanity check
|
|
|
|
- CSQueueUtils.checkMaxCapacity(getQueueName(), capacity, maximumCapacity);
|
|
|
|
- float absMaxCapacity = CSQueueUtils.computeAbsoluteMaximumCapacity(maximumCapacity, parent);
|
|
|
|
- CSQueueUtils.checkAbsoluteCapacities(getQueueName(), absoluteCapacity, absMaxCapacity);
|
|
|
|
-
|
|
|
|
- this.maximumCapacity = maximumCapacity;
|
|
|
|
- this.absoluteMaxCapacity = absMaxCapacity;
|
|
|
|
- }
|
|
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public synchronized CSAssignment assignContainers(
|
|
public synchronized CSAssignment assignContainers(
|
|
@@ -563,6 +421,12 @@ public class ParentQueue implements CSQueue {
|
|
CSAssignment assignment =
|
|
CSAssignment assignment =
|
|
new CSAssignment(Resources.createResource(0, 0), NodeType.NODE_LOCAL);
|
|
new CSAssignment(Resources.createResource(0, 0), NodeType.NODE_LOCAL);
|
|
|
|
|
|
|
|
+ // if our queue cannot access this node, just return
|
|
|
|
+ if (!SchedulerUtils.checkQueueAccessToNode(accessibleLabels,
|
|
|
|
+ labelManager.getLabelsOnNode(node.getNodeID()))) {
|
|
|
|
+ return assignment;
|
|
|
|
+ }
|
|
|
|
+
|
|
while (canAssign(clusterResource, node)) {
|
|
while (canAssign(clusterResource, node)) {
|
|
if (LOG.isDebugEnabled()) {
|
|
if (LOG.isDebugEnabled()) {
|
|
LOG.debug("Trying to assign containers to child-queue of "
|
|
LOG.debug("Trying to assign containers to child-queue of "
|
|
@@ -570,8 +434,10 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
|
|
|
|
boolean localNeedToUnreserve = false;
|
|
boolean localNeedToUnreserve = false;
|
|
|
|
+ Set<String> nodeLabels = labelManager.getLabelsOnNode(node.getNodeID());
|
|
|
|
+
|
|
// Are we over maximum-capacity for this queue?
|
|
// Are we over maximum-capacity for this queue?
|
|
- if (!assignToQueue(clusterResource)) {
|
|
|
|
|
|
+ if (!canAssignToThisQueue(clusterResource, nodeLabels)) {
|
|
// check to see if we could if we unreserve first
|
|
// check to see if we could if we unreserve first
|
|
localNeedToUnreserve = assignToQueueIfUnreserve(clusterResource);
|
|
localNeedToUnreserve = assignToQueueIfUnreserve(clusterResource);
|
|
if (!localNeedToUnreserve) {
|
|
if (!localNeedToUnreserve) {
|
|
@@ -589,7 +455,8 @@ public class ParentQueue implements CSQueue {
|
|
resourceCalculator, clusterResource,
|
|
resourceCalculator, clusterResource,
|
|
assignedToChild.getResource(), Resources.none())) {
|
|
assignedToChild.getResource(), Resources.none())) {
|
|
// Track resource utilization for the parent-queue
|
|
// Track resource utilization for the parent-queue
|
|
- allocateResource(clusterResource, assignedToChild.getResource());
|
|
|
|
|
|
+ super.allocateResource(clusterResource, assignedToChild.getResource(),
|
|
|
|
+ nodeLabels);
|
|
|
|
|
|
// Track resource utilization in this pass of the scheduler
|
|
// Track resource utilization in this pass of the scheduler
|
|
Resources.addTo(assignment.getResource(), assignedToChild.getResource());
|
|
Resources.addTo(assignment.getResource(), assignedToChild.getResource());
|
|
@@ -628,22 +495,41 @@ public class ParentQueue implements CSQueue {
|
|
return assignment;
|
|
return assignment;
|
|
}
|
|
}
|
|
|
|
|
|
- private synchronized boolean assignToQueue(Resource clusterResource) {
|
|
|
|
- // Check how of the cluster's absolute capacity we are currently using...
|
|
|
|
- float currentCapacity =
|
|
|
|
- Resources.divide(
|
|
|
|
- resourceCalculator, clusterResource,
|
|
|
|
- usedResources, clusterResource);
|
|
|
|
|
|
+ private synchronized boolean canAssignToThisQueue(Resource clusterResource,
|
|
|
|
+ Set<String> nodeLabels) {
|
|
|
|
+ Set<String> labelCanAccess =
|
|
|
|
+ new HashSet<String>(
|
|
|
|
+ accessibleLabels.contains(CommonNodeLabelsManager.ANY) ? nodeLabels
|
|
|
|
+ : Sets.intersection(accessibleLabels, nodeLabels));
|
|
|
|
+ if (nodeLabels.isEmpty()) {
|
|
|
|
+ // Any queue can always access any node without label
|
|
|
|
+ labelCanAccess.add(RMNodeLabelsManager.NO_LABEL);
|
|
|
|
+ }
|
|
|
|
|
|
- if (currentCapacity >= absoluteMaxCapacity) {
|
|
|
|
- LOG.info(getQueueName() +
|
|
|
|
- " used=" + usedResources +
|
|
|
|
- " current-capacity (" + currentCapacity + ") " +
|
|
|
|
- " >= max-capacity (" + absoluteMaxCapacity + ")");
|
|
|
|
- return false;
|
|
|
|
|
|
+ boolean canAssign = true;
|
|
|
|
+ for (String label : labelCanAccess) {
|
|
|
|
+ if (!usedResourcesByNodeLabels.containsKey(label)) {
|
|
|
|
+ usedResourcesByNodeLabels.put(label, Resources.createResource(0));
|
|
|
|
+ }
|
|
|
|
+ float currentAbsoluteLabelUsedCapacity =
|
|
|
|
+ Resources.divide(resourceCalculator, clusterResource,
|
|
|
|
+ usedResourcesByNodeLabels.get(label),
|
|
|
|
+ labelManager.getResourceByLabel(label, clusterResource));
|
|
|
|
+ // if any of the label doesn't beyond limit, we can allocate on this node
|
|
|
|
+ if (currentAbsoluteLabelUsedCapacity >=
|
|
|
|
+ getAbsoluteMaximumCapacityByNodeLabel(label)) {
|
|
|
|
+ if (LOG.isDebugEnabled()) {
|
|
|
|
+ LOG.debug(getQueueName() + " used=" + usedResources
|
|
|
|
+ + " current-capacity (" + usedResourcesByNodeLabels.get(label) + ") "
|
|
|
|
+ + " >= max-capacity ("
|
|
|
|
+ + labelManager.getResourceByLabel(label, clusterResource) + ")");
|
|
|
|
+ }
|
|
|
|
+ canAssign = false;
|
|
|
|
+ break;
|
|
|
|
+ }
|
|
}
|
|
}
|
|
- return true;
|
|
|
|
-
|
|
|
|
|
|
+
|
|
|
|
+ return canAssign;
|
|
}
|
|
}
|
|
|
|
|
|
|
|
|
|
@@ -685,7 +571,7 @@ public class ParentQueue implements CSQueue {
|
|
node.getAvailableResource(), minimumAllocation);
|
|
node.getAvailableResource(), minimumAllocation);
|
|
}
|
|
}
|
|
|
|
|
|
- synchronized CSAssignment assignContainersToChildQueues(Resource cluster,
|
|
|
|
|
|
+ private synchronized CSAssignment assignContainersToChildQueues(Resource cluster,
|
|
FiCaSchedulerNode node, boolean needToUnreserve) {
|
|
FiCaSchedulerNode node, boolean needToUnreserve) {
|
|
CSAssignment assignment =
|
|
CSAssignment assignment =
|
|
new CSAssignment(Resources.createResource(0, 0), NodeType.NODE_LOCAL);
|
|
new CSAssignment(Resources.createResource(0, 0), NodeType.NODE_LOCAL);
|
|
@@ -728,11 +614,16 @@ public class ParentQueue implements CSQueue {
|
|
String getChildQueuesToPrint() {
|
|
String getChildQueuesToPrint() {
|
|
StringBuilder sb = new StringBuilder();
|
|
StringBuilder sb = new StringBuilder();
|
|
for (CSQueue q : childQueues) {
|
|
for (CSQueue q : childQueues) {
|
|
- sb.append(q.getQueuePath() + "(" + q.getUsedCapacity() + "), ");
|
|
|
|
|
|
+ sb.append(q.getQueuePath() +
|
|
|
|
+ "usedCapacity=(" + q.getUsedCapacity() + "), " +
|
|
|
|
+ " label=("
|
|
|
|
+ + StringUtils.join(q.getAccessibleNodeLabels().iterator(), ",")
|
|
|
|
+ + ")");
|
|
}
|
|
}
|
|
return sb.toString();
|
|
return sb.toString();
|
|
}
|
|
}
|
|
- void printChildQueues() {
|
|
|
|
|
|
+
|
|
|
|
+ private void printChildQueues() {
|
|
if (LOG.isDebugEnabled()) {
|
|
if (LOG.isDebugEnabled()) {
|
|
LOG.debug("printChildQueues - queue: " + getQueuePath()
|
|
LOG.debug("printChildQueues - queue: " + getQueuePath()
|
|
+ " child-queues: " + getChildQueuesToPrint());
|
|
+ " child-queues: " + getChildQueuesToPrint());
|
|
@@ -749,8 +640,8 @@ public class ParentQueue implements CSQueue {
|
|
// Careful! Locking order is important!
|
|
// Careful! Locking order is important!
|
|
// Book keeping
|
|
// Book keeping
|
|
synchronized (this) {
|
|
synchronized (this) {
|
|
- releaseResource(clusterResource,
|
|
|
|
- rmContainer.getContainer().getResource());
|
|
|
|
|
|
+ super.releaseResource(clusterResource, rmContainer.getContainer()
|
|
|
|
+ .getResource(), labelManager.getLabelsOnNode(node.getNodeID()));
|
|
|
|
|
|
LOG.info("completedContainer" +
|
|
LOG.info("completedContainer" +
|
|
" queue=" + getQueueName() +
|
|
" queue=" + getQueueName() +
|
|
@@ -787,27 +678,6 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- @Private
|
|
|
|
- boolean getReservationContinueLooking() {
|
|
|
|
- return reservationsContinueLooking;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- synchronized void allocateResource(Resource clusterResource,
|
|
|
|
- Resource resource) {
|
|
|
|
- Resources.addTo(usedResources, resource);
|
|
|
|
- CSQueueUtils.updateQueueStatistics(
|
|
|
|
- resourceCalculator, this, parent, clusterResource, minimumAllocation);
|
|
|
|
- ++numContainers;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- synchronized void releaseResource(Resource clusterResource,
|
|
|
|
- Resource resource) {
|
|
|
|
- Resources.subtractFrom(usedResources, resource);
|
|
|
|
- CSQueueUtils.updateQueueStatistics(
|
|
|
|
- resourceCalculator, this, parent, clusterResource, minimumAllocation);
|
|
|
|
- --numContainers;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
@Override
|
|
@Override
|
|
public synchronized void updateClusterResource(Resource clusterResource) {
|
|
public synchronized void updateClusterResource(Resource clusterResource) {
|
|
// Update all children
|
|
// Update all children
|
|
@@ -821,10 +691,9 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
- public QueueMetrics getMetrics() {
|
|
|
|
- return metrics;
|
|
|
|
|
|
+ public synchronized List<CSQueue> getChildQueues() {
|
|
|
|
+ return new ArrayList<CSQueue>(childQueues);
|
|
}
|
|
}
|
|
-
|
|
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public void recoverContainer(Resource clusterResource,
|
|
public void recoverContainer(Resource clusterResource,
|
|
@@ -834,12 +703,20 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
// Careful! Locking order is important!
|
|
// Careful! Locking order is important!
|
|
synchronized (this) {
|
|
synchronized (this) {
|
|
- allocateResource(clusterResource,rmContainer.getContainer().getResource());
|
|
|
|
|
|
+ super.allocateResource(clusterResource, rmContainer.getContainer()
|
|
|
|
+ .getResource(), labelManager.getLabelsOnNode(rmContainer
|
|
|
|
+ .getContainer().getNodeId()));
|
|
}
|
|
}
|
|
if (parent != null) {
|
|
if (parent != null) {
|
|
parent.recoverContainer(clusterResource, attempt, rmContainer);
|
|
parent.recoverContainer(clusterResource, attempt, rmContainer);
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
+
|
|
|
|
+ @Override
|
|
|
|
+ public ActiveUsersManager getActiveUsersManager() {
|
|
|
|
+ // Should never be called since all applications are submitted to LeafQueues
|
|
|
|
+ return null;
|
|
|
|
+ }
|
|
|
|
|
|
@Override
|
|
@Override
|
|
public void collectSchedulerApplications(
|
|
public void collectSchedulerApplications(
|
|
@@ -853,8 +730,9 @@ public class ParentQueue implements CSQueue {
|
|
public void attachContainer(Resource clusterResource,
|
|
public void attachContainer(Resource clusterResource,
|
|
FiCaSchedulerApp application, RMContainer rmContainer) {
|
|
FiCaSchedulerApp application, RMContainer rmContainer) {
|
|
if (application != null) {
|
|
if (application != null) {
|
|
- allocateResource(clusterResource, rmContainer.getContainer()
|
|
|
|
- .getResource());
|
|
|
|
|
|
+ super.allocateResource(clusterResource, rmContainer.getContainer()
|
|
|
|
+ .getResource(), labelManager.getLabelsOnNode(rmContainer
|
|
|
|
+ .getContainer().getNodeId()));
|
|
LOG.info("movedContainer" + " queueMoveIn=" + getQueueName()
|
|
LOG.info("movedContainer" + " queueMoveIn=" + getQueueName()
|
|
+ " usedCapacity=" + getUsedCapacity() + " absoluteUsedCapacity="
|
|
+ " usedCapacity=" + getUsedCapacity() + " absoluteUsedCapacity="
|
|
+ getAbsoluteUsedCapacity() + " used=" + usedResources + " cluster="
|
|
+ getAbsoluteUsedCapacity() + " used=" + usedResources + " cluster="
|
|
@@ -870,7 +748,9 @@ public class ParentQueue implements CSQueue {
|
|
public void detachContainer(Resource clusterResource,
|
|
public void detachContainer(Resource clusterResource,
|
|
FiCaSchedulerApp application, RMContainer rmContainer) {
|
|
FiCaSchedulerApp application, RMContainer rmContainer) {
|
|
if (application != null) {
|
|
if (application != null) {
|
|
- releaseResource(clusterResource, rmContainer.getContainer().getResource());
|
|
|
|
|
|
+ super.releaseResource(clusterResource,
|
|
|
|
+ rmContainer.getContainer().getResource(),
|
|
|
|
+ labelManager.getLabelsOnNode(rmContainer.getContainer().getNodeId()));
|
|
LOG.info("movedContainer" + " queueMoveOut=" + getQueueName()
|
|
LOG.info("movedContainer" + " queueMoveOut=" + getQueueName()
|
|
+ " usedCapacity=" + getUsedCapacity() + " absoluteUsedCapacity="
|
|
+ " usedCapacity=" + getUsedCapacity() + " absoluteUsedCapacity="
|
|
+ getAbsoluteUsedCapacity() + " used=" + usedResources + " cluster="
|
|
+ getAbsoluteUsedCapacity() + " used=" + usedResources + " cluster="
|
|
@@ -882,7 +762,14 @@ public class ParentQueue implements CSQueue {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- public Map<QueueACL, AccessControlList> getACLs() {
|
|
|
|
- return acls;
|
|
|
|
|
|
+ @Override
|
|
|
|
+ public float getAbsActualCapacity() {
|
|
|
|
+ // for now, simply return actual capacity = guaranteed capacity for parent
|
|
|
|
+ // queue
|
|
|
|
+ return absoluteCapacity;
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ public synchronized int getNumApplications() {
|
|
|
|
+ return numApplications;
|
|
}
|
|
}
|
|
}
|
|
}
|