public abstract class Container extends Object implements Killable
Represents a container that a worker will run in.
Modifier and Type | Class and Description |
---|---|
static class |
Container.ContainerType |
Modifier and Type | Field and Description |
---|---|
protected LocalAssignment |
assignment |
protected Map<String,Object> |
conf |
protected ContainerMemoryTracker |
containerMemoryTracker |
protected AdvancedFSOps |
ops |
protected int |
port |
protected ResourceIsolationInterface |
resourceIsolationManager |
protected boolean |
runAsUser |
protected String |
supervisorId |
protected int |
supervisorPort |
protected boolean |
symlinksDisabled |
protected Map<String,Object> |
topoConf |
protected String |
topologyId |
protected Container.ContainerType |
type |
protected String |
workerId |
Modifier | Constructor and Description |
---|---|
protected |
Container(Container.ContainerType type,
Map<String,Object> conf,
String supervisorId,
int supervisorPort,
int port,
LocalAssignment assignment,
ResourceIsolationInterface resourceIsolationManager,
String workerId,
Map<String,Object> topoConf,
AdvancedFSOps ops,
StormMetricsRegistry metricsRegistry,
ContainerMemoryTracker containerMemoryTracker)
Create a new Container.
|
Modifier and Type | Method and Description |
---|---|
boolean |
areAllProcessesDead()
Check whether all processes are dead.
|
void |
cleanUp()
Clean up the container.
|
void |
cleanUpForRestart()
Clean up the container partly preparing for restart.
|
protected void |
createArtifactsLink()
Create symlink from the containers directory/artifacts to the artifacts directory.
|
protected void |
createBlobstoreLinks()
Create symlinks for each of the blobs from the container’s directory to corresponding links in the storm dist directory.
|
protected void |
deleteSavedWorkerUser() |
abstract boolean |
didMainProcessExit()
Return true if the main process exited, else false.
|
void |
forceKill()
Kill the processes in this container violently.
|
long |
getMemoryReservationMb()
Get the current memory reservation of this container.
|
long |
getMemoryUsageMb()
Get the current memory usage of this container.
|
long |
getTotalTopologyMemoryReserved(LocalAssignment withUpdatedLimits)
Get the total memory reserved.
|
long |
getTotalTopologyMemoryUsed()
Get the total memory used (on and off heap).
|
long |
getTotalWorkersForThisTopology()
Get the number of workers for this topology.
|
String |
getWorkerId()
Get the id of the container or null if there is no worker id right now.
|
protected String |
getWorkerUser()
Get the user of the worker.
|
boolean |
isMemoryLimitViolated(LocalAssignment withUpdatedLimits)
Check if the container is over its memory limit AND needs to be killed.
|
void |
kill()
Kill the processes in this container nicely.
|
abstract void |
launch()
Launch the process for the first time.
|
LSWorkerHeartbeat |
readHeartbeat()
Read the Heartbeat for the current container.
|
protected Map<String,Object> |
readTopoConf() |
abstract void |
relaunch()
Restart the processes in this container.
|
abstract boolean |
runProfiling(ProfileRequest request,
boolean stop)
Run a profiling request.
|
protected void |
saveWorkerUser(String user) |
protected void |
setup()
Setup the container to run.
|
String |
toString() |
protected void |
updateMemoryAccounting() |
protected void |
writeLogMetadata(String user)
Write out the file used by the log viewer to allow/reject log access.
|
protected final String topologyId
protected final String supervisorId
protected final int supervisorPort
protected final int port
protected final LocalAssignment assignment
protected final AdvancedFSOps ops
protected final ResourceIsolationInterface resourceIsolationManager
protected final boolean symlinksDisabled
protected String workerId
protected Container.ContainerType type
protected ContainerMemoryTracker containerMemoryTracker
protected boolean runAsUser
protected Container(Container.ContainerType type, Map<String,Object> conf, String supervisorId, int supervisorPort, int port, LocalAssignment assignment, ResourceIsolationInterface resourceIsolationManager, String workerId, Map<String,Object> topoConf, AdvancedFSOps ops, StormMetricsRegistry metricsRegistry, ContainerMemoryTracker containerMemoryTracker) throws IOException
Create a new Container.
type
- the type of container being made.conf
- the supervisor configsupervisorId
- the ID of the supervisor this is a part of.supervisorPort
- the thrift server port of the supervisor this is a part of.port
- the port the container is on. Should be <= 0 if only a partial recovery @param assignment the assignment for this container. Should be null if only a partial recovery.resourceIsolationManager
- used to isolate resources for a container can be null if no isolation is used.workerId
- the id of the worker to use. Must not be null if doing a partial recovery.topoConf
- the config of the topology (mostly for testing) if null and not a partial recovery the real conf is read.ops
- file system operations (mostly for testing) if null a new one is mademetricsRegistry
- The metrics registry.containerMemoryTracker
- The shared memory tracker for the supervisor’s containersIOException
- on any error.protected Map<String,Object> readTopoConf() throws IOException
IOException
public void kill() throws IOException
Killable
Kill the processes in this container nicely. kill -15 equivalent
kill
in interface Killable
IOException
- on any errorpublic void forceKill() throws IOException
Killable
Kill the processes in this container violently. kill -9 equivalent
forceKill
in interface Killable
IOException
- on any errorpublic LSWorkerHeartbeat readHeartbeat() throws IOException
Read the Heartbeat for the current container.
IOException
- on any errorpublic boolean areAllProcessesDead() throws IOException
Killable
Check whether all processes are dead.
areAllProcessesDead
in interface Killable
IOException
- on any errorpublic void cleanUp() throws IOException
Killable
Clean up the container. It is not coming back. by default do the same thing as when restarting.
cleanUp
in interface Killable
IOException
- on any errorprotected void setup() throws IOException
Setup the container to run. By default this creates the needed directories/links in the local file system PREREQUISITE: All needed blobs and topology, jars/configs have been downloaded and placed in the appropriate locations
IOException
- on any errorprotected void writeLogMetadata(String user) throws IOException
Write out the file used by the log viewer to allow/reject log access.
user
- the user this is going to run asIOException
- on any errorprotected void createArtifactsLink() throws IOException
Create symlink from the containers directory/artifacts to the artifacts directory.
IOException
- on any errorprotected void createBlobstoreLinks() throws IOException
Create symlinks for each of the blobs from the container’s directory to corresponding links in the storm dist directory.
IOException
- on any error.protected String getWorkerUser() throws IOException
Get the user of the worker.
IOException
- on any errorprotected void saveWorkerUser(String user) throws IOException
IOException
protected void deleteSavedWorkerUser() throws IOException
IOException
public void cleanUpForRestart() throws IOException
Clean up the container partly preparing for restart. By default delete all of the temp directories we are going to get a new worker_id anyways. POST CONDITION: the workerId will be set to null
IOException
- on any errorpublic boolean isMemoryLimitViolated(LocalAssignment withUpdatedLimits) throws IOException
Check if the container is over its memory limit AND needs to be killed. This does not necessarily mean that it just went over the limit.
IOException
- on any errorprotected void updateMemoryAccounting()
public long getTotalTopologyMemoryUsed()
Get the total memory used (on and off heap).
public long getTotalTopologyMemoryReserved(LocalAssignment withUpdatedLimits)
Get the total memory reserved.
withUpdatedLimits
- the local assignment with shared memorypublic long getTotalWorkersForThisTopology()
Get the number of workers for this topology.
public long getMemoryUsageMb()
Get the current memory usage of this container.
public long getMemoryReservationMb()
Get the current memory reservation of this container.
public abstract void launch() throws IOException
Launch the process for the first time. PREREQUISITE: setup has run and passed
IOException
- on any errorpublic abstract void relaunch() throws IOException
Restart the processes in this container. PREREQUISITE: cleanUpForRestart has run and passed
IOException
- on any errorpublic abstract boolean didMainProcessExit()
Return true if the main process exited, else false. This is just best effort return false if unknown.
public abstract boolean runProfiling(ProfileRequest request, boolean stop) throws IOException, InterruptedException
Run a profiling request.
request
- the request to runstop
- is this a stop request?IOException
- on any errorInterruptedException
- if running the command is interrupted.public String getWorkerId()
Get the id of the container or null if there is no worker id right now.
Copyright © 2022 The Apache Software Foundation. All rights reserved.