org.apache.hadoop.yarn.client.api.async
Class NMClientAsync

java.lang.Object
  extended by org.apache.hadoop.service.AbstractService
      extended by org.apache.hadoop.yarn.client.api.async.NMClientAsync
All Implemented Interfaces:
Closeable, Service

@InterfaceAudience.Public
@InterfaceStability.Stable
public abstract class NMClientAsync
extends AbstractService

NMClientAsync handles communication with all the NodeManagers and provides asynchronous updates on getting responses from them. It maintains a thread pool to communicate with individual NMs where a number of worker threads process requests to NMs by using NMClientImpl. The max size of the thread pool is configurable through YarnConfiguration.NM_CLIENT_ASYNC_THREAD_POOL_MAX_SIZE. It should be used in conjunction with a CallbackHandler. For example

 class MyCallbackHandler implements NMClientAsync.CallbackHandler {
   public void onContainerStarted(ContainerId containerId,
       Map<String, ByteBuffer> allServiceResponse) {
     [post process after the container is started, process the response]
   }

   public void onContainerStatusReceived(ContainerId containerId,
       ContainerStatus containerStatus) {
     [make use of the status of the container]
   }

   public void onContainerStopped(ContainerId containerId) {
     [post process after the container is stopped]
   }

   public void onStartContainerError(
       ContainerId containerId, Throwable t) {
     [handle the raised exception]
   }

   public void onGetContainerStatusError(
       ContainerId containerId, Throwable t) {
     [handle the raised exception]
   }

   public void onStopContainerError(
       ContainerId containerId, Throwable t) {
     [handle the raised exception]
   }
 }
 
 
The client's life-cycle should be managed like the following:
 NMClientAsync asyncClient = 
     NMClientAsync.createNMClientAsync(new MyCallbackhandler());
 asyncClient.init(conf);
 asyncClient.start();
 asyncClient.startContainer(container, containerLaunchContext);
 [... wait for container being started]
 asyncClient.getContainerStatus(container.getId(), container.getNodeId(),
     container.getContainerToken());
 [... handle the status in the callback instance]
 asyncClient.stopContainer(container.getId(), container.getNodeId(),
     container.getContainerToken());
 [... wait for container being stopped]
 asyncClient.stop();
 
 


Field Summary
protected  org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler
           
protected  NMClient client
           
 
Constructor Summary
protected NMClientAsync(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
           
protected NMClientAsync(String name, org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
           
protected NMClientAsync(String name, NMClient client, org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
           
 
Method Summary
static NMClientAsync createNMClientAsync(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
           
 org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler getCallbackHandler()
           
 NMClient getClient()
           
abstract  void getContainerStatusAsync(ContainerId containerId, NodeId nodeId)
           
 void setCallbackHandler(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
           
 void setClient(NMClient client)
           
abstract  void startContainerAsync(Container container, ContainerLaunchContext containerLaunchContext)
           
abstract  void stopContainerAsync(ContainerId containerId, NodeId nodeId)
           
 
Methods inherited from class org.apache.hadoop.service.AbstractService
close, getBlockers, getConfig, getFailureCause, getFailureState, getLifecycleHistory, getName, getServiceState, getStartTime, init, isInState, noteFailure, putBlocker, registerGlobalListener, registerServiceListener, removeBlocker, serviceInit, serviceStart, serviceStop, setConfig, start, stop, toString, unregisterGlobalListener, unregisterServiceListener, waitForServiceToStop
 
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
 

Field Detail

client

protected NMClient client

callbackHandler

protected org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler
Constructor Detail

NMClientAsync

protected NMClientAsync(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)

NMClientAsync

protected NMClientAsync(String name,
                        org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)

NMClientAsync

@InterfaceAudience.Private
protected NMClientAsync(String name,
                                                  NMClient client,
                                                  org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)
Method Detail

createNMClientAsync

public static NMClientAsync createNMClientAsync(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)

startContainerAsync

public abstract void startContainerAsync(Container container,
                                         ContainerLaunchContext containerLaunchContext)

stopContainerAsync

public abstract void stopContainerAsync(ContainerId containerId,
                                        NodeId nodeId)

getContainerStatusAsync

public abstract void getContainerStatusAsync(ContainerId containerId,
                                             NodeId nodeId)

getClient

public NMClient getClient()

setClient

public void setClient(NMClient client)

getCallbackHandler

public org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler getCallbackHandler()

setCallbackHandler

public void setCallbackHandler(org.apache.hadoop.yarn.client.api.async.NMClientAsync.CallbackHandler callbackHandler)


Copyright © 2014 Apache Software Foundation. All Rights Reserved.