public interface SlotPool extends AllocatedSlotActions, AutoCloseable
Modifier and Type | Method and Description |
---|---|
Optional<PhysicalSlot> |
allocateAvailableSlot(SlotRequestId slotRequestId,
AllocationID allocationID)
Allocates the available slot with the given allocation id under the given request id.
|
void |
close() |
void |
connectToResourceManager(ResourceManagerGateway resourceManagerGateway)
Connects the SlotPool to the given ResourceManager.
|
AllocatedSlotReport |
createAllocatedSlotReport(ResourceID taskManagerId)
Create report about the allocated slots belonging to the specified task manager.
|
void |
disableBatchSlotRequestTimeoutCheck()
Disables batch slot request timeout check.
|
void |
disconnectResourceManager()
Disconnects the slot pool from its current Resource Manager.
|
Optional<ResourceID> |
failAllocation(AllocationID allocationID,
Exception cause)
Fails the slot with the given allocation id.
|
Collection<SlotInfo> |
getAllocatedSlotsInformation()
Returns a list of
SlotInfo objects about all slots that are currently allocated in
the slot pool. |
Collection<SlotInfoWithUtilization> |
getAvailableSlotsInformation()
Returns a list of
SlotInfoWithUtilization objects about all slots that are currently
available in the slot pool. |
Collection<SlotOffer> |
offerSlots(TaskManagerLocation taskManagerLocation,
TaskManagerGateway taskManagerGateway,
Collection<SlotOffer> offers)
Offers multiple slots to the
SlotPool . |
boolean |
registerTaskManager(ResourceID resourceID)
Registers a TaskExecutor with the given
ResourceID at SlotPool . |
boolean |
releaseTaskManager(ResourceID resourceId,
Exception cause)
Releases a TaskExecutor with the given
ResourceID from the SlotPool . |
CompletableFuture<PhysicalSlot> |
requestNewAllocatedBatchSlot(SlotRequestId slotRequestId,
ResourceProfile resourceProfile)
Requests the allocation of a new batch slot from the resource manager.
|
CompletableFuture<PhysicalSlot> |
requestNewAllocatedSlot(SlotRequestId slotRequestId,
ResourceProfile resourceProfile,
Time timeout)
Request the allocation of a new slot from the resource manager.
|
void |
start(JobMasterId jobMasterId,
String newJobManagerAddress,
ComponentMainThreadExecutor jmMainThreadScheduledExecutor) |
void |
suspend() |
releaseSlot
void start(JobMasterId jobMasterId, String newJobManagerAddress, ComponentMainThreadExecutor jmMainThreadScheduledExecutor) throws Exception
Exception
void suspend()
void close()
close
in interface AutoCloseable
void connectToResourceManager(ResourceManagerGateway resourceManagerGateway)
resourceManagerGateway
- The RPC gateway for the resource manager.void disconnectResourceManager()
The slot pool will still be able to serve slots from its internal pool.
boolean registerTaskManager(ResourceID resourceID)
ResourceID
at SlotPool
.resourceID
- identifying the TaskExecutor to registerboolean releaseTaskManager(ResourceID resourceId, Exception cause)
ResourceID
from the SlotPool
.resourceId
- identifying the TaskExecutor which shall be released from the SlotPoolcause
- for the releasing of the TaskManagerCollection<SlotOffer> offerSlots(TaskManagerLocation taskManagerLocation, TaskManagerGateway taskManagerGateway, Collection<SlotOffer> offers)
SlotPool
. The slot offerings can be individually
accepted or rejected by returning the collection of accepted slot offers.taskManagerLocation
- from which the slot offers originatetaskManagerGateway
- to talk to the slot offereroffers
- slot offers which are offered to the SlotPool
Optional<ResourceID> failAllocation(AllocationID allocationID, Exception cause)
allocationID
- identifying the slot which is being failedcause
- of the failure@Nonnull Collection<SlotInfoWithUtilization> getAvailableSlotsInformation()
SlotInfoWithUtilization
objects about all slots that are currently
available in the slot pool.SlotInfoWithUtilization
objects about all slots that are currently
available in the slot pool.Collection<SlotInfo> getAllocatedSlotsInformation()
SlotInfo
objects about all slots that are currently allocated in
the slot pool.SlotInfo
objects about all slots that are currently allocated in
the slot pool.Optional<PhysicalSlot> allocateAvailableSlot(@Nonnull SlotRequestId slotRequestId, @Nonnull AllocationID allocationID)
null
if no slot with the given allocation id is available.slotRequestId
- identifying the requested slotallocationID
- the allocation id of the requested available slotnull
if no such
slot existed.@Nonnull CompletableFuture<PhysicalSlot> requestNewAllocatedSlot(@Nonnull SlotRequestId slotRequestId, @Nonnull ResourceProfile resourceProfile, @Nullable Time timeout)
slotRequestId
- identifying the requested slotresourceProfile
- resource profile that specifies the resource requirements for the
requested slottimeout
- timeout for the allocation procedure@Nonnull CompletableFuture<PhysicalSlot> requestNewAllocatedBatchSlot(@Nonnull SlotRequestId slotRequestId, @Nonnull ResourceProfile resourceProfile)
slotRequestId
- identifying the requested slotresourceProfile
- resource profile that specifies the resource requirements for the
requested batch slotvoid disableBatchSlotRequestTimeoutCheck()
AllocatedSlotReport createAllocatedSlotReport(ResourceID taskManagerId)
taskManagerId
- identifies the task managerCopyright © 2014–2021 The Apache Software Foundation. All rights reserved.