public interface ISchedulingState
An interface that provides access to the current scheduling state. The scheduling state is not guaranteed to be thread safe.
| Modifier and Type | Method and Description | 
|---|---|
| NormalizedResourceRequest | getAllScheduledResourcesForNode(String nodeId)Get all scheduled resources for node. | 
| Set<Integer> | getAssignablePorts(SupervisorDetails supervisor)Get the ports that are not blacklisted. | 
| List<WorkerSlot> | getAssignableSlots()Get all non-blacklisted slots in the cluster. | 
| List<WorkerSlot> | getAssignableSlots(SupervisorDetails supervisor)Return all non-blacklisted slots on this supervisor. | 
| int | getAssignedNumWorkers(TopologyDetails topology)Get the number of workers assigned to a topology. | 
| default Set<String> | getAssignedRacks(String... topologyIds)Determine the list of racks on which topologyIds have been assigned. | 
| SchedulerAssignment | getAssignmentById(String topologyId)get the current assignment for the topology. | 
| Map<String,SchedulerAssignment> | getAssignments()Get all the assignments. | 
| Set<Integer> | getAvailablePorts(SupervisorDetails supervisor)Return the available ports of this supervisor. | 
| NormalizedResourceOffer | getAvailableResources(SupervisorDetails sd)Get the resources on the supervisor that are available to be scheduled. | 
| List<WorkerSlot> | getAvailableSlots()Get all the available worker slots in the cluster. | 
| List<WorkerSlot> | getAvailableSlots(SupervisorDetails supervisor)Return all the available slots on this supervisor. | 
| Set<String> | getBlacklistedHosts()Get all of the hosts that are blacklisted. | 
| double | getClusterTotalCpuResource()Get the total amount of CPU resources in cluster. | 
| Map<String,Double> | getClusterTotalGenericResources()Get the total amount of generic resources (excluding CPU and memory) in cluster. | 
| double | getClusterTotalMemoryResource()Get the total amount of memory resources in cluster. | 
| Map<String,Object> | getConf()Get the nimbus configuration. | 
| String | getHost(String supervisorId)Map a supervisor to a given host. | 
| default Map<String,String> | getHostToRack()Get host -> rack map - the inverse of networkTopography. | 
| Map<String,List<ExecutorDetails>> | getNeedsSchedulingComponentToExecutors(TopologyDetails topology)Get the component name to executor list for executors that need to be scheduled. | 
| Map<ExecutorDetails,String> | getNeedsSchedulingExecutorToComponents(TopologyDetails topology)Get the executor to component name map for executors that need to be scheduled. | 
| Map<String,List<String>> | getNetworkTopography()Get the network topography (rackId -> nodes in the rack). | 
| List<WorkerSlot> | getNonBlacklistedAvailableSlots(List<String> blacklistedSupervisorIds)Get all the available worker slots in the cluster, that are not blacklisted. | 
| NormalizedResourceOffer | getNonBlacklistedClusterAvailableResources(Collection<String> blacklistedSupervisorIds)Get the resources in the cluster that are available for scheduling. | 
| double | getScheduledCpuForNode(String nodeId)Get the total cpu currently scheduled on a node. | 
| double | getScheduledMemoryForNode(String nodeId)Get the total memory currently scheduled on a node. | 
| Map<String,String> | getStatusMap()Get all topology scheduler statuses. | 
| SupervisorDetails | getSupervisorById(String nodeId)Get a specific supervisor with the  nodeId. | 
| Map<String,SupervisorDetails> | getSupervisors()Get all the supervisors. | 
| List<SupervisorDetails> | getSupervisorsByHost(String host)Get all the supervisors on the specified  host. | 
| Map<String,SupervisorResources> | getSupervisorsResourcesMap()Get the amount of used and free resources on a supervisor. | 
| Topologies | getTopologies()Get all of the topologies. | 
| Map<String,TopologyResources> | getTopologyResourcesMap()Get the amount of resources used by topologies. | 
| Collection<ExecutorDetails> | getUnassignedExecutors(TopologyDetails topology)get the unassigned executors of the topology. | 
| Set<Integer> | getUsedPorts(SupervisorDetails supervisor)Get all the used ports of this supervisor. | 
| Collection<WorkerSlot> | getUsedSlots()Get all currently occupied slots. | 
| Collection<WorkerSlot> | getUsedSlotsByTopologyId(String topologyId)get slots used by a topology. | 
| WorkerResources | getWorkerResources(WorkerSlot ws)Get the resources for a given slot. | 
| Map<String,Map<WorkerSlot,WorkerResources>> | getWorkerResourcesMap()Gets the reference to the full topology->worker resource map. | 
| boolean | isBlackListed(String supervisorId)Check is a given supervisor is on a blacklisted host. | 
| boolean | isBlacklistedHost(String host)Check if a given host is blacklisted. | 
| boolean | isSlotOccupied(WorkerSlot slot)Check if a slot is occupied or not. | 
| boolean | needsScheduling(TopologyDetails topology)Does the topology need scheduling. | 
| boolean | needsSchedulingRas(TopologyDetails topology)Like  needsScheduling(TopologyDetails)but does not take into account the number of workers requested. | 
| List<TopologyDetails> | needsSchedulingTopologies()Get all of the topologies that need scheduling. | 
| boolean | wouldFit(WorkerSlot ws,
        ExecutorDetails exec,
        TopologyDetails td,
        NormalizedResourceOffer resourcesAvailable,
        double maxHeap)Would scheduling exec on ws fit? With a heap <= maxHeap total memory added <= memoryAvailable and cpu added <= cpuAvailable. | 
Topologies getTopologies()
Get all of the topologies.
List<TopologyDetails> needsSchedulingTopologies()
Get all of the topologies that need scheduling.
boolean needsScheduling(TopologyDetails topology)
Does the topology need scheduling.
A topology needs scheduling if one of the following conditions holds:
boolean needsSchedulingRas(TopologyDetails topology)
Like needsScheduling(TopologyDetails) but does not take into account the number of workers requested. This is because the number of workers is ignored in RAS
topology - the topology to checkSet<String> getBlacklistedHosts()
Get all of the hosts that are blacklisted.
boolean isBlackListed(String supervisorId)
Check is a given supervisor is on a blacklisted host.
supervisorId - the id of the supervisorboolean isBlacklistedHost(String host)
Check if a given host is blacklisted.
host - the name of the hostString getHost(String supervisorId)
Map a supervisor to a given host.
supervisorId - the id of the supervisorCollection<ExecutorDetails> getUnassignedExecutors(TopologyDetails topology)
get the unassigned executors of the topology.
topology - the topology to checkMap<ExecutorDetails,String> getNeedsSchedulingExecutorToComponents(TopologyDetails topology)
Get the executor to component name map for executors that need to be scheduled.
topology - the topology this is forMap<String,List<ExecutorDetails>> getNeedsSchedulingComponentToExecutors(TopologyDetails topology)
Get the component name to executor list for executors that need to be scheduled.
topology - the topology this is forSet<Integer> getUsedPorts(SupervisorDetails supervisor)
Get all the used ports of this supervisor.
Set<Integer> getAvailablePorts(SupervisorDetails supervisor)
Return the available ports of this supervisor.
Set<Integer> getAssignablePorts(SupervisorDetails supervisor)
Get the ports that are not blacklisted.
supervisor - the supervisorList<WorkerSlot> getAvailableSlots(SupervisorDetails supervisor)
Return all the available slots on this supervisor.
List<WorkerSlot> getAvailableSlots()
Get all the available worker slots in the cluster.
List<WorkerSlot> getNonBlacklistedAvailableSlots(List<String> blacklistedSupervisorIds)
Get all the available worker slots in the cluster, that are not blacklisted.
blacklistedSupervisorIds - list of supervisor ids that should also be considered blacklisted.List<WorkerSlot> getAssignableSlots(SupervisorDetails supervisor)
Return all non-blacklisted slots on this supervisor.
supervisor - the supervisorList<WorkerSlot> getAssignableSlots()
Get all non-blacklisted slots in the cluster.
Collection<WorkerSlot> getUsedSlots()
Get all currently occupied slots.
boolean isSlotOccupied(WorkerSlot slot)
Check if a slot is occupied or not.
slot - the slot be to checked.int getAssignedNumWorkers(TopologyDetails topology)
Get the number of workers assigned to a topology.
topology - the topology this is forNormalizedResourceOffer getAvailableResources(SupervisorDetails sd)
Get the resources on the supervisor that are available to be scheduled.
sd - the supervisor.boolean wouldFit(WorkerSlot ws, ExecutorDetails exec, TopologyDetails td, NormalizedResourceOffer resourcesAvailable, double maxHeap)
Would scheduling exec on ws fit? With a heap <= maxHeap total memory added <= memoryAvailable and cpu added <= cpuAvailable.
ws - the slot to put it inexec - the executor to investigatetd - the topology detains for this executorresourcesAvailable - all the available resourcesmaxHeap - the maximum heap size for wsSchedulerAssignment getAssignmentById(String topologyId)
get the current assignment for the topology.
Collection<WorkerSlot> getUsedSlotsByTopologyId(String topologyId)
get slots used by a topology.
SupervisorDetails getSupervisorById(String nodeId)
Get a specific supervisor with the nodeId.
List<SupervisorDetails> getSupervisorsByHost(String host)
Get all the supervisors on the specified host.
host - hostname of the supervisorSupervisorDetails object.Map<String,SchedulerAssignment> getAssignments()
Get all the assignments.
Map<String,SupervisorDetails> getSupervisors()
Get all the supervisors.
NormalizedResourceRequest getAllScheduledResourcesForNode(String nodeId)
Get all scheduled resources for node.
NormalizedResourceOffer getNonBlacklistedClusterAvailableResources(Collection<String> blacklistedSupervisorIds)
Get the resources in the cluster that are available for scheduling.
blacklistedSupervisorIds - other ids that are tentatively blacklisted.double getClusterTotalCpuResource()
Get the total amount of CPU resources in cluster.
double getClusterTotalMemoryResource()
Get the total amount of memory resources in cluster.
Map<String,Double> getClusterTotalGenericResources()
Get the total amount of generic resources (excluding CPU and memory) in cluster.
Map<String,List<String>> getNetworkTopography()
Get the network topography (rackId -> nodes in the rack).
default Map<String,String> getHostToRack()
Get host -> rack map - the inverse of networkTopography.
Map<String,TopologyResources> getTopologyResourcesMap()
Get the amount of resources used by topologies. Used for displaying resource information on the UI.
Map<String,SupervisorResources> getSupervisorsResourcesMap()
Get the amount of used and free resources on a supervisor. Used for displaying resource information on the UI
Map<String,Map<WorkerSlot,WorkerResources>> getWorkerResourcesMap()
Gets the reference to the full topology->worker resource map.
WorkerResources getWorkerResources(WorkerSlot ws)
Get the resources for a given slot.
ws - the slotdouble getScheduledMemoryForNode(String nodeId)
Get the total memory currently scheduled on a node.
nodeId - the id of the nodedouble getScheduledCpuForNode(String nodeId)
Get the total cpu currently scheduled on a node.
nodeId - the id of the nodedefault Set<String> getAssignedRacks(String... topologyIds)
Determine the list of racks on which topologyIds have been assigned. Note that the returned set may contain DNSToSwitchMapping.DEFAULT_RACK if getHostToRack() is null or does not contain the assigned host.
topologyIds - for which assignments are examined.Copyright © 2022 The Apache Software Foundation. All rights reserved.