Class BaseResourceAwareStrategy
java.lang.Object
org.apache.storm.scheduler.resource.strategies.scheduling.BaseResourceAwareStrategy
- All Implemented Interfaces:
IStrategy
- Direct Known Subclasses:
ConstraintSolverStrategy
,DefaultResourceAwareStrategy
,DefaultResourceAwareStrategyOld
,GenericResourceAwareStrategy
,GenericResourceAwareStrategyOld
,RoundRobinResourceAwareStrategy
-
Nested Class Summary
Modifier and TypeClassDescriptionstatic enum
Different node sorting types available. -
Field Summary
Modifier and TypeFieldDescriptionprotected Cluster
protected Map<String,
Set<ExecutorDetails>> protected IExecSorter
protected Map<ExecutorDetails,
String> protected RasNodes
protected INodeSorter
protected final BaseResourceAwareStrategy.NodeSortType
protected boolean
protected SchedulingSearcherState
protected final boolean
protected TopologyDetails
protected String
-
Constructor Summary
ConstructorDescriptionBaseResourceAwareStrategy
(boolean sortNodesForEachExecutor, BaseResourceAwareStrategy.NodeSortType nodeSortType) Initialize for the default implementation of schedule(). -
Method Summary
Modifier and TypeMethodDescriptionprotected int
assignBoundAckersForNewWorkerSlot
(ExecutorDetails exec, RasNode node, WorkerSlot workerSlot) Determine how many bound ackers to put into the given workerSlot.protected SchedulingResult
Check scheduling feasibility for a quick failure as the second step inschedule(Cluster, TopologyDetails)
.static int
getMaxStateSearchFromTopoConf
(Map<String, Object> topoConf) hostnameToNodes
(String hostname) hostname to Ids.Find RASNode for specified node id.protected boolean
isExecAssignmentToWorkerValid
(ExecutorDetails exec, WorkerSlot worker) Check if the assignment of the executor to the worker is valid.static boolean
isOrderByProximity
(Map<String, Object> topoConf) void
Prepare the Strategy for scheduling.protected void
prepareForScheduling
(Cluster cluster, TopologyDetails topologyDetails) Initialize instance variables as the first step inschedule(Cluster, TopologyDetails)
.schedule
(Cluster cluster, TopologyDetails td) Note that this method is not thread-safe.protected SchedulingResult
scheduleExecutorsOnNodes
(List<ExecutorDetails> orderedExecutors, Iterable<String> sortedNodesIter) Try to schedule till successful or till limits (backtrack count or time) have been exceeded.protected void
setExecSorter
(IExecSorter execSorter) Set the pluggable sorter for ExecutorDetails.protected void
setNodeSorter
(INodeSorter nodeSorter) Set the pluggable sorter for Nodes.
-
Field Details
-
sortNodesForEachExecutor
protected final boolean sortNodesForEachExecutor -
nodeSortType
-
config
-
cluster
-
topologyDetails
-
nodes
-
topoName
-
compToExecs
-
execToComp
-
orderExecutorsByProximity
protected boolean orderExecutorsByProximity -
searcherState
-
execSorter
-
nodeSorter
-
-
Constructor Details
-
BaseResourceAwareStrategy
public BaseResourceAwareStrategy() -
BaseResourceAwareStrategy
public BaseResourceAwareStrategy(boolean sortNodesForEachExecutor, BaseResourceAwareStrategy.NodeSortType nodeSortType) Initialize for the default implementation of schedule().- Parameters:
sortNodesForEachExecutor
- Sort nodes before scheduling each executor.nodeSortType
- type of sorting to be applied to object resource collectionBaseResourceAwareStrategy.NodeSortType
.
-
-
Method Details
-
prepare
Description copied from interface:IStrategy
Prepare the Strategy for scheduling. -
schedule
Note that this method is not thread-safe. Several instance variables are generated from supplied parameters. In addition, the following instance variables are set to complete scheduling:searcherState
execSorter
to sort executorsnodeSorter
to sort nodesScheduling consists of three main steps:
prepareForScheduling(Cluster, TopologyDetails)
checkSchedulingFeasibility()
, andscheduleExecutorsOnNodes(List, Iterable)
The executors and nodes are sorted in the order most conducive to scheduling for the strategy. Those interfaces may be overridden by subclasses using mutators:
setExecSorter(IExecSorter)
andsetNodeSorter(INodeSorter)
-
prepareForScheduling
Initialize instance variables as the first step inschedule(Cluster, TopologyDetails)
. This method may be extended by subclasses to initialize additional variables as inConstraintSolverStrategy.prepareForScheduling(Cluster, TopologyDetails)
.- Parameters:
cluster
- on which executors will be scheduled.topologyDetails
- to be scheduled.
-
setExecSorter
Set the pluggable sorter for ExecutorDetails.- Parameters:
execSorter
- to use for sorting executorDetails when scheduling.
-
setNodeSorter
Set the pluggable sorter for Nodes.- Parameters:
nodeSorter
- to use for sorting nodes when scheduling.
-
getMaxStateSearchFromTopoConf
-
isOrderByProximity
-
checkSchedulingFeasibility
Check scheduling feasibility for a quick failure as the second step inschedule(Cluster, TopologyDetails)
. If scheduling is not possible, then return a SchedulingStatus object with a failure status. If fully scheduled then return a successful SchedulingStatus. This method can be extended by subclassesConstraintSolverStrategy.checkSchedulingFeasibility()
to check for additional failure conditions.- Returns:
- A non-null
SchedulingResult
to terminate scheduling, otherwise return null to continue scheduling.
-
isExecAssignmentToWorkerValid
Check if the assignment of the executor to the worker is valid. In simple cases, this is simply a check ofRasNode.wouldFit(WorkerSlot, ExecutorDetails, TopologyDetails)
. This method may be extended by subclasses to add additional checks, seeConstraintSolverStrategy.isExecAssignmentToWorkerValid(ExecutorDetails, WorkerSlot)
.- Parameters:
exec
- being scheduled.worker
- on which to schedule.- Returns:
- true if executor can be assigned to the worker, false otherwise.
-
hostnameToNodes
hostname to Ids.- Parameters:
hostname
- the hostname.- Returns:
- the ids n that node.
-
idToNode
Find RASNode for specified node id.- Parameters:
id
- the node/supervisor id to lookup- Returns:
- a RASNode object
-
scheduleExecutorsOnNodes
protected SchedulingResult scheduleExecutorsOnNodes(List<ExecutorDetails> orderedExecutors, Iterable<String> sortedNodesIter) Try to schedule till successful or till limits (backtrack count or time) have been exceeded.- Parameters:
orderedExecutors
- Executors sorted in the preferred order cannot be null - note that ackers are isolated at the end.sortedNodesIter
- Node iterable which may be null.- Returns:
- SchedulingResult with success attribute set to true or false indicting whether ALL executors were assigned.
-
assignBoundAckersForNewWorkerSlot
protected int assignBoundAckersForNewWorkerSlot(ExecutorDetails exec, RasNode node, WorkerSlot workerSlot) Determine how many bound ackers to put into the given workerSlot. Then try to assign the ackers one by one into this workerSlot upto the calculated maximum required. Return the number of ackers assigned. Return 0 if one of the conditions hold true: 1. No bound ackers are used. 2. This is not first exec assigned to this worker. 3. No ackers could be assigned because of space or exception.
- Parameters:
exec
- being scheduled.node
- RasNode on which to schedule.workerSlot
- WorkerSlot on which to schedule.- Returns:
- Number of ackers assigned.
-