public class Cluster extends Object implements ISchedulingState
| Constructor and Description |
|---|
Cluster(Cluster src)
Copy constructor.
|
Cluster(Cluster src,
Topologies topologies)
Testing Constructor that takes an existing cluster and replaces the topologies in it.
|
Cluster(INimbus nimbus,
ResourceMetrics resourceMetrics,
Map<String,SupervisorDetails> supervisors,
Map<String,? extends SchedulerAssignment> assignments,
Topologies topologies,
Map<String,Object> conf) |
| Modifier and Type | Method and Description |
|---|---|
protected void |
assertValidTopologyForModification(String topologyId)
Check if the given topology is allowed for modification right now.
|
void |
assign(SchedulerAssignment assignment,
boolean ignoreSingleExceptions)
Assign everything for the given topology.
|
void |
assign(WorkerSlot slot,
String topologyId,
Collection<ExecutorDetails> executors)
Assign the slot to the executors for this topology.
|
void |
blacklistHost(String host) |
void |
freeSlot(WorkerSlot slot)
Free the specified slot.
|
void |
freeSlots(Collection<WorkerSlot> slots)
free the slots.
|
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.
|
static double |
getAssignedMemoryForSlot(Map<String,Object> topConf)
Get heap memory usage for a worker's main process and logwriter process.
|
int |
getAssignedNumWorkers(TopologyDetails topology)
Get the number of workers assigned to a topology.
|
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.
|
List<String> |
getGreyListedSupervisors() |
String |
getHost(String supervisorId)
Map a supervisor to a given host.
|
INimbus |
getINimbus() |
double |
getMinWorkerCpu() |
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.
|
ResourceMetrics |
getResourceMetrics() |
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.
|
String |
getStatus(String topoId) |
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
ISchedulingState.needsScheduling(TopologyDetails) but does not take into account the number of workers requested. |
List<TopologyDetails> |
needsSchedulingTopologies()
Get all of the topologies that need scheduling.
|
void |
setAssignments(Map<String,? extends SchedulerAssignment> newAssignments,
boolean ignoreSingleExceptions)
Set assignments for cluster.
|
void |
setBlacklistedHosts(Set<String> hosts)
Set the list of hosts that are blacklisted.
|
void |
setGreyListedSupervisors(Set<String> greyListedSupervisors) |
void |
setNetworkTopography(Map<String,List<String>> networkTopography) |
void |
setStatus(String topologyId,
String statusMessage)
set scheduler status for a topology.
|
void |
setStatus(TopologyDetails td,
String statusMessage)
set scheduler status for a topology.
|
void |
setStatusIfAbsent(String topologyId,
String statusMessage) |
void |
setStatusMap(Map<String,String> statusMap)
set scheduler status map.
|
void |
unassign(String topoId)
Unassign everything for the given topology id.
|
void |
updateFrom(Cluster other)
Update the assignments and status from the other cluster.
|
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.
|
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, waitgetAssignedRacks, getHostToRackpublic Cluster(INimbus nimbus, ResourceMetrics resourceMetrics, Map<String,SupervisorDetails> supervisors, Map<String,? extends SchedulerAssignment> assignments, Topologies topologies, Map<String,Object> conf)
public Cluster(Cluster src)
public Cluster(Cluster src, Topologies topologies)
src - the original clustertopologies - the new topolgoies to usepublic static double getAssignedMemoryForSlot(Map<String,Object> topConf)
topConf - - the topology configprotected void assertValidTopologyForModification(String topologyId)
topologyId - the id of the topology to checkpublic Topologies getTopologies()
ISchedulingStategetTopologies in interface ISchedulingStatepublic Set<String> getBlacklistedHosts()
ISchedulingStategetBlacklistedHosts in interface ISchedulingStatepublic void setBlacklistedHosts(Set<String> hosts)
hosts - the new hosts that are blacklisted.public void blacklistHost(String host)
public boolean isBlackListed(String supervisorId)
ISchedulingStateisBlackListed in interface ISchedulingStatesupervisorId - the id of the supervisorpublic boolean isBlacklistedHost(String host)
ISchedulingStateisBlacklistedHost in interface ISchedulingStatehost - the name of the hostpublic String getHost(String supervisorId)
ISchedulingStategetHost in interface ISchedulingStatesupervisorId - the id of the supervisorpublic List<TopologyDetails> needsSchedulingTopologies()
ISchedulingStateneedsSchedulingTopologies in interface ISchedulingStatepublic boolean needsScheduling(TopologyDetails topology)
ISchedulingStateA topology needs scheduling if one of the following conditions holds:
needsScheduling in interface ISchedulingStatepublic boolean needsSchedulingRas(TopologyDetails topology)
ISchedulingStateISchedulingState.needsScheduling(TopologyDetails) but does not take into account the number of workers requested. This is because the
number of workers is ignored in RASneedsSchedulingRas in interface ISchedulingStatetopology - the topology to checkpublic Map<ExecutorDetails,String> getNeedsSchedulingExecutorToComponents(TopologyDetails topology)
ISchedulingStategetNeedsSchedulingExecutorToComponents in interface ISchedulingStatetopology - the topology this is forpublic Map<String,List<ExecutorDetails>> getNeedsSchedulingComponentToExecutors(TopologyDetails topology)
ISchedulingStategetNeedsSchedulingComponentToExecutors in interface ISchedulingStatetopology - the topology this is forpublic Set<Integer> getUsedPorts(SupervisorDetails supervisor)
ISchedulingStategetUsedPorts in interface ISchedulingStatepublic Set<Integer> getAvailablePorts(SupervisorDetails supervisor)
ISchedulingStategetAvailablePorts in interface ISchedulingStatepublic Set<Integer> getAssignablePorts(SupervisorDetails supervisor)
ISchedulingStategetAssignablePorts in interface ISchedulingStatesupervisor - the supervisorpublic List<WorkerSlot> getNonBlacklistedAvailableSlots(List<String> blacklistedSupervisorIds)
ISchedulingStategetNonBlacklistedAvailableSlots in interface ISchedulingStateblacklistedSupervisorIds - list of supervisor ids that should also be considered blacklisted.public List<WorkerSlot> getAvailableSlots()
ISchedulingStategetAvailableSlots in interface ISchedulingStatepublic List<WorkerSlot> getAvailableSlots(SupervisorDetails supervisor)
ISchedulingStategetAvailableSlots in interface ISchedulingStatepublic List<WorkerSlot> getAssignableSlots(SupervisorDetails supervisor)
ISchedulingStategetAssignableSlots in interface ISchedulingStatesupervisor - the supervisorpublic List<WorkerSlot> getAssignableSlots()
ISchedulingStategetAssignableSlots in interface ISchedulingStatepublic Collection<ExecutorDetails> getUnassignedExecutors(TopologyDetails topology)
ISchedulingStategetUnassignedExecutors in interface ISchedulingStatetopology - the topology to checkpublic int getAssignedNumWorkers(TopologyDetails topology)
ISchedulingStategetAssignedNumWorkers in interface ISchedulingStatetopology - the topology this is forpublic NormalizedResourceOffer getAvailableResources(SupervisorDetails sd)
ISchedulingStategetAvailableResources in interface ISchedulingStatesd - the supervisor.public boolean wouldFit(WorkerSlot ws, ExecutorDetails exec, TopologyDetails td, NormalizedResourceOffer resourcesAvailable, double maxHeap)
ISchedulingStatewouldFit in interface ISchedulingStatews - 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 wspublic void assign(WorkerSlot slot, String topologyId, Collection<ExecutorDetails> executors)
RuntimeException - if the specified slot is already occupied.public void assign(SchedulerAssignment assignment, boolean ignoreSingleExceptions)
assignment - the new assignment to makepublic void freeSlot(WorkerSlot slot)
slot - the slot to freepublic void freeSlots(Collection<WorkerSlot> slots)
slots - multiple slots to freepublic boolean isSlotOccupied(WorkerSlot slot)
ISchedulingStateisSlotOccupied in interface ISchedulingStateslot - the slot be to checked.public SchedulerAssignment getAssignmentById(String topologyId)
ISchedulingStategetAssignmentById in interface ISchedulingStatepublic Collection<WorkerSlot> getUsedSlotsByTopologyId(String topologyId)
ISchedulingStategetUsedSlotsByTopologyId in interface ISchedulingStatepublic SupervisorDetails getSupervisorById(String nodeId)
ISchedulingStatenodeId.getSupervisorById in interface ISchedulingStatepublic Collection<WorkerSlot> getUsedSlots()
ISchedulingStategetUsedSlots in interface ISchedulingStatepublic List<SupervisorDetails> getSupervisorsByHost(String host)
ISchedulingStatehost.getSupervisorsByHost in interface ISchedulingStatehost - hostname of the supervisorSupervisorDetails object.public Map<String,SchedulerAssignment> getAssignments()
ISchedulingStategetAssignments in interface ISchedulingStatepublic void setAssignments(Map<String,? extends SchedulerAssignment> newAssignments, boolean ignoreSingleExceptions)
public Map<String,SupervisorDetails> getSupervisors()
ISchedulingStategetSupervisors in interface ISchedulingStatepublic NormalizedResourceOffer getNonBlacklistedClusterAvailableResources(Collection<String> blacklistedSupervisorIds)
ISchedulingStategetNonBlacklistedClusterAvailableResources in interface ISchedulingStateblacklistedSupervisorIds - other ids that are tentatively blacklisted.public double getClusterTotalCpuResource()
ISchedulingStategetClusterTotalCpuResource in interface ISchedulingStatepublic double getClusterTotalMemoryResource()
ISchedulingStategetClusterTotalMemoryResource in interface ISchedulingStatepublic Map<String,Double> getClusterTotalGenericResources()
ISchedulingStategetClusterTotalGenericResources in interface ISchedulingStatepublic Map<String,List<String>> getNetworkTopography()
ISchedulingStategetNetworkTopography in interface ISchedulingStatepublic void setStatus(TopologyDetails td, String statusMessage)
public void setStatus(String topologyId, String statusMessage)
public Map<String,String> getStatusMap()
ISchedulingStategetStatusMap in interface ISchedulingStatepublic Map<String,TopologyResources> getTopologyResourcesMap()
ISchedulingStategetTopologyResourcesMap in interface ISchedulingStatepublic Map<String,SupervisorResources> getSupervisorsResourcesMap()
ISchedulingStategetSupervisorsResourcesMap in interface ISchedulingStatepublic Map<String,Map<WorkerSlot,WorkerResources>> getWorkerResourcesMap()
ISchedulingStategetWorkerResourcesMap in interface ISchedulingStatepublic WorkerResources getWorkerResources(WorkerSlot ws)
ISchedulingStategetWorkerResources in interface ISchedulingStatews - the slotpublic ResourceMetrics getResourceMetrics()
public NormalizedResourceRequest getAllScheduledResourcesForNode(String nodeId)
ISchedulingStategetAllScheduledResourcesForNode in interface ISchedulingStatepublic double getScheduledMemoryForNode(String nodeId)
ISchedulingStategetScheduledMemoryForNode in interface ISchedulingStatenodeId - the id of the nodepublic double getScheduledCpuForNode(String nodeId)
ISchedulingStategetScheduledCpuForNode in interface ISchedulingStatenodeId - the id of the nodepublic INimbus getINimbus()
public Map<String,Object> getConf()
ISchedulingStategetConf in interface ISchedulingStatepublic void unassign(String topoId)
topoId - the is of the topology to unassignpublic void updateFrom(Cluster other)
other - the cluster to get the assignments and status frompublic double getMinWorkerCpu()
Copyright © 2023 The Apache Software Foundation. All rights reserved.