public class Cluster extends Object implements ISchedulingState
The current state of the storm cluster. Cluster is not currently thread safe.
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, wait
getAssignedRacks, getHostToRack
public Cluster(INimbus nimbus, ResourceMetrics resourceMetrics, Map<String,SupervisorDetails> supervisors, Map<String,? extends SchedulerAssignment> assignments, Topologies topologies, Map<String,Object> conf)
public Cluster(Cluster src)
Copy constructor.
public Cluster(Cluster src, Topologies topologies)
Testing Constructor that takes an existing cluster and replaces the topologies in it.
src
- the original clustertopologies
- the new topolgoies to usepublic static double getAssignedMemoryForSlot(Map<String,Object> topConf)
Get heap memory usage for a worker’s main process and logwriter process.
topConf
- protected void assertValidTopologyForModification(String topologyId)
Check if the given topology is allowed for modification right now. If not throw an IllegalArgumentException else go on.
topologyId
- the id of the topology to checkpublic Topologies getTopologies()
ISchedulingState
Get all of the topologies.
getTopologies
in interface ISchedulingState
public Set<String> getBlacklistedHosts()
ISchedulingState
Get all of the hosts that are blacklisted.
getBlacklistedHosts
in interface ISchedulingState
public void setBlacklistedHosts(Set<String> hosts)
Set the list of hosts that are blacklisted.
hosts
- the new hosts that are blacklisted.public void blacklistHost(String host)
public boolean isBlackListed(String supervisorId)
ISchedulingState
Check is a given supervisor is on a blacklisted host.
isBlackListed
in interface ISchedulingState
supervisorId
- the id of the supervisorpublic boolean isBlacklistedHost(String host)
ISchedulingState
Check if a given host is blacklisted.
isBlacklistedHost
in interface ISchedulingState
host
- the name of the hostpublic String getHost(String supervisorId)
ISchedulingState
Map a supervisor to a given host.
getHost
in interface ISchedulingState
supervisorId
- the id of the supervisorpublic List<TopologyDetails> needsSchedulingTopologies()
ISchedulingState
Get all of the topologies that need scheduling.
needsSchedulingTopologies
in interface ISchedulingState
public boolean needsScheduling(TopologyDetails topology)
ISchedulingState
Does the topology need scheduling.
A topology needs scheduling if one of the following conditions holds:
needsScheduling
in interface ISchedulingState
public boolean needsSchedulingRas(TopologyDetails topology)
ISchedulingState
Like ISchedulingState.needsScheduling(TopologyDetails)
but does not take into account the number of workers requested. This is because the number of workers is ignored in RAS
needsSchedulingRas
in interface ISchedulingState
topology
- the topology to checkpublic Map<ExecutorDetails,String> getNeedsSchedulingExecutorToComponents(TopologyDetails topology)
ISchedulingState
Get the executor to component name map for executors that need to be scheduled.
getNeedsSchedulingExecutorToComponents
in interface ISchedulingState
topology
- the topology this is forpublic Map<String,List<ExecutorDetails>> getNeedsSchedulingComponentToExecutors(TopologyDetails topology)
ISchedulingState
Get the component name to executor list for executors that need to be scheduled.
getNeedsSchedulingComponentToExecutors
in interface ISchedulingState
topology
- the topology this is forpublic Set<Integer> getUsedPorts(SupervisorDetails supervisor)
ISchedulingState
Get all the used ports of this supervisor.
getUsedPorts
in interface ISchedulingState
public Set<Integer> getAvailablePorts(SupervisorDetails supervisor)
ISchedulingState
Return the available ports of this supervisor.
getAvailablePorts
in interface ISchedulingState
public Set<Integer> getAssignablePorts(SupervisorDetails supervisor)
ISchedulingState
Get the ports that are not blacklisted.
getAssignablePorts
in interface ISchedulingState
supervisor
- the supervisorpublic List<WorkerSlot> getNonBlacklistedAvailableSlots(List<String> blacklistedSupervisorIds)
ISchedulingState
Get all the available worker slots in the cluster, that are not blacklisted.
getNonBlacklistedAvailableSlots
in interface ISchedulingState
blacklistedSupervisorIds
- list of supervisor ids that should also be considered blacklisted.public List<WorkerSlot> getAvailableSlots()
ISchedulingState
Get all the available worker slots in the cluster.
getAvailableSlots
in interface ISchedulingState
public List<WorkerSlot> getAvailableSlots(SupervisorDetails supervisor)
ISchedulingState
Return all the available slots on this supervisor.
getAvailableSlots
in interface ISchedulingState
public List<WorkerSlot> getAssignableSlots(SupervisorDetails supervisor)
ISchedulingState
Return all non-blacklisted slots on this supervisor.
getAssignableSlots
in interface ISchedulingState
supervisor
- the supervisorpublic List<WorkerSlot> getAssignableSlots()
ISchedulingState
Get all non-blacklisted slots in the cluster.
getAssignableSlots
in interface ISchedulingState
public Collection<ExecutorDetails> getUnassignedExecutors(TopologyDetails topology)
ISchedulingState
get the unassigned executors of the topology.
getUnassignedExecutors
in interface ISchedulingState
topology
- the topology to checkpublic int getAssignedNumWorkers(TopologyDetails topology)
ISchedulingState
Get the number of workers assigned to a topology.
getAssignedNumWorkers
in interface ISchedulingState
topology
- the topology this is forpublic NormalizedResourceOffer getAvailableResources(SupervisorDetails sd)
ISchedulingState
Get the resources on the supervisor that are available to be scheduled.
getAvailableResources
in interface ISchedulingState
sd
- the supervisor.public boolean wouldFit(WorkerSlot ws, ExecutorDetails exec, TopologyDetails td, NormalizedResourceOffer resourcesAvailable, double maxHeap)
ISchedulingState
Would scheduling exec on ws fit? With a heap <= maxHeap total memory added <= memoryAvailable and cpu added <= cpuAvailable.
wouldFit
in interface ISchedulingState
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 wspublic void assign(WorkerSlot slot, String topologyId, Collection<ExecutorDetails> executors)
Assign the slot to the executors for this topology.
RuntimeException
- if the specified slot is already occupied.public void assign(SchedulerAssignment assignment, boolean ignoreSingleExceptions)
Assign everything for the given topology.
assignment
- the new assignment to makepublic void freeSlot(WorkerSlot slot)
Free the specified slot.
slot
- the slot to freepublic void freeSlots(Collection<WorkerSlot> slots)
free the slots.
slots
- multiple slots to freepublic boolean isSlotOccupied(WorkerSlot slot)
ISchedulingState
Check if a slot is occupied or not.
isSlotOccupied
in interface ISchedulingState
slot
- the slot be to checked.public SchedulerAssignment getAssignmentById(String topologyId)
ISchedulingState
get the current assignment for the topology.
getAssignmentById
in interface ISchedulingState
public Collection<WorkerSlot> getUsedSlotsByTopologyId(String topologyId)
ISchedulingState
get slots used by a topology.
getUsedSlotsByTopologyId
in interface ISchedulingState
public SupervisorDetails getSupervisorById(String nodeId)
ISchedulingState
Get a specific supervisor with the nodeId
.
getSupervisorById
in interface ISchedulingState
public Collection<WorkerSlot> getUsedSlots()
ISchedulingState
Get all currently occupied slots.
getUsedSlots
in interface ISchedulingState
public List<SupervisorDetails> getSupervisorsByHost(String host)
ISchedulingState
Get all the supervisors on the specified host
.
getSupervisorsByHost
in interface ISchedulingState
host
- hostname of the supervisorSupervisorDetails
object.public Map<String,SchedulerAssignment> getAssignments()
ISchedulingState
Get all the assignments.
getAssignments
in interface ISchedulingState
public void setAssignments(Map<String,? extends SchedulerAssignment> newAssignments, boolean ignoreSingleExceptions)
Set assignments for cluster.
public Map<String,SupervisorDetails> getSupervisors()
ISchedulingState
Get all the supervisors.
getSupervisors
in interface ISchedulingState
public NormalizedResourceOffer getNonBlacklistedClusterAvailableResources(Collection<String> blacklistedSupervisorIds)
ISchedulingState
Get the resources in the cluster that are available for scheduling.
getNonBlacklistedClusterAvailableResources
in interface ISchedulingState
blacklistedSupervisorIds
- other ids that are tentatively blacklisted.public double getClusterTotalCpuResource()
ISchedulingState
Get the total amount of CPU resources in cluster.
getClusterTotalCpuResource
in interface ISchedulingState
public double getClusterTotalMemoryResource()
ISchedulingState
Get the total amount of memory resources in cluster.
getClusterTotalMemoryResource
in interface ISchedulingState
public Map<String,Double> getClusterTotalGenericResources()
ISchedulingState
Get the total amount of generic resources (excluding CPU and memory) in cluster.
getClusterTotalGenericResources
in interface ISchedulingState
public Map<String,List<String>> getNetworkTopography()
ISchedulingState
Get the network topography (rackId -> nodes in the rack).
getNetworkTopography
in interface ISchedulingState
public void setStatus(TopologyDetails td, String statusMessage)
set scheduler status for a topology.
public void setStatus(String topologyId, String statusMessage)
set scheduler status for a topology.
public Map<String,String> getStatusMap()
ISchedulingState
Get all topology scheduler statuses.
getStatusMap
in interface ISchedulingState
public Map<String,TopologyResources> getTopologyResourcesMap()
ISchedulingState
Get the amount of resources used by topologies. Used for displaying resource information on the UI.
getTopologyResourcesMap
in interface ISchedulingState
public Map<String,SupervisorResources> getSupervisorsResourcesMap()
ISchedulingState
Get the amount of used and free resources on a supervisor. Used for displaying resource information on the UI
getSupervisorsResourcesMap
in interface ISchedulingState
public Map<String,Map<WorkerSlot,WorkerResources>> getWorkerResourcesMap()
ISchedulingState
Gets the reference to the full topology->worker resource map.
getWorkerResourcesMap
in interface ISchedulingState
public WorkerResources getWorkerResources(WorkerSlot ws)
ISchedulingState
Get the resources for a given slot.
getWorkerResources
in interface ISchedulingState
ws
- the slotpublic ResourceMetrics getResourceMetrics()
public NormalizedResourceRequest getAllScheduledResourcesForNode(String nodeId)
ISchedulingState
Get all scheduled resources for node.
getAllScheduledResourcesForNode
in interface ISchedulingState
public double getScheduledMemoryForNode(String nodeId)
ISchedulingState
Get the total memory currently scheduled on a node.
getScheduledMemoryForNode
in interface ISchedulingState
nodeId
- the id of the nodepublic double getScheduledCpuForNode(String nodeId)
ISchedulingState
Get the total cpu currently scheduled on a node.
getScheduledCpuForNode
in interface ISchedulingState
nodeId
- the id of the nodepublic INimbus getINimbus()
public Map<String,Object> getConf()
ISchedulingState
Get the nimbus configuration.
getConf
in interface ISchedulingState
public void unassign(String topoId)
Unassign everything for the given topology id.
topoId
- the is of the topology to unassignpublic void updateFrom(Cluster other)
Update the assignments and status from the other cluster.
other
- the cluster to get the assignments and status frompublic double getMinWorkerCpu()
Copyright © 2022 The Apache Software Foundation. All rights reserved.