瀏覽代碼

AMBARI-12506. Server Side stages and tasks should not need server name (ncole)

Nate Cole 10 年之前
父節點
當前提交
8e9f5e8dac
共有 23 個文件被更改,包括 660 次插入419 次删除
  1. 35 21
      ambari-server/src/main/java/org/apache/ambari/server/actionmanager/ActionDBAccessorImpl.java
  2. 35 10
      ambari-server/src/main/java/org/apache/ambari/server/actionmanager/ActionScheduler.java
  3. 5 1
      ambari-server/src/main/java/org/apache/ambari/server/actionmanager/HostRoleCommand.java
  4. 132 71
      ambari-server/src/main/java/org/apache/ambari/server/actionmanager/Stage.java
  5. 23 23
      ambari-server/src/main/java/org/apache/ambari/server/controller/KerberosHelperImpl.java
  6. 6 22
      ambari-server/src/main/java/org/apache/ambari/server/controller/internal/UpgradeResourceProvider.java
  7. 14 7
      ambari-server/src/main/java/org/apache/ambari/server/orm/dao/HostRoleCommandDAO.java
  8. 5 3
      ambari-server/src/main/java/org/apache/ambari/server/orm/entities/HostRoleCommandEntity.java
  9. 1 1
      ambari-server/src/main/java/org/apache/ambari/server/serveraction/ServerActionExecutor.java
  10. 15 14
      ambari-server/src/main/java/org/apache/ambari/server/upgrade/SchemaUpgradeHelper.java
  11. 101 0
      ambari-server/src/main/java/org/apache/ambari/server/upgrade/UpgradeCatalog220.java
  12. 1 1
      ambari-server/src/main/resources/Ambari-DDL-MySQL-CREATE.sql
  13. 1 1
      ambari-server/src/main/resources/Ambari-DDL-Oracle-CREATE.sql
  14. 1 1
      ambari-server/src/main/resources/Ambari-DDL-Postgres-CREATE.sql
  15. 1 1
      ambari-server/src/main/resources/Ambari-DDL-Postgres-EMBEDDED-CREATE.sql
  16. 1 1
      ambari-server/src/main/resources/Ambari-DDL-SQLServer-CREATE.sql
  17. 10 9
      ambari-server/src/test/java/org/apache/ambari/server/actionmanager/StageTest.java
  18. 15 26
      ambari-server/src/test/java/org/apache/ambari/server/actionmanager/TestActionDBAccessorImpl.java
  19. 119 190
      ambari-server/src/test/java/org/apache/ambari/server/actionmanager/TestActionScheduler.java
  20. 12 0
      ambari-server/src/test/java/org/apache/ambari/server/controller/internal/UpgradeResourceProviderTest.java
  21. 3 5
      ambari-server/src/test/java/org/apache/ambari/server/serveraction/ServerActionExecutorTest.java
  22. 12 11
      ambari-server/src/test/java/org/apache/ambari/server/stageplanner/TestStagePlanner.java
  23. 112 0
      ambari-server/src/test/java/org/apache/ambari/server/upgrade/UpgradeCatalog220Test.java

+ 35 - 21
ambari-server/src/main/java/org/apache/ambari/server/actionmanager/ActionDBAccessorImpl.java

@@ -49,6 +49,7 @@ import org.apache.ambari.server.orm.entities.StageEntity;
 import org.apache.ambari.server.state.Clusters;
 import org.apache.ambari.server.state.Clusters;
 import org.apache.ambari.server.state.Host;
 import org.apache.ambari.server.state.Host;
 import org.apache.ambari.server.utils.StageUtils;
 import org.apache.ambari.server.utils.StageUtils;
+import org.apache.commons.lang.StringUtils;
 import org.slf4j.Logger;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 import org.slf4j.LoggerFactory;
 
 
@@ -262,35 +263,46 @@ public class ActionDBAccessorImpl implements ActionDBAccessor {
         HostRoleCommandEntity hostRoleCommandEntity = hostRoleCommand.constructNewPersistenceEntity();
         HostRoleCommandEntity hostRoleCommandEntity = hostRoleCommand.constructNewPersistenceEntity();
         hostRoleCommandEntity.setStage(stageEntity);
         hostRoleCommandEntity.setStage(stageEntity);
 
 
-        HostEntity hostEntity = hostDAO.findById(hostRoleCommandEntity.getHostId());
-        if (hostEntity == null) {
-          String msg = String.format("Host %s doesn't exist in database", hostRoleCommandEntity.getHostName());
-          LOG.error(msg);
-          throw new AmbariException(msg);
-        }
-        hostRoleCommandEntity.setHostEntity(hostEntity);
+        HostEntity hostEntity = null;
+
         hostRoleCommandDAO.create(hostRoleCommandEntity);
         hostRoleCommandDAO.create(hostRoleCommandEntity);
 
 
         assert hostRoleCommandEntity.getTaskId() != null;
         assert hostRoleCommandEntity.getTaskId() != null;
         hostRoleCommand.setTaskId(hostRoleCommandEntity.getTaskId());
         hostRoleCommand.setTaskId(hostRoleCommandEntity.getTaskId());
 
 
-        try {
-          // Get the in-memory host object and its prefix to construct the output and error log paths.
-          Host hostObject = clusters.getHost(hostRoleCommandEntity.getHostName());
-          String prefix = hostObject.getPrefix();
-          if (null != prefix && !prefix.isEmpty()) {
-            if (!prefix.endsWith("/")) {
-              prefix = prefix + "/";
+        String prefix = "";
+        String output = "output-" + hostRoleCommandEntity.getTaskId() + ".txt";
+        String error = "errors-" + hostRoleCommandEntity.getTaskId() + ".txt";
+
+        if (null != hostRoleCommandEntity.getHostId()) {
+          hostEntity = hostDAO.findById(hostRoleCommandEntity.getHostId());
+          if (hostEntity == null) {
+            String msg = String.format("Host %s doesn't exist in database", hostRoleCommandEntity.getHostName());
+            LOG.error(msg);
+            throw new AmbariException(msg);
+          }
+          hostRoleCommandEntity.setHostEntity(hostEntity);
+
+          try {
+            // Get the in-memory host object and its prefix to construct the output and error log paths.
+            Host hostObject = clusters.getHost(hostEntity.getHostName());
+
+            if (!StringUtils.isBlank(hostObject.getPrefix())) {
+              prefix = hostObject.getPrefix();
+              if (!prefix.endsWith("/")) {
+                prefix = prefix + "/";
+              }
             }
             }
-            hostRoleCommand.setOutputLog(prefix + "output-" + hostRoleCommandEntity.getTaskId() + ".txt");
-            hostRoleCommand.setErrorLog(prefix + "errors-" + hostRoleCommandEntity.getTaskId() + ".txt");
-            hostRoleCommandEntity.setOutputLog(hostRoleCommand.getOutputLog());
-            hostRoleCommandEntity.setErrorLog(hostRoleCommand.getErrorLog());
+          } catch (AmbariException e) {
+            LOG.warn("Exception in getting prefix for host and setting output and error log files.  Using no prefix");
           }
           }
-        } catch (AmbariException e) {
-          LOG.warn("Exception in getting prefix for host and setting output and error log files.");
         }
         }
 
 
+        hostRoleCommand.setOutputLog(prefix + output);
+        hostRoleCommand.setErrorLog(prefix + error);
+        hostRoleCommandEntity.setOutputLog(hostRoleCommand.getOutputLog());
+        hostRoleCommandEntity.setErrorLog(hostRoleCommand.getErrorLog());
+
         ExecutionCommandEntity executionCommandEntity = hostRoleCommand.constructExecutionCommandEntity();
         ExecutionCommandEntity executionCommandEntity = hostRoleCommand.constructExecutionCommandEntity();
         executionCommandEntity.setHostRoleCommand(hostRoleCommandEntity);
         executionCommandEntity.setHostRoleCommand(hostRoleCommandEntity);
 
 
@@ -299,7 +311,9 @@ public class ActionDBAccessorImpl implements ActionDBAccessor {
 
 
         executionCommandDAO.create(hostRoleCommandEntity.getExecutionCommand());
         executionCommandDAO.create(hostRoleCommandEntity.getExecutionCommand());
         hostRoleCommandDAO.merge(hostRoleCommandEntity);
         hostRoleCommandDAO.merge(hostRoleCommandEntity);
-        hostDAO.merge(hostEntity);
+        if (null != hostEntity) {
+          hostDAO.merge(hostEntity);
+        }
       }
       }
 
 
       for (RoleSuccessCriteriaEntity roleSuccessCriteriaEntity : stageEntity.getRoleSuccessCriterias()) {
       for (RoleSuccessCriteriaEntity roleSuccessCriteriaEntity : stageEntity.getRoleSuccessCriterias()) {

+ 35 - 10
ambari-server/src/main/java/org/apache/ambari/server/actionmanager/ActionScheduler.java

@@ -89,7 +89,7 @@ class ActionScheduler implements Runnable {
   private final ActionDBAccessor db;
   private final ActionDBAccessor db;
   private final short maxAttempts;
   private final short maxAttempts;
   private final ActionQueue actionQueue;
   private final ActionQueue actionQueue;
-  private final Clusters fsmObject;
+  private final Clusters clusters;
   private final AmbariEventPublisher ambariEventPublisher;
   private final AmbariEventPublisher ambariEventPublisher;
   private boolean taskTimeoutAdjustment = true;
   private boolean taskTimeoutAdjustment = true;
   private final HostsMap hostsMap;
   private final HostsMap hostsMap;
@@ -135,7 +135,7 @@ class ActionScheduler implements Runnable {
     actionTimeout = actionTimeoutMilliSec;
     actionTimeout = actionTimeoutMilliSec;
     this.db = db;
     this.db = db;
     this.actionQueue = actionQueue;
     this.actionQueue = actionQueue;
-    this.fsmObject = fsmObject;
+    this.clusters = fsmObject;
     this.ambariEventPublisher = ambariEventPublisher;
     this.ambariEventPublisher = ambariEventPublisher;
     this.maxAttempts = (short) maxAttempts;
     this.maxAttempts = (short) maxAttempts;
     serverActionExecutor = new ServerActionExecutor(db, sleepTimeMilliSec);
     serverActionExecutor = new ServerActionExecutor(db, sleepTimeMilliSec);
@@ -338,7 +338,7 @@ class ActionScheduler implements Runnable {
         Map<ExecutionCommand, String> commandsToAbort = new HashMap<ExecutionCommand, String>();
         Map<ExecutionCommand, String> commandsToAbort = new HashMap<ExecutionCommand, String>();
         if (!eventMap.isEmpty()) {
         if (!eventMap.isEmpty()) {
           LOG.debug("==> processing {} serviceComponentHostEvents...", eventMap.size());
           LOG.debug("==> processing {} serviceComponentHostEvents...", eventMap.size());
-          Cluster cluster = fsmObject.getCluster(stage.getClusterName());
+          Cluster cluster = clusters.getCluster(stage.getClusterName());
           if (cluster != null) {
           if (cluster != null) {
             Map<ServiceComponentHostEvent, String> failedEvents = cluster.processServiceComponentHostEvents(eventMap);
             Map<ServiceComponentHostEvent, String> failedEvents = cluster.processServiceComponentHostEvents(eventMap);
 
 
@@ -524,14 +524,22 @@ class ActionScheduler implements Runnable {
 
 
     Cluster cluster = null;
     Cluster cluster = null;
     if (null != s.getClusterName()) {
     if (null != s.getClusterName()) {
-      cluster = fsmObject.getCluster(s.getClusterName());
+      cluster = clusters.getCluster(s.getClusterName());
     }
     }
 
 
     for (String host : s.getHosts()) {
     for (String host : s.getHosts()) {
+
       List<ExecutionCommandWrapper> commandWrappers = s.getExecutionCommands(host);
       List<ExecutionCommandWrapper> commandWrappers = s.getExecutionCommands(host);
-      Host hostObj = fsmObject.getHost(host);
+      Host hostObj = null;
+      try {
+        hostObj = clusters.getHost(host);
+      } catch (AmbariException e) {
+        LOG.debug("Host {} not found, stage is likely a server side action", host);
+      }
+
       int i_my = 0;
       int i_my = 0;
       LOG.trace("===>host=" + host);
       LOG.trace("===>host=" + host);
+
       for(ExecutionCommandWrapper wrapper : commandWrappers) {
       for(ExecutionCommandWrapper wrapper : commandWrappers) {
         ExecutionCommand c = wrapper.getExecutionCommand();
         ExecutionCommand c = wrapper.getExecutionCommand();
         String roleStr = c.getRole();
         String roleStr = c.getRole();
@@ -683,7 +691,7 @@ class ActionScheduler implements Runnable {
                                        boolean ignoreTransitionException) {
                                        boolean ignoreTransitionException) {
 
 
     try {
     try {
-      Cluster cluster = fsmObject.getCluster(clusterName);
+      Cluster cluster = clusters.getCluster(clusterName);
 
 
       ServiceComponentHostOpFailedEvent failedEvent =
       ServiceComponentHostOpFailedEvent failedEvent =
         new ServiceComponentHostOpFailedEvent(componentName,
         new ServiceComponentHostOpFailedEvent(componentName,
@@ -749,6 +757,17 @@ class ActionScheduler implements Runnable {
     return roleStats;
     return roleStats;
   }
   }
 
 
+  /**
+   * Checks if timeout is required.
+   * @param status      the status of the current role
+   * @param stage       the stage
+   * @param host        the host object; can be {@code null} for server-side tasks
+   * @param role        the role
+   * @param currentTime the current
+   * @param taskTimeout the amount of time to determine timeout
+   * @return {@code true} if timeout is needed
+   * @throws AmbariException
+   */
   private boolean timeOutActionNeeded(HostRoleStatus status, Stage stage,
   private boolean timeOutActionNeeded(HostRoleStatus status, Stage stage,
       Host host, String role, long currentTime, long taskTimeout) throws
       Host host, String role, long currentTime, long taskTimeout) throws
     AmbariException {
     AmbariException {
@@ -756,17 +775,23 @@ class ActionScheduler implements Runnable {
         ( ! status.equals(HostRoleStatus.IN_PROGRESS) )) {
         ( ! status.equals(HostRoleStatus.IN_PROGRESS) )) {
       return false;
       return false;
     }
     }
+
     // Fast fail task if host state is unknown
     // Fast fail task if host state is unknown
-    if (host.getState().equals(HostState.HEARTBEAT_LOST)) {
+    if (null != host && host.getState().equals(HostState.HEARTBEAT_LOST)) {
       LOG.debug("Timing out action since agent is not heartbeating.");
       LOG.debug("Timing out action since agent is not heartbeating.");
       return true;
       return true;
     }
     }
+
+    // tasks are held in a variety of in-memory maps that require a hostname key
+    // host being null is ok - that means it's a server-side task
+    String hostName = (null == host) ? null : host.getHostName();
+
     // If we have other command in progress for this stage do not timeout this one
     // If we have other command in progress for this stage do not timeout this one
-    if (hasCommandInProgress(stage, host.getHostName())
+    if (hasCommandInProgress(stage, hostName)
             && !status.equals(HostRoleStatus.IN_PROGRESS)) {
             && !status.equals(HostRoleStatus.IN_PROGRESS)) {
       return false;
       return false;
     }
     }
-    if (currentTime > stage.getLastAttemptTime(host.getHostName(), role)
+    if (currentTime > stage.getLastAttemptTime(hostName, role)
         + taskTimeout) {
         + taskTimeout) {
       return true;
       return true;
     }
     }
@@ -945,7 +970,7 @@ class ActionScheduler implements Runnable {
       // "Distribute repositories/install packages" action has been issued
       // "Distribute repositories/install packages" action has been issued
       // against a concrete host without binding to a cluster)
       // against a concrete host without binding to a cluster)
       Long clusterId = clusterName != null ?
       Long clusterId = clusterName != null ?
-              fsmObject.getCluster(clusterName).getClusterId() : null;
+              clusters.getCluster(clusterName).getClusterId() : null;
       ActionFinalReportReceivedEvent event = new ActionFinalReportReceivedEvent(
       ActionFinalReportReceivedEvent event = new ActionFinalReportReceivedEvent(
               clusterId, hostname, null,
               clusterId, hostname, null,
               role);
               role);

+ 5 - 1
ambari-server/src/main/java/org/apache/ambari/server/actionmanager/HostRoleCommand.java

@@ -137,7 +137,11 @@ public class HostRoleCommand {
     taskId = hostRoleCommandEntity.getTaskId();
     taskId = hostRoleCommandEntity.getTaskId();
     stageId = hostRoleCommandEntity.getStage().getStageId();
     stageId = hostRoleCommandEntity.getStage().getStageId();
     requestId = hostRoleCommandEntity.getStage().getRequestId();
     requestId = hostRoleCommandEntity.getStage().getRequestId();
-    hostId = hostRoleCommandEntity.getHostId();
+
+    if (null != hostRoleCommandEntity.getHostEntity()) {
+      hostId = hostRoleCommandEntity.getHostId();
+    }
+
     hostName = hostRoleCommandEntity.getHostName();
     hostName = hostRoleCommandEntity.getHostName();
     role = hostRoleCommandEntity.getRole();
     role = hostRoleCommandEntity.getRole();
     status = hostRoleCommandEntity.getStatus();
     status = hostRoleCommandEntity.getStatus();

+ 132 - 71
ambari-server/src/main/java/org/apache/ambari/server/actionmanager/Stage.java

@@ -57,6 +57,14 @@ import com.google.inject.persist.Transactional;
 //required to persist an action.
 //required to persist an action.
 public class Stage {
 public class Stage {
 
 
+  /**
+   * Used because in-memory storage of commands requires a hostname for maps
+   * when the underlying store does not (host_id is {@code null}).  We also
+   * don't want stages getting confused with Ambari vs cluster hosts, so
+   * don't use {@link StageUtils#getHostName()}
+   */
+  private static final String INTERNAL_HOSTNAME = "_internal_ambari";
+
   private static Logger LOG = LoggerFactory.getLogger(Stage.class);
   private static Logger LOG = LoggerFactory.getLogger(Stage.class);
   private final long requestId;
   private final long requestId;
   private String clusterName;
   private String clusterName;
@@ -86,12 +94,6 @@ public class Stage {
   @Inject
   @Inject
   private HostRoleCommandFactory hostRoleCommandFactory;
   private HostRoleCommandFactory hostRoleCommandFactory;
 
 
-  @Inject
-  private HostRoleCommandDAO hostRoleCommandDAO;
-
-  @Inject
-  private ActionDBAccessor dbAccessor;
-
   @AssistedInject
   @AssistedInject
   public Stage(@Assisted long requestId,
   public Stage(@Assisted long requestId,
       @Assisted("logDir") String logDir,
       @Assisted("logDir") String logDir,
@@ -119,8 +121,6 @@ public class Stage {
   public Stage(@Assisted StageEntity stageEntity, HostRoleCommandDAO hostRoleCommandDAO,
   public Stage(@Assisted StageEntity stageEntity, HostRoleCommandDAO hostRoleCommandDAO,
                ActionDBAccessor dbAccessor, Clusters clusters, HostRoleCommandFactory hostRoleCommandFactory) {
                ActionDBAccessor dbAccessor, Clusters clusters, HostRoleCommandFactory hostRoleCommandFactory) {
     this.hostRoleCommandFactory = hostRoleCommandFactory;
     this.hostRoleCommandFactory = hostRoleCommandFactory;
-    this.hostRoleCommandDAO = hostRoleCommandDAO;
-    this.dbAccessor = dbAccessor;
 
 
     requestId = stageEntity.getRequestId();
     requestId = stageEntity.getRequestId();
     stageId = stageEntity.getStageId();
     stageId = stageEntity.getStageId();
@@ -146,7 +146,11 @@ public class Stage {
     Collection<HostRoleCommand> commands = dbAccessor.getTasks(taskIds);
     Collection<HostRoleCommand> commands = dbAccessor.getTasks(taskIds);
 
 
     for (HostRoleCommand command : commands) {
     for (HostRoleCommand command : commands) {
-      String hostname = command.getHostName();
+      // !!! some commands won't have a hostname, because they are server-side and
+      // don't hold that information.  In that case, use the special key to
+      // use in the map
+      String hostname = getSafeHost(command.getHostName());
+
       if (!hostRoleCommands.containsKey(hostname)) {
       if (!hostRoleCommands.containsKey(hostname)) {
         hostRoleCommands.put(hostname, new LinkedHashMap<String, HostRoleCommand>());
         hostRoleCommands.put(hostname, new LinkedHashMap<String, HostRoleCommand>());
       }
       }
@@ -356,42 +360,6 @@ public class Stage {
   }
   }
 
 
   /**
   /**
-   * Creates server-side execution command.
-   * <p/>
-   * The action name for this command is expected to be the classname of a
-   * {@link org.apache.ambari.server.serveraction.ServerAction} implementation which will be
-   * instantiated and invoked as needed.
-   *
-   * @param actionName    a String declaring the action name (in the form of a classname) to execute
-   * @param role          the Role for this command
-   * @param command       the RoleCommand for this command
-   * @param clusterName   a String identifying the cluster on which to to execute this command
-   * @param event         a ServiceComponentHostServerActionEvent
-   * @param commandParams a Map of String to String data used to pass to the action - this may be
-   *                      empty or null if no data is relevant
-   * @param commandDetail a String declaring a descriptive name to pass to the action - null or an
-   *                      empty string indicates no value is to be set
-   * @param configTags    a Map of configuration tags to set for this command - if null, no
-   *                      configurations will be available for the command
-   * @param timeout       an Integer declaring the timeout for this action - if null, a default
-   * @param retryAllowed   indicates whether retry after failure is allowed
-   */
-  public synchronized void addServerActionCommand(String actionName, Role role, RoleCommand command,
-                                                  String clusterName, ServiceComponentHostServerActionEvent event,
-                                                  @Nullable Map<String, String> commandParams,
-                                                  @Nullable String commandDetail,
-                                                  @Nullable Map<String, Map<String,String>> configTags,
-                                                  @Nullable Integer timeout,
-                                                  boolean retryAllowed) {
-
-    addServerActionCommand(actionName, null, role, command, clusterName, StageUtils.getHostName(), event,
-        commandParams, commandDetail, configTags, timeout, retryAllowed);
-  }
-
-  /**
-   * THIS METHOD IS TO WORKAROUND A BUG!  The assumption of the framework
-   * is that the Ambari Server is installed on a host WITHIN the cluster, which
-   * is not always true.  This method adds a host parameter.
    * <p/>
    * <p/>
    * Creates server-side execution command.
    * Creates server-side execution command.
    * <p/>
    * <p/>
@@ -404,7 +372,6 @@ public class Stage {
    * @param role          the Role for this command
    * @param role          the Role for this command
    * @param command       the RoleCommand for this command
    * @param command       the RoleCommand for this command
    * @param clusterName   a String identifying the cluster on which to to execute this command
    * @param clusterName   a String identifying the cluster on which to to execute this command
-   * @param hostName      the name of the host
    * @param event         a ServiceComponentHostServerActionEvent
    * @param event         a ServiceComponentHostServerActionEvent
    * @param commandParams a Map of String to String data used to pass to the action - this may be
    * @param commandParams a Map of String to String data used to pass to the action - this may be
 *                      empty or null if no data is relevant
 *                      empty or null if no data is relevant
@@ -416,16 +383,17 @@ public class Stage {
    * @param retryAllowed  indicates whether retry after failure is allowed
    * @param retryAllowed  indicates whether retry after failure is allowed
    */
    */
   public synchronized void addServerActionCommand(String actionName,
   public synchronized void addServerActionCommand(String actionName,
-                                                  @Nullable  String userName,
+                                                  @Nullable String userName,
                                                   Role role, RoleCommand command,
                                                   Role role, RoleCommand command,
-                                                  String clusterName, String hostName,
+                                                  String clusterName,
                                                   ServiceComponentHostServerActionEvent event,
                                                   ServiceComponentHostServerActionEvent event,
                                                   @Nullable Map<String, String> commandParams,
                                                   @Nullable Map<String, String> commandParams,
                                                   @Nullable String commandDetail,
                                                   @Nullable String commandDetail,
                                                   @Nullable Map<String, Map<String, String>> configTags,
                                                   @Nullable Map<String, Map<String, String>> configTags,
                                                   @Nullable Integer timeout, boolean retryAllowed) {
                                                   @Nullable Integer timeout, boolean retryAllowed) {
+
     ExecutionCommandWrapper commandWrapper =
     ExecutionCommandWrapper commandWrapper =
-        addGenericExecutionCommand(clusterName, hostName, role, command, event, retryAllowed);
+        addGenericExecutionCommand(clusterName, INTERNAL_HOSTNAME, role, command, event, retryAllowed);
 
 
     ExecutionCommand cmd = commandWrapper.getExecutionCommand();
     ExecutionCommand cmd = commandWrapper.getExecutionCommand();
 
 
@@ -457,7 +425,7 @@ public class Stage {
     cmd.setRoleParams(roleParams);
     cmd.setRoleParams(roleParams);
 
 
     if(commandDetail != null) {
     if(commandDetail != null) {
-      HostRoleCommand hostRoleCommand = getHostRoleCommand(hostName, role.toString());
+      HostRoleCommand hostRoleCommand = getHostRoleCommand(INTERNAL_HOSTNAME, role.toString());
       if (hostRoleCommand != null) {
       if (hostRoleCommand != null) {
         hostRoleCommand.setCommandDetail(commandDetail);
         hostRoleCommand.setCommandDetail(commandDetail);
         hostRoleCommand.setCustomCommandName(actionName);
         hostRoleCommand.setCustomCommandName(actionName);
@@ -534,25 +502,49 @@ public class Stage {
     return requestContext;
     return requestContext;
   }
   }
 
 
-  public long getLastAttemptTime(String host, String role) {
-    return hostRoleCommands.get(host).get(role).getLastAttemptTime();
+  /**
+   * @param hostname  the hostname; {@code null} for a server-side stage
+   * @param role      the role
+   * @return the last attempt time
+   */
+  public long getLastAttemptTime(String hostname, String role) {
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role).getLastAttemptTime();
   }
   }
 
 
-  public short getAttemptCount(String host, String role) {
-    return hostRoleCommands.get(host).get(role).getAttemptCount();
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @return the number of attempts
+   */
+  public short getAttemptCount(String hostname, String role) {
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role).getAttemptCount();
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   */
   public void incrementAttemptCount(String hostname, String role) {
   public void incrementAttemptCount(String hostname, String role) {
-    hostRoleCommands.get(hostname).get(role).incrementAttemptCount();
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).incrementAttemptCount();
   }
   }
 
 
-  public void setLastAttemptTime(String host, String role, long t) {
-    hostRoleCommands.get(host).get(role).setLastAttemptTime(t);
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param t           the last time the role was attempted
+   */
+  public void setLastAttemptTime(String hostname, String role, long t) {
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setLastAttemptTime(t);
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @return            the wrapper
+   */
   public ExecutionCommandWrapper getExecutionCommandWrapper(String hostname,
   public ExecutionCommandWrapper getExecutionCommandWrapper(String hostname,
       String role) {
       String role) {
-    HostRoleCommand hrc = hostRoleCommands.get(hostname).get(role);
+    HostRoleCommand hrc = hostRoleCommands.get(getSafeHost(hostname)).get(role);
     if (hrc != null) {
     if (hrc != null) {
       return hrc.getExecutionCommandWrapper();
       return hrc.getExecutionCommandWrapper();
     } else {
     } else {
@@ -560,47 +552,95 @@ public class Stage {
     }
     }
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @return  the list of commands for the host
+   */
   public List<ExecutionCommandWrapper> getExecutionCommands(String hostname) {
   public List<ExecutionCommandWrapper> getExecutionCommands(String hostname) {
     checkWrappersLoaded();
     checkWrappersLoaded();
-    return commandsToSend.get(hostname);
+    return commandsToSend.get(getSafeHost(hostname));
   }
   }
 
 
+/**
+ * @param hostname    the hostname; {@code null} for a server-side stage
+ * @param role        the role
+ * @return the start time for the task
+ */
   public long getStartTime(String hostname, String role) {
   public long getStartTime(String hostname, String role) {
-    return hostRoleCommands.get(hostname).get(role).getStartTime();
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role).getStartTime();
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param startTime   the start time
+   */
   public void setStartTime(String hostname, String role, long startTime) {
   public void setStartTime(String hostname, String role, long startTime) {
-    hostRoleCommands.get(hostname).get(role).setStartTime(startTime);
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setStartTime(startTime);
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @return the status
+   */
   public HostRoleStatus getHostRoleStatus(String hostname, String role) {
   public HostRoleStatus getHostRoleStatus(String hostname, String role) {
-    return hostRoleCommands.get(hostname).get(role).getStatus();
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role).getStatus();
   }
   }
 
 
-  public void setHostRoleStatus(String host, String role,
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param status      the status
+   */
+  public void setHostRoleStatus(String hostname, String role,
       HostRoleStatus status) {
       HostRoleStatus status) {
-    hostRoleCommands.get(host).get(role).setStatus(status);
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setStatus(status);
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param roleStr     the role name
+   * @return the wrapper event
+   */
   public ServiceComponentHostEventWrapper getFsmEvent(String hostname, String roleStr) {
   public ServiceComponentHostEventWrapper getFsmEvent(String hostname, String roleStr) {
-    return hostRoleCommands.get(hostname).get(roleStr).getEvent();
+    return hostRoleCommands.get(getSafeHost(hostname)).get(roleStr).getEvent();
   }
   }
 
 
-
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param exitCode    the exit code
+   */
   public void setExitCode(String hostname, String role, int exitCode) {
   public void setExitCode(String hostname, String role, int exitCode) {
-    hostRoleCommands.get(hostname).get(role).setExitCode(exitCode);
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setExitCode(exitCode);
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @return the exit code
+   */
   public int getExitCode(String hostname, String role) {
   public int getExitCode(String hostname, String role) {
-    return hostRoleCommands.get(hostname).get(role).getExitCode();
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role).getExitCode();
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param stdErr      the standard error string
+   */
   public void setStderr(String hostname, String role, String stdErr) {
   public void setStderr(String hostname, String role, String stdErr) {
-    hostRoleCommands.get(hostname).get(role).setStderr(stdErr);
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setStderr(stdErr);
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @param stdOut      the standard output string
+   */
   public void setStdout(String hostname, String role, String stdOut) {
   public void setStdout(String hostname, String role, String stdOut) {
-    hostRoleCommands.get(hostname).get(role).setStdout(stdOut);
+    hostRoleCommands.get(getSafeHost(hostname)).get(role).setStdout(stdOut);
   }
   }
 
 
   public synchronized boolean isStageInProgress() {
   public synchronized boolean isStageInProgress() {
@@ -655,10 +695,16 @@ public class Stage {
    * This method should be used only in stage planner. To add
    * This method should be used only in stage planner. To add
    * a new execution command use
    * a new execution command use
    * {@link #addHostRoleExecutionCommand(String, org.apache.ambari.server.Role, org.apache.ambari.server.RoleCommand, org.apache.ambari.server.state.ServiceComponentHostEvent, String, String, boolean)}
    * {@link #addHostRoleExecutionCommand(String, org.apache.ambari.server.Role, org.apache.ambari.server.RoleCommand, org.apache.ambari.server.state.ServiceComponentHostEvent, String, String, boolean)}
+   * @param origStage the stage
+   * @param hostname  the hostname; {@code null} for a server-side stage
+   * @param r         the role
    */
    */
   public synchronized void addExecutionCommandWrapper(Stage origStage,
   public synchronized void addExecutionCommandWrapper(Stage origStage,
       String hostname, Role r) {
       String hostname, Role r) {
     //used on stage creation only, no need to check if wrappers loaded
     //used on stage creation only, no need to check if wrappers loaded
+
+    hostname = getSafeHost(hostname);
+
     String role = r.toString();
     String role = r.toString();
     if (commandsToSend.get(hostname) == null) {
     if (commandsToSend.get(hostname) == null) {
       commandsToSend.put(hostname, new ArrayList<ExecutionCommandWrapper>());
       commandsToSend.put(hostname, new ArrayList<ExecutionCommandWrapper>());
@@ -673,8 +719,13 @@ public class Stage {
         origStage.getHostRoleCommand(hostname, role));
         origStage.getHostRoleCommand(hostname, role));
   }
   }
 
 
+  /**
+   * @param hostname    the hostname; {@code null} for a server-side stage
+   * @param role        the role
+   * @return the role command
+   */
   public HostRoleCommand getHostRoleCommand(String hostname, String role) {
   public HostRoleCommand getHostRoleCommand(String hostname, String role) {
-    return hostRoleCommands.get(hostname).get(role);
+    return hostRoleCommands.get(getSafeHost(hostname)).get(role);
   }
   }
 
 
   /**
   /**
@@ -761,4 +812,14 @@ public class Stage {
     builder.append("STAGE DESCRIPTION END\n");
     builder.append("STAGE DESCRIPTION END\n");
     return builder.toString();
     return builder.toString();
   }
   }
+
+  /**
+   * Helper to make sure the hostname is non-null for internal command map.
+   * @param hostname  the hostname for the map key
+   * @return the hostname when not {@code null}, otherwise {@link #INTERNAL_HOSTNAME}
+   */
+  private static String getSafeHost(String hostname) {
+    return (null == hostname) ? INTERNAL_HOSTNAME : hostname;
+  }
+
 }
 }

+ 23 - 23
ambari-server/src/main/java/org/apache/ambari/server/controller/KerberosHelperImpl.java

@@ -18,9 +18,23 @@
 
 
 package org.apache.ambari.server.controller;
 package org.apache.ambari.server.controller;
 
 
-import com.google.inject.Inject;
-import com.google.inject.Singleton;
-import com.google.inject.persist.Transactional;
+import java.io.File;
+import java.io.IOException;
+import java.text.SimpleDateFormat;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Iterator;
+import java.util.List;
+import java.util.Map;
+import java.util.Random;
+import java.util.Set;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.RoleCommand;
@@ -47,14 +61,15 @@ import org.apache.ambari.server.controller.utilities.ClusterControllerHelper;
 import org.apache.ambari.server.controller.utilities.PredicateBuilder;
 import org.apache.ambari.server.controller.utilities.PredicateBuilder;
 import org.apache.ambari.server.metadata.RoleCommandOrder;
 import org.apache.ambari.server.metadata.RoleCommandOrder;
 import org.apache.ambari.server.serveraction.ServerAction;
 import org.apache.ambari.server.serveraction.ServerAction;
+import org.apache.ambari.server.serveraction.kerberos.CleanupServerAction;
 import org.apache.ambari.server.serveraction.kerberos.CreateKeytabFilesServerAction;
 import org.apache.ambari.server.serveraction.kerberos.CreateKeytabFilesServerAction;
 import org.apache.ambari.server.serveraction.kerberos.CreatePrincipalsServerAction;
 import org.apache.ambari.server.serveraction.kerberos.CreatePrincipalsServerAction;
 import org.apache.ambari.server.serveraction.kerberos.DestroyPrincipalsServerAction;
 import org.apache.ambari.server.serveraction.kerberos.DestroyPrincipalsServerAction;
 import org.apache.ambari.server.serveraction.kerberos.FinalizeKerberosServerAction;
 import org.apache.ambari.server.serveraction.kerberos.FinalizeKerberosServerAction;
 import org.apache.ambari.server.serveraction.kerberos.KDCType;
 import org.apache.ambari.server.serveraction.kerberos.KDCType;
-import org.apache.ambari.server.serveraction.kerberos.KerberosIdentityDataFileWriter;
 import org.apache.ambari.server.serveraction.kerberos.KerberosAdminAuthenticationException;
 import org.apache.ambari.server.serveraction.kerberos.KerberosAdminAuthenticationException;
 import org.apache.ambari.server.serveraction.kerberos.KerberosCredential;
 import org.apache.ambari.server.serveraction.kerberos.KerberosCredential;
+import org.apache.ambari.server.serveraction.kerberos.KerberosIdentityDataFileWriter;
 import org.apache.ambari.server.serveraction.kerberos.KerberosIdentityDataFileWriterFactory;
 import org.apache.ambari.server.serveraction.kerberos.KerberosIdentityDataFileWriterFactory;
 import org.apache.ambari.server.serveraction.kerberos.KerberosInvalidConfigurationException;
 import org.apache.ambari.server.serveraction.kerberos.KerberosInvalidConfigurationException;
 import org.apache.ambari.server.serveraction.kerberos.KerberosKDCConnectionException;
 import org.apache.ambari.server.serveraction.kerberos.KerberosKDCConnectionException;
@@ -69,7 +84,6 @@ import org.apache.ambari.server.serveraction.kerberos.PrepareDisableKerberosServ
 import org.apache.ambari.server.serveraction.kerberos.PrepareEnableKerberosServerAction;
 import org.apache.ambari.server.serveraction.kerberos.PrepareEnableKerberosServerAction;
 import org.apache.ambari.server.serveraction.kerberos.PrepareKerberosIdentitiesServerAction;
 import org.apache.ambari.server.serveraction.kerberos.PrepareKerberosIdentitiesServerAction;
 import org.apache.ambari.server.serveraction.kerberos.UpdateKerberosConfigsServerAction;
 import org.apache.ambari.server.serveraction.kerberos.UpdateKerberosConfigsServerAction;
-import org.apache.ambari.server.serveraction.kerberos.CleanupServerAction;
 import org.apache.ambari.server.stageplanner.RoleGraph;
 import org.apache.ambari.server.stageplanner.RoleGraph;
 import org.apache.ambari.server.stageplanner.RoleGraphFactory;
 import org.apache.ambari.server.stageplanner.RoleGraphFactory;
 import org.apache.ambari.server.state.Cluster;
 import org.apache.ambari.server.state.Cluster;
@@ -81,7 +95,6 @@ import org.apache.ambari.server.state.HostState;
 import org.apache.ambari.server.state.SecurityState;
 import org.apache.ambari.server.state.SecurityState;
 import org.apache.ambari.server.state.SecurityType;
 import org.apache.ambari.server.state.SecurityType;
 import org.apache.ambari.server.state.Service;
 import org.apache.ambari.server.state.Service;
-import org.apache.ambari.server.state.ServiceComponent;
 import org.apache.ambari.server.state.ServiceComponentHost;
 import org.apache.ambari.server.state.ServiceComponentHost;
 import org.apache.ambari.server.state.StackId;
 import org.apache.ambari.server.state.StackId;
 import org.apache.ambari.server.state.State;
 import org.apache.ambari.server.state.State;
@@ -101,22 +114,9 @@ import org.apache.commons.lang.StringUtils;
 import org.slf4j.Logger;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 import org.slf4j.LoggerFactory;
 
 
-import java.io.File;
-import java.io.IOException;
-import java.text.SimpleDateFormat;
-import java.util.ArrayList;
-import java.util.Collection;
-import java.util.Collections;
-import java.util.Date;
-import java.util.HashMap;
-import java.util.HashSet;
-import java.util.Iterator;
-import java.util.List;
-import java.util.Map;
-import java.util.Random;
-import java.util.Set;
-import java.util.regex.Matcher;
-import java.util.regex.Pattern;
+import com.google.inject.Inject;
+import com.google.inject.Singleton;
+import com.google.inject.persist.Transactional;
 
 
 @Singleton
 @Singleton
 public class KerberosHelperImpl implements KerberosHelper {
 public class KerberosHelperImpl implements KerberosHelper {
@@ -1653,7 +1653,7 @@ public class KerberosHelperImpl implements KerberosHelper {
                                         Integer timeout) throws AmbariException {
                                         Integer timeout) throws AmbariException {
 
 
     Stage stage = createNewStage(id, cluster, requestId, requestContext, clusterHostInfo, commandParams, hostParams);
     Stage stage = createNewStage(id, cluster, requestId, requestContext, clusterHostInfo, commandParams, hostParams);
-    stage.addServerActionCommand(actionClass.getName(),
+    stage.addServerActionCommand(actionClass.getName(), null,
         Role.AMBARI_SERVER_ACTION,
         Role.AMBARI_SERVER_ACTION,
         RoleCommand.EXECUTE,
         RoleCommand.EXECUTE,
         cluster.getClusterName(),
         cluster.getClusterName(),

+ 6 - 22
ambari-server/src/main/java/org/apache/ambari/server/controller/internal/UpgradeResourceProvider.java

@@ -23,7 +23,6 @@ import static org.apache.ambari.server.agent.ExecutionCommand.KeyNames.VERSION;
 
 
 import java.util.ArrayList;
 import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Arrays;
-import java.util.Collection;
 import java.util.Collections;
 import java.util.Collections;
 import java.util.EnumSet;
 import java.util.EnumSet;
 import java.util.HashMap;
 import java.util.HashMap;
@@ -69,7 +68,6 @@ import org.apache.ambari.server.orm.dao.HostRoleCommandStatusSummaryDTO;
 import org.apache.ambari.server.orm.dao.RepositoryVersionDAO;
 import org.apache.ambari.server.orm.dao.RepositoryVersionDAO;
 import org.apache.ambari.server.orm.dao.RequestDAO;
 import org.apache.ambari.server.orm.dao.RequestDAO;
 import org.apache.ambari.server.orm.dao.UpgradeDAO;
 import org.apache.ambari.server.orm.dao.UpgradeDAO;
-import org.apache.ambari.server.orm.entities.HostEntity;
 import org.apache.ambari.server.orm.entities.RepositoryVersionEntity;
 import org.apache.ambari.server.orm.entities.RepositoryVersionEntity;
 import org.apache.ambari.server.orm.entities.RequestEntity;
 import org.apache.ambari.server.orm.entities.RequestEntity;
 import org.apache.ambari.server.orm.entities.StackEntity;
 import org.apache.ambari.server.orm.entities.StackEntity;
@@ -97,7 +95,6 @@ import org.apache.ambari.server.state.stack.upgrade.StageWrapper;
 import org.apache.ambari.server.state.stack.upgrade.Task;
 import org.apache.ambari.server.state.stack.upgrade.Task;
 import org.apache.ambari.server.state.stack.upgrade.TaskWrapper;
 import org.apache.ambari.server.state.stack.upgrade.TaskWrapper;
 import org.apache.ambari.server.state.svccomphost.ServiceComponentHostServerActionEvent;
 import org.apache.ambari.server.state.svccomphost.ServiceComponentHostServerActionEvent;
-import org.apache.ambari.server.utils.StageUtils;
 import org.apache.commons.lang.StringUtils;
 import org.apache.commons.lang.StringUtils;
 import org.slf4j.Logger;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 import org.slf4j.LoggerFactory;
@@ -688,7 +685,7 @@ public class UpgradeResourceProvider extends AbstractControllerResourceProvider
    * @param direction
    * @param direction
    *          upgrade or downgrade
    *          upgrade or downgrade
    * @param upgradePack
    * @param upgradePack
-   *          upgrade pack used for upgrade or downgrade. This is needed to determine 
+   *          upgrade pack used for upgrade or downgrade. This is needed to determine
    *          which services are effected.
    *          which services are effected.
    * @throws AmbariException
    * @throws AmbariException
    */
    */
@@ -1090,21 +1087,6 @@ public class UpgradeResourceProvider extends AbstractControllerResourceProvider
     String itemDetail = entity.getText();
     String itemDetail = entity.getText();
     String stageText = StringUtils.abbreviate(entity.getText(), 255);
     String stageText = StringUtils.abbreviate(entity.getText(), 255);
 
 
-    String hostName = null;
-    Collection<Long> hostIds = cluster.getAllHostsDesiredConfigs().keySet();
-    if (!hostIds.isEmpty()) {
-      Long hostId = hostIds.iterator().next();
-      HostEntity hostEntity = s_hostDAO.findById(hostId);
-      if (hostEntity != null) {
-        hostName = hostEntity.getHostName();
-      }
-    }
-
-    if (StringUtils.isBlank(hostName)) {
-      throw new AmbariException(
-          "Could not retrieve an arbitrary host name to use for the server-side command.");
-    }
-
     switch (task.getType()) {
     switch (task.getType()) {
       case MANUAL: {
       case MANUAL: {
         ManualTask mt = (ManualTask) task;
         ManualTask mt = (ManualTask) task;
@@ -1175,9 +1157,11 @@ public class UpgradeResourceProvider extends AbstractControllerResourceProvider
     entity.setStageId(Long.valueOf(stageId));
     entity.setStageId(Long.valueOf(stageId));
 
 
     stage.addServerActionCommand(task.getImplementationClass(),
     stage.addServerActionCommand(task.getImplementationClass(),
-        getManagementController().getAuthName(), Role.AMBARI_SERVER_ACTION, RoleCommand.EXECUTE,
-        cluster.getClusterName(), hostName,
-        new ServiceComponentHostServerActionEvent(StageUtils.getHostName(),
+        getManagementController().getAuthName(),
+        Role.AMBARI_SERVER_ACTION,
+        RoleCommand.EXECUTE,
+        cluster.getClusterName(),
+        new ServiceComponentHostServerActionEvent(null,
             System.currentTimeMillis()),
             System.currentTimeMillis()),
         commandParams, itemDetail, null, Integer.valueOf(1200), allowRetry);
         commandParams, itemDetail, null, Integer.valueOf(1200), allowRetry);
 
 

+ 14 - 7
ambari-server/src/main/java/org/apache/ambari/server/orm/dao/HostRoleCommandDAO.java

@@ -32,7 +32,6 @@ import java.util.Map;
 import javax.persistence.EntityManager;
 import javax.persistence.EntityManager;
 import javax.persistence.TypedQuery;
 import javax.persistence.TypedQuery;
 
 
-import org.apache.ambari.server.actionmanager.HostRoleCommand;
 import org.apache.ambari.server.actionmanager.HostRoleStatus;
 import org.apache.ambari.server.actionmanager.HostRoleStatus;
 import org.apache.ambari.server.orm.RequiresSession;
 import org.apache.ambari.server.orm.RequiresSession;
 import org.apache.ambari.server.orm.entities.HostEntity;
 import org.apache.ambari.server.orm.entities.HostEntity;
@@ -239,13 +238,21 @@ public class HostRoleCommandDAO {
 
 
   @RequiresSession
   @RequiresSession
   public List<HostRoleCommandEntity> findByHostRole(String hostName, long requestId, long stageId, String role) {
   public List<HostRoleCommandEntity> findByHostRole(String hostName, long requestId, long stageId, String role) {
-    TypedQuery<HostRoleCommandEntity> query = entityManagerProvider.get().createQuery("SELECT command " +
-        "FROM HostRoleCommandEntity command " +
-        "WHERE command.hostEntity.hostName=?1 AND command.requestId=?2 " +
-        "AND command.stageId=?3 AND command.role=?4 " +
-        "ORDER BY command.taskId", HostRoleCommandEntity.class);
 
 
-    return daoUtils.selectList(query, hostName, requestId, stageId, role);
+    String queryName = (null == hostName) ? "HostRoleCommandEntity.findByHostRoleNullHost" :
+        "HostRoleCommandEntity.findByHostRole";
+
+    TypedQuery<HostRoleCommandEntity> query = entityManagerProvider.get().createNamedQuery(
+        queryName, HostRoleCommandEntity.class);
+
+    if (null != hostName) {
+      query.setParameter("hostName", hostName);
+    }
+    query.setParameter("requestId", requestId);
+    query.setParameter("stageId", stageId);
+    query.setParameter("role", role);
+
+    return daoUtils.selectList(query);
   }
   }
 
 
   @RequiresSession
   @RequiresSession

+ 5 - 3
ambari-server/src/main/java/org/apache/ambari/server/orm/entities/HostRoleCommandEntity.java

@@ -57,7 +57,9 @@ import org.apache.commons.lang.ArrayUtils;
 @NamedQueries({
 @NamedQueries({
     @NamedQuery(name = "HostRoleCommandEntity.findCountByCommandStatuses", query = "SELECT COUNT(command.taskId) FROM HostRoleCommandEntity command WHERE command.status IN :statuses"),
     @NamedQuery(name = "HostRoleCommandEntity.findCountByCommandStatuses", query = "SELECT COUNT(command.taskId) FROM HostRoleCommandEntity command WHERE command.status IN :statuses"),
     @NamedQuery(name = "HostRoleCommandEntity.findByCommandStatuses", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.status IN :statuses ORDER BY command.requestId, command.stageId"),
     @NamedQuery(name = "HostRoleCommandEntity.findByCommandStatuses", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.status IN :statuses ORDER BY command.requestId, command.stageId"),
-    @NamedQuery(name = "HostRoleCommandEntity.findByHostId", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.hostId=:hostId")
+    @NamedQuery(name = "HostRoleCommandEntity.findByHostId", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.hostId=:hostId"),
+    @NamedQuery(name = "HostRoleCommandEntity.findByHostRole", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.hostEntity.hostName=:hostName AND command.requestId=:requestId AND command.stageId=:stageId AND command.role=:role ORDER BY command.taskId"),
+    @NamedQuery(name = "HostRoleCommandEntity.findByHostRoleNullHost", query = "SELECT command FROM HostRoleCommandEntity command WHERE command.hostEntity IS NULL AND command.requestId=:requestId AND command.stageId=:stageId AND command.role=:role")
 })
 })
 public class HostRoleCommandEntity {
 public class HostRoleCommandEntity {
 
 
@@ -76,7 +78,7 @@ public class HostRoleCommandEntity {
   @Basic
   @Basic
   private Long stageId;
   private Long stageId;
 
 
-  @Column(name = "host_id", insertable = false, updatable = false, nullable = false)
+  @Column(name = "host_id", insertable = false, updatable = false, nullable = true)
   @Basic
   @Basic
   private Long hostId;
   private Long hostId;
 
 
@@ -162,7 +164,7 @@ public class HostRoleCommandEntity {
   private StageEntity stage;
   private StageEntity stage;
 
 
   @ManyToOne(cascade = {CascadeType.MERGE, CascadeType.REFRESH})
   @ManyToOne(cascade = {CascadeType.MERGE, CascadeType.REFRESH})
-  @JoinColumn(name = "host_id", referencedColumnName = "host_id", nullable = false)
+  @JoinColumn(name = "host_id", referencedColumnName = "host_id", nullable = true)
   private HostEntity hostEntity;
   private HostEntity hostEntity;
 
 
   @OneToOne(mappedBy = "hostRoleCommandEntity", cascade = CascadeType.REMOVE)
   @OneToOne(mappedBy = "hostRoleCommandEntity", cascade = CascadeType.REMOVE)

+ 1 - 1
ambari-server/src/main/java/org/apache/ambari/server/serveraction/ServerActionExecutor.java

@@ -304,7 +304,7 @@ public class ServerActionExecutor {
       commandReport = createErrorReport("Unknown error condition");
       commandReport = createErrorReport("Unknown error condition");
     }
     }
 
 
-    db.updateHostRoleState(executionCommand.getHostname(), hostRoleCommand.getRequestId(),
+    db.updateHostRoleState(null, hostRoleCommand.getRequestId(),
         hostRoleCommand.getStageId(), executionCommand.getRole(), commandReport);
         hostRoleCommand.getStageId(), executionCommand.getRole(), commandReport);
   }
   }
 
 

+ 15 - 14
ambari-server/src/main/java/org/apache/ambari/server/upgrade/SchemaUpgradeHelper.java

@@ -17,20 +17,6 @@
  */
  */
 package org.apache.ambari.server.upgrade;
 package org.apache.ambari.server.upgrade;
 
 
-import com.google.inject.Guice;
-import com.google.inject.Inject;
-import com.google.inject.Injector;
-import com.google.inject.multibindings.Multibinder;
-import com.google.inject.persist.PersistService;
-import org.apache.ambari.server.AmbariException;
-import org.apache.ambari.server.configuration.Configuration;
-import org.apache.ambari.server.controller.ControllerModule;
-import org.apache.ambari.server.orm.DBAccessor;
-import org.apache.ambari.server.utils.VersionUtils;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-import java.sql.Connection;
 import java.sql.ResultSet;
 import java.sql.ResultSet;
 import java.sql.SQLException;
 import java.sql.SQLException;
 import java.sql.Statement;
 import java.sql.Statement;
@@ -40,6 +26,20 @@ import java.util.List;
 import java.util.Properties;
 import java.util.Properties;
 import java.util.Set;
 import java.util.Set;
 
 
+import org.apache.ambari.server.AmbariException;
+import org.apache.ambari.server.configuration.Configuration;
+import org.apache.ambari.server.controller.ControllerModule;
+import org.apache.ambari.server.orm.DBAccessor;
+import org.apache.ambari.server.utils.VersionUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.google.inject.Guice;
+import com.google.inject.Inject;
+import com.google.inject.Injector;
+import com.google.inject.multibindings.Multibinder;
+import com.google.inject.persist.PersistService;
+
 public class SchemaUpgradeHelper {
 public class SchemaUpgradeHelper {
   private static final Logger LOG = LoggerFactory.getLogger
   private static final Logger LOG = LoggerFactory.getLogger
     (SchemaUpgradeHelper.class);
     (SchemaUpgradeHelper.class);
@@ -177,6 +177,7 @@ public class SchemaUpgradeHelper {
       catalogBinder.addBinding().to(UpgradeCatalog170.class);
       catalogBinder.addBinding().to(UpgradeCatalog170.class);
       catalogBinder.addBinding().to(UpgradeCatalog200.class);
       catalogBinder.addBinding().to(UpgradeCatalog200.class);
       catalogBinder.addBinding().to(UpgradeCatalog210.class);
       catalogBinder.addBinding().to(UpgradeCatalog210.class);
+      catalogBinder.addBinding().to(UpgradeCatalog220.class);
       catalogBinder.addBinding().to(FinalUpgradeCatalog.class);
       catalogBinder.addBinding().to(FinalUpgradeCatalog.class);
     }
     }
   }
   }

+ 101 - 0
ambari-server/src/main/java/org/apache/ambari/server/upgrade/UpgradeCatalog220.java

@@ -0,0 +1,101 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.ambari.server.upgrade;
+
+import java.sql.SQLException;
+
+import org.apache.ambari.server.AmbariException;
+import org.apache.ambari.server.orm.DBAccessor.DBColumnInfo;
+import org.apache.ambari.server.orm.dao.DaoUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import com.google.inject.Inject;
+import com.google.inject.Injector;
+
+
+/**
+ * Upgrade catalog for version 2.2.0.
+ */
+public class UpgradeCatalog220 extends AbstractUpgradeCatalog {
+  private static final String HOST_ROLE_COMMAND_TABLE = "host_role_command";
+  private static final String HOST_ID_COL = "host_id";
+
+  @Inject
+  DaoUtils daoUtils;
+
+
+  /**
+   * {@inheritDoc}
+   */
+  @Override
+  public String getSourceVersion() {
+    return "2.1.0";
+  }
+
+  /**
+   * {@inheritDoc}
+   */
+  @Override
+  public String getTargetVersion() {
+    return "2.2.0";
+  }
+
+  /**
+   * Logger.
+   */
+  private static final Logger LOG = LoggerFactory.getLogger(UpgradeCatalog220.class);
+
+  // ----- Constructors ------------------------------------------------------
+
+  /**
+   * Don't forget to register new UpgradeCatalogs in {@link org.apache.ambari.server.upgrade.SchemaUpgradeHelper.UpgradeHelperModule#configure()}
+   * @param injector Guice injector to track dependencies and uses bindings to inject them.
+   */
+  @Inject
+  public UpgradeCatalog220(Injector injector) {
+    super(injector);
+    this.injector = injector;
+
+    daoUtils = injector.getInstance(DaoUtils.class);
+  }
+
+  // ----- AbstractUpgradeCatalog --------------------------------------------
+
+  /**
+   * {@inheritDoc}
+   */
+  @Override
+  protected void executeDDLUpdates() throws AmbariException, SQLException {
+
+    dbAccessor.alterColumn(HOST_ROLE_COMMAND_TABLE, new DBColumnInfo(HOST_ID_COL, Long.class, null, null, true));
+  }
+
+  @Override
+  protected void executePreDMLUpdates() throws AmbariException, SQLException {
+  }
+
+  @Override
+  protected void executeDMLUpdates() throws AmbariException, SQLException {
+  }
+
+
+  // ----- UpgradeCatalog ----------------------------------------------------
+
+}

+ 1 - 1
ambari-server/src/main/resources/Ambari-DDL-MySQL-CREATE.sql

@@ -223,7 +223,7 @@ CREATE TABLE host_role_command (
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   event LONGTEXT NOT NULL,
   event LONGTEXT NOT NULL,
   exitcode INTEGER NOT NULL,
   exitcode INTEGER NOT NULL,
-  host_id BIGINT NOT NULL,
+  host_id BIGINT,
   last_attempt_time BIGINT NOT NULL,
   last_attempt_time BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   role VARCHAR(255),
   role VARCHAR(255),

+ 1 - 1
ambari-server/src/main/resources/Ambari-DDL-Oracle-CREATE.sql

@@ -213,7 +213,7 @@ CREATE TABLE host_role_command (
   retry_allowed NUMBER(1) DEFAULT 0 NOT NULL,
   retry_allowed NUMBER(1) DEFAULT 0 NOT NULL,
   event CLOB NULL,
   event CLOB NULL,
   exitcode NUMBER(10) NOT NULL,
   exitcode NUMBER(10) NOT NULL,
-  host_id NUMBER(19) NOT NULL,
+  host_id NUMBER(19),
   last_attempt_time NUMBER(19) NOT NULL,
   last_attempt_time NUMBER(19) NOT NULL,
   request_id NUMBER(19) NOT NULL,
   request_id NUMBER(19) NOT NULL,
   role VARCHAR2(255) NULL,
   role VARCHAR2(255) NULL,

+ 1 - 1
ambari-server/src/main/resources/Ambari-DDL-Postgres-CREATE.sql

@@ -229,7 +229,7 @@ CREATE TABLE host_role_command (
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   event VARCHAR(32000) NOT NULL,
   event VARCHAR(32000) NOT NULL,
   exitcode INTEGER NOT NULL,
   exitcode INTEGER NOT NULL,
-  host_id BIGINT NOT NULL,
+  host_id BIGINT,
   last_attempt_time BIGINT NOT NULL,
   last_attempt_time BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   role VARCHAR(255),
   role VARCHAR(255),

+ 1 - 1
ambari-server/src/main/resources/Ambari-DDL-Postgres-EMBEDDED-CREATE.sql

@@ -261,7 +261,7 @@ CREATE TABLE ambari.host_role_command (
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   event VARCHAR(32000) NOT NULL,
   event VARCHAR(32000) NOT NULL,
   exitcode INTEGER NOT NULL,
   exitcode INTEGER NOT NULL,
-  host_id BIGINT NOT NULL,
+  host_id BIGINT,
   last_attempt_time BIGINT NOT NULL,
   last_attempt_time BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   role VARCHAR(255),
   role VARCHAR(255),

+ 1 - 1
ambari-server/src/main/resources/Ambari-DDL-SQLServer-CREATE.sql

@@ -242,7 +242,7 @@ CREATE TABLE host_role_command (
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   retry_allowed SMALLINT DEFAULT 0 NOT NULL,
   event VARCHAR(MAX) NOT NULL,
   event VARCHAR(MAX) NOT NULL,
   exitcode INTEGER NOT NULL,
   exitcode INTEGER NOT NULL,
-  host_id BIGINT NOT NULL,
+  host_id BIGINT,
   last_attempt_time BIGINT NOT NULL,
   last_attempt_time BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   request_id BIGINT NOT NULL,
   role VARCHAR(255),
   role VARCHAR(255),

+ 10 - 9
ambari-server/src/test/java/org/apache/ambari/server/actionmanager/StageTest.java

@@ -18,9 +18,11 @@
 
 
 package org.apache.ambari.server.actionmanager;
 package org.apache.ambari.server.actionmanager;
 
 
-import com.google.inject.Guice;
-import com.google.inject.Inject;
-import com.google.inject.Injector;
+import static org.junit.Assert.assertEquals;
+
+import java.util.Collections;
+import java.util.List;
+
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.orm.GuiceJpaInitializer;
 import org.apache.ambari.server.orm.GuiceJpaInitializer;
@@ -32,10 +34,9 @@ import org.apache.ambari.server.utils.StageUtils;
 import org.junit.Before;
 import org.junit.Before;
 import org.junit.Test;
 import org.junit.Test;
 
 
-import java.util.Collections;
-import java.util.List;
-
-import static org.junit.Assert.assertEquals;
+import com.google.inject.Guice;
+import com.google.inject.Inject;
+import com.google.inject.Injector;
 
 
 /**
 /**
  * Stage tests.
  * Stage tests.
@@ -67,11 +68,11 @@ public class StageTest {
     stage.addServerActionCommand(ConfigureAction.class.getName(),
     stage.addServerActionCommand(ConfigureAction.class.getName(),
         "user1", Role.AMBARI_SERVER_ACTION,
         "user1", Role.AMBARI_SERVER_ACTION,
         RoleCommand.EXECUTE,
         RoleCommand.EXECUTE,
-        "cluster1", SERVER_HOST_NAME,
+        "cluster1",
         new ServiceComponentHostServerActionEvent(StageUtils.getHostName(), System.currentTimeMillis()),
         new ServiceComponentHostServerActionEvent(StageUtils.getHostName(), System.currentTimeMillis()),
         Collections.<String, String>emptyMap(), null, null, 1200, false);
         Collections.<String, String>emptyMap(), null, null, 1200, false);
 
 
-    List<ExecutionCommandWrapper> executionCommands = stage.getExecutionCommands(SERVER_HOST_NAME);
+    List<ExecutionCommandWrapper> executionCommands = stage.getExecutionCommands(null);
     assertEquals(1, executionCommands.size());
     assertEquals(1, executionCommands.size());
 
 
     String actionUserName = executionCommands.get(0).getExecutionCommand().getRoleParams().get(ServerAction.ACTION_USER_NAME);
     String actionUserName = executionCommands.get(0).getExecutionCommand().getRoleParams().get(ServerAction.ACTION_USER_NAME);

+ 15 - 26
ambari-server/src/test/java/org/apache/ambari/server/actionmanager/TestActionDBAccessorImpl.java

@@ -23,14 +23,11 @@ import static org.junit.Assert.assertNotNull;
 import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.assertTrue;
 
 
 import java.util.ArrayList;
 import java.util.ArrayList;
-import java.util.Arrays;
 import java.util.Collections;
 import java.util.Collections;
 import java.util.List;
 import java.util.List;
 
 
 import javax.persistence.EntityManager;
 import javax.persistence.EntityManager;
 
 
-import junit.framework.Assert;
-
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.RoleCommand;
@@ -39,7 +36,6 @@ import org.apache.ambari.server.agent.CommandReport;
 import org.apache.ambari.server.api.services.AmbariMetaInfo;
 import org.apache.ambari.server.api.services.AmbariMetaInfo;
 import org.apache.ambari.server.api.services.BaseRequest;
 import org.apache.ambari.server.api.services.BaseRequest;
 import org.apache.ambari.server.configuration.Configuration;
 import org.apache.ambari.server.configuration.Configuration;
-import org.apache.ambari.server.controller.ExecuteActionRequest;
 import org.apache.ambari.server.controller.HostsMap;
 import org.apache.ambari.server.controller.HostsMap;
 import org.apache.ambari.server.controller.internal.RequestResourceFilter;
 import org.apache.ambari.server.controller.internal.RequestResourceFilter;
 import org.apache.ambari.server.orm.DBAccessor;
 import org.apache.ambari.server.orm.DBAccessor;
@@ -69,6 +65,8 @@ import com.google.inject.persist.PersistService;
 import com.google.inject.persist.UnitOfWork;
 import com.google.inject.persist.UnitOfWork;
 import com.google.inject.util.Modules;
 import com.google.inject.util.Modules;
 
 
+import junit.framework.Assert;
+
 public class TestActionDBAccessorImpl {
 public class TestActionDBAccessorImpl {
   private static final Logger log = LoggerFactory.getLogger(TestActionDBAccessorImpl.class);
   private static final Logger log = LoggerFactory.getLogger(TestActionDBAccessorImpl.class);
 
 
@@ -355,7 +353,6 @@ public class TestActionDBAccessorImpl {
     assertEquals(HostRoleStatus.QUEUED, stage.getHostRoleStatus(hostName, actionName));
     assertEquals(HostRoleStatus.QUEUED, stage.getHostRoleStatus(hostName, actionName));
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
 
 
-    long now = System.currentTimeMillis();
     db.hostRoleScheduled(stage, hostName, actionName);
     db.hostRoleScheduled(stage, hostName, actionName);
 
 
     entities = hostRoleCommandDAO.findByHostRole(
     entities = hostRoleCommandDAO.findByHostRole(
@@ -383,25 +380,25 @@ public class TestActionDBAccessorImpl {
 
 
   @Test
   @Test
   public void testServerActionScheduled() throws InterruptedException, AmbariException {
   public void testServerActionScheduled() throws InterruptedException, AmbariException {
-    populateActionDBWithServerAction(db, serverHostName, requestId, stageId);
+    populateActionDBWithServerAction(db, null, requestId, stageId);
 
 
     final String roleName = Role.AMBARI_SERVER_ACTION.toString();
     final String roleName = Role.AMBARI_SERVER_ACTION.toString();
     Stage stage = db.getStage(StageUtils.getActionId(requestId, stageId));
     Stage stage = db.getStage(StageUtils.getActionId(requestId, stageId));
-    assertEquals(HostRoleStatus.PENDING, stage.getHostRoleStatus(serverHostName, roleName));
+    assertEquals(HostRoleStatus.PENDING, stage.getHostRoleStatus(null, roleName));
     List<HostRoleCommandEntity> entities =
     List<HostRoleCommandEntity> entities =
-        hostRoleCommandDAO.findByHostRole(serverHostName, requestId, stageId, roleName);
+        hostRoleCommandDAO.findByHostRole(null, requestId, stageId, roleName);
 
 
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
-    stage.setHostRoleStatus(serverHostName, roleName, HostRoleStatus.QUEUED);
+    stage.setHostRoleStatus(null, roleName, HostRoleStatus.QUEUED);
 
 
-    entities = hostRoleCommandDAO.findByHostRole(serverHostName, requestId, stageId, roleName);
-    assertEquals(HostRoleStatus.QUEUED, stage.getHostRoleStatus(serverHostName, roleName));
+    entities = hostRoleCommandDAO.findByHostRole(null, requestId, stageId, roleName);
+    assertEquals(HostRoleStatus.QUEUED, stage.getHostRoleStatus(null, roleName));
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
     assertEquals(HostRoleStatus.PENDING, entities.get(0).getStatus());
 
 
-    db.hostRoleScheduled(stage, serverHostName, roleName);
+    db.hostRoleScheduled(stage, null, roleName);
 
 
     entities = hostRoleCommandDAO.findByHostRole(
     entities = hostRoleCommandDAO.findByHostRole(
-        serverHostName, requestId, stageId, roleName);
+        null, requestId, stageId, roleName);
     assertEquals(HostRoleStatus.QUEUED, entities.get(0).getStatus());
     assertEquals(HostRoleStatus.QUEUED, entities.get(0).getStatus());
 
 
 
 
@@ -409,8 +406,8 @@ public class TestActionDBAccessorImpl {
       @Override
       @Override
       public void run() {
       public void run() {
         Stage stage1 = db.getStage("23-31");
         Stage stage1 = db.getStage("23-31");
-        stage1.setHostRoleStatus(serverHostName, roleName, HostRoleStatus.COMPLETED);
-        db.hostRoleScheduled(stage1, serverHostName, roleName);
+        stage1.setHostRoleStatus(null, roleName, HostRoleStatus.COMPLETED);
+        db.hostRoleScheduled(stage1, null, roleName);
         injector.getInstance(EntityManager.class).clear();
         injector.getInstance(EntityManager.class).clear();
       }
       }
     };
     };
@@ -419,7 +416,7 @@ public class TestActionDBAccessorImpl {
     thread.join();
     thread.join();
 
 
     injector.getInstance(EntityManager.class).clear();
     injector.getInstance(EntityManager.class).clear();
-    entities = hostRoleCommandDAO.findByHostRole(serverHostName, requestId, stageId, roleName);
+    entities = hostRoleCommandDAO.findByHostRole(null, requestId, stageId, roleName);
     assertEquals("Concurrent update failed", HostRoleStatus.COMPLETED, entities.get(0).getStatus());
     assertEquals("Concurrent update failed", HostRoleStatus.COMPLETED, entities.get(0).getStatus());
   }
   }
 
 
@@ -706,8 +703,6 @@ public class TestActionDBAccessorImpl {
     final RequestResourceFilter resourceFilter = new RequestResourceFilter("HBASE", "HBASE_MASTER", null);
     final RequestResourceFilter resourceFilter = new RequestResourceFilter("HBASE", "HBASE_MASTER", null);
     List<RequestResourceFilter> resourceFilters = new
     List<RequestResourceFilter> resourceFilters = new
       ArrayList<RequestResourceFilter>() {{ add(resourceFilter); }};
       ArrayList<RequestResourceFilter>() {{ add(resourceFilter); }};
-    ExecuteActionRequest executeActionRequest = new ExecuteActionRequest
-        ("cluster1", null, actionName, resourceFilters, null, null, false);
     Request request = new Request(stages, clusters);
     Request request = new Request(stages, clusters);
     db.persistActions(request);
     db.persistActions(request);
   }
   }
@@ -717,16 +712,10 @@ public class TestActionDBAccessorImpl {
     Stage s = stageFactory.createNew(requestId, "/a/b", "cluster1", 1L, "action db accessor test",
     Stage s = stageFactory.createNew(requestId, "/a/b", "cluster1", 1L, "action db accessor test",
         "", "commandParamsStage", "hostParamsStage");
         "", "commandParamsStage", "hostParamsStage");
     s.setStageId(stageId);
     s.setStageId(stageId);
-    s.addServerActionCommand(serverActionName, Role.AMBARI_SERVER_ACTION, RoleCommand.ACTIONEXECUTE, clusterName, null, null, "command details", null, 300, false);
+    s.addServerActionCommand(serverActionName, null, Role.AMBARI_SERVER_ACTION,
+        RoleCommand.ACTIONEXECUTE, clusterName, null, null, "command details", null, 300, false);
     List<Stage> stages = new ArrayList<Stage>();
     List<Stage> stages = new ArrayList<Stage>();
     stages.add(s);
     stages.add(s);
-    final RequestResourceFilter resourceFilter = new RequestResourceFilter("AMBARI", "SERVER", Arrays.asList(hostname));
-    List<RequestResourceFilter> resourceFilters = new
-        ArrayList<RequestResourceFilter>() {{
-          add(resourceFilter);
-        }};
-    ExecuteActionRequest executeActionRequest = new ExecuteActionRequest
-        ("cluster1", null, serverActionName, resourceFilters, null, null, false);
     Request request = new Request(stages, clusters);
     Request request = new Request(stages, clusters);
     db.persistActions(request);
     db.persistActions(request);
   }
   }

+ 119 - 190
ambari-server/src/test/java/org/apache/ambari/server/actionmanager/TestActionScheduler.java

@@ -46,10 +46,6 @@ import java.util.TreeMap;
 import java.util.concurrent.CountDownLatch;
 import java.util.concurrent.CountDownLatch;
 import java.util.concurrent.TimeUnit;
 import java.util.concurrent.TimeUnit;
 
 
-import com.google.inject.Inject;
-import com.google.inject.persist.PersistService;
-import junit.framework.Assert;
-
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.RoleCommand;
@@ -98,9 +94,13 @@ import org.slf4j.LoggerFactory;
 import com.google.common.reflect.TypeToken;
 import com.google.common.reflect.TypeToken;
 import com.google.inject.AbstractModule;
 import com.google.inject.AbstractModule;
 import com.google.inject.Guice;
 import com.google.inject.Guice;
+import com.google.inject.Inject;
 import com.google.inject.Injector;
 import com.google.inject.Injector;
+import com.google.inject.persist.PersistService;
 import com.google.inject.persist.UnitOfWork;
 import com.google.inject.persist.UnitOfWork;
 
 
+import junit.framework.Assert;
+
 public class TestActionScheduler {
 public class TestActionScheduler {
 
 
   private static final Logger log = LoggerFactory.getLogger(TestActionScheduler.class);
   private static final Logger log = LoggerFactory.getLogger(TestActionScheduler.class);
@@ -113,7 +113,6 @@ public class TestActionScheduler {
 
 
   private static Injector injector;
   private static Injector injector;
 
 
-  private final String serverHostname = StageUtils.getHostName();
   private final String hostname = "ahost.ambari.apache.org";
   private final String hostname = "ahost.ambari.apache.org";
   private final int MAX_CYCLE_ITERATIONS = 100;
   private final int MAX_CYCLE_ITERATIONS = 100;
 
 
@@ -291,9 +290,9 @@ public class TestActionScheduler {
     when(request.isExclusive()).thenReturn(false);
     when(request.isExclusive()).thenReturn(false);
     when(db.getRequestEntity(anyLong())).thenReturn(request);
     when(db.getRequestEntity(anyLong())).thenReturn(request);
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         HostRoleCommand command = s.getHostRoleCommand(host, role);
         HostRoleCommand command = s.getHostRoleCommand(host, role);
@@ -371,9 +370,9 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         HostRoleCommand command = s.getHostRoleCommand(host, role);
         HostRoleCommand command = s.getHostRoleCommand(host, role);
@@ -466,9 +465,9 @@ public class TestActionScheduler {
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         //HostRoleCommand command = stages.get(0).getHostRoleCommand(host, role);
         //HostRoleCommand command = stages.get(0).getHostRoleCommand(host, role);
@@ -482,9 +481,9 @@ public class TestActionScheduler {
       }
       }
     }).when(db).timeoutHostRole(anyString(), anyLong(), anyLong(), anyString());
     }).when(db).timeoutHostRole(anyString(), anyLong(), anyLong(), anyString());
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         Long requestId = (Long) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[0];
         for (Stage stage : stages) {
         for (Stage stage : stages) {
           if (requestId.equals(stage.getRequestId())) {
           if (requestId.equals(stage.getRequestId())) {
@@ -564,27 +563,7 @@ public class TestActionScheduler {
     Properties properties = new Properties();
     Properties properties = new Properties();
     Configuration conf = new Configuration(properties);
     Configuration conf = new Configuration(properties);
     Clusters fsm = mock(Clusters.class);
     Clusters fsm = mock(Clusters.class);
-    Cluster oneClusterMock = mock(Cluster.class);
-    Service serviceObj = mock(Service.class);
-    ServiceComponent scomp = mock(ServiceComponent.class);
-    ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    Host host = mock(Host.class);
-
-    when(host.getHostName()).thenReturn(serverHostname);
-    when(host.getState()).thenReturn(HostState.HEALTHY);
-
-    when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
-    when(fsm.getHost(anyString())).thenReturn(host);
-    when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
-    when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
-    when(scomp.getServiceComponentHost(anyString())).thenReturn(sch);
-    when(serviceObj.getCluster()).thenReturn(oneClusterMock);
-
-    HashMap<String, ServiceComponentHost> hosts =
-            new HashMap<String, ServiceComponentHost>();
-    hosts.put(serverHostname, sch);
-    when(scomp.getServiceComponentHosts()).thenReturn(hosts);
 
 
     List<Stage> stages = new ArrayList<Stage>();
     List<Stage> stages = new ArrayList<Stage>();
     Map<String, String> payload = new HashMap<String, String>();
     Map<String, String> payload = new HashMap<String, String>();
@@ -599,37 +578,44 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
-        HostRoleCommand command = s.getHostRoleCommand(host, role);
+
+        HostRoleCommand command = null;
+        if (null == host) {
+          command = s.getHostRoleCommand(null, role);
+        } else {
+          command = s.getHostRoleCommand(host, role);
+        }
+
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         return null;
         return null;
       }
       }
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<HostRoleCommand>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        return s.getHostRoleCommand(serverHostname, "AMBARI_SERVER_ACTION");
+      public HostRoleCommand answer(InvocationOnMock invocation) throws Throwable {
+        return s.getHostRoleCommand(null, "AMBARI_SERVER_ACTION");
       }
       }
     }).when(db).getTask(anyLong());
     }).when(db).getTask(anyLong());
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<List<HostRoleCommand>>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        String host = (String) invocation.getArguments()[0];
+      public List<HostRoleCommand> answer(InvocationOnMock invocation) throws Throwable {
         String role = (String) invocation.getArguments()[1];
         String role = (String) invocation.getArguments()[1];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
 
 
-        HostRoleCommand task = s.getHostRoleCommand(host, role);
+        HostRoleCommand task = s.getHostRoleCommand(null, role);
 
 
         if (task.getStatus() == status) {
         if (task.getStatus() == status) {
           return Arrays.asList(task);
           return Arrays.asList(task);
         } else {
         } else {
-          return null;
+          return Collections.emptyList();
         }
         }
       }
       }
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
@@ -639,13 +625,13 @@ public class TestActionScheduler {
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
     int cycleCount = 0;
     int cycleCount = 0;
-    while (!stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION")
+    while (!stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION")
         .equals(HostRoleStatus.COMPLETED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
         .equals(HostRoleStatus.COMPLETED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
       scheduler.doWork();
       scheduler.doWork();
       scheduler.getServerActionExecutor().doWork();
       scheduler.getServerActionExecutor().doWork();
     }
     }
 
 
-    assertEquals(stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION"),
+    assertEquals(stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION"),
         HostRoleStatus.COMPLETED);
         HostRoleStatus.COMPLETED);
   }
   }
 
 
@@ -658,26 +644,7 @@ public class TestActionScheduler {
     Properties properties = new Properties();
     Properties properties = new Properties();
     Configuration conf = new Configuration(properties);
     Configuration conf = new Configuration(properties);
     Clusters fsm = mock(Clusters.class);
     Clusters fsm = mock(Clusters.class);
-    Cluster oneClusterMock = mock(Cluster.class);
-    Service serviceObj = mock(Service.class);
-    ServiceComponent scomp = mock(ServiceComponent.class);
-    ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
-    when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
-    when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
-    when(scomp.getServiceComponentHost(anyString())).thenReturn(sch);
-    when(serviceObj.getCluster()).thenReturn(oneClusterMock);
-
-    Host host = mock(Host.class);
-    HashMap<String, ServiceComponentHost> hosts =
-            new HashMap<String, ServiceComponentHost>();
-    hosts.put(serverHostname, sch);
-    when(scomp.getServiceComponentHosts()).thenReturn(hosts);
-
-    when(fsm.getHost(anyString())).thenReturn(host);
-    when(host.getState()).thenReturn(HostState.HEALTHY);
-    when(host.getHostName()).thenReturn(serverHostname);
 
 
     List<Stage> stages = new ArrayList<Stage>();
     List<Stage> stages = new ArrayList<Stage>();
     Map<String, String> payload = new HashMap<String, String>();
     Map<String, String> payload = new HashMap<String, String>();
@@ -693,38 +660,46 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
-        HostRoleCommand command = s.getHostRoleCommand(host, role);
+
+        HostRoleCommand command = null;
+        if (null == host) {
+          command = s.getHostRoleCommand(null, role);
+        } else {
+          command = s.getHostRoleCommand(host, role);
+        }
+
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         return null;
         return null;
       }
       }
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<HostRoleCommand>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        return s.getHostRoleCommand(serverHostname, "AMBARI_SERVER_ACTION");
+      public HostRoleCommand answer(InvocationOnMock invocation) throws Throwable {
+        return s.getHostRoleCommand(null, "AMBARI_SERVER_ACTION");
       }
       }
     }).when(db).getTask(anyLong());
     }).when(db).getTask(anyLong());
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<List<HostRoleCommand>>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        String host = (String) invocation.getArguments()[0];
+      public List<HostRoleCommand> answer(InvocationOnMock invocation) throws Throwable {
         String role = (String) invocation.getArguments()[1];
         String role = (String) invocation.getArguments()[1];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
 
 
-        HostRoleCommand task = s.getHostRoleCommand(host, role);
+        HostRoleCommand task = s.getHostRoleCommand(null, role);
 
 
         if (task.getStatus() == status) {
         if (task.getStatus() == status) {
           return Arrays.asList(task);
           return Arrays.asList(task);
         } else {
         } else {
-          return null;
+          return Collections.emptyList();
         }
         }
+
       }
       }
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
 
 
@@ -733,14 +708,14 @@ public class TestActionScheduler {
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
     int cycleCount = 0;
     int cycleCount = 0;
-    while (!stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION").isCompletedState()
+    while (!stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION").isCompletedState()
         && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
         && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
       scheduler.doWork();
       scheduler.doWork();
       scheduler.getServerActionExecutor().doWork();
       scheduler.getServerActionExecutor().doWork();
     }
     }
 
 
     assertEquals(HostRoleStatus.TIMEDOUT,
     assertEquals(HostRoleStatus.TIMEDOUT,
-        stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION"));
+        stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION"));
   }
   }
 
 
   @Test
   @Test
@@ -749,21 +724,7 @@ public class TestActionScheduler {
     Properties properties = new Properties();
     Properties properties = new Properties();
     Configuration conf = new Configuration(properties);
     Configuration conf = new Configuration(properties);
     Clusters fsm = mock(Clusters.class);
     Clusters fsm = mock(Clusters.class);
-    Cluster oneClusterMock = mock(Cluster.class);
-    Service serviceObj = mock(Service.class);
-    ServiceComponent scomp = mock(ServiceComponent.class);
-    ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
-    when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
-    when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
-    when(scomp.getServiceComponentHost(anyString())).thenReturn(sch);
-    when(serviceObj.getCluster()).thenReturn(oneClusterMock);
-
-    HashMap<String, ServiceComponentHost> hosts =
-            new HashMap<String, ServiceComponentHost>();
-    hosts.put(serverHostname, sch);
-    when(scomp.getServiceComponentHosts()).thenReturn(hosts);
 
 
     List<Stage> stages = new ArrayList<Stage>();
     List<Stage> stages = new ArrayList<Stage>();
     Map<String, String> payload = new HashMap<String, String>();
     Map<String, String> payload = new HashMap<String, String>();
@@ -779,37 +740,45 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
-        HostRoleCommand command = s.getHostRoleCommand(host, role);
+
+        HostRoleCommand command = null;
+        if (null == host) {
+          command = s.getHostRoleCommand(null, role);
+        } else {
+          command = s.getHostRoleCommand(host, role);
+        }
+
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         return null;
         return null;
       }
       }
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
 
 
-    doAnswer(new Answer() {
+    doAnswer(new Answer<HostRoleCommand>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        return s.getHostRoleCommand(serverHostname, "AMBARI_SERVER_ACTION");
+      public HostRoleCommand answer(InvocationOnMock invocation) throws Throwable {
+        return s.getHostRoleCommand(null, "AMBARI_SERVER_ACTION");
       }
       }
     }).when(db).getTask(anyLong());
     }).when(db).getTask(anyLong());
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<List<HostRoleCommand>>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        String host = (String) invocation.getArguments()[0];
+      public List<HostRoleCommand> answer(InvocationOnMock invocation) throws Throwable {
         String role = (String) invocation.getArguments()[1];
         String role = (String) invocation.getArguments()[1];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
 
 
-        HostRoleCommand task = s.getHostRoleCommand(host, role);
+        HostRoleCommand task = s.getHostRoleCommand(null, role);
 
 
         if (task.getStatus() == status) {
         if (task.getStatus() == status) {
           return Arrays.asList(task);
           return Arrays.asList(task);
         } else {
         } else {
-          return null;
+          return Collections.emptyList();
         }
         }
       }
       }
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
@@ -818,12 +787,12 @@ public class TestActionScheduler {
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
     int cycleCount = 0;
     int cycleCount = 0;
-    while (!stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION")
+    while (!stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION")
         .equals(HostRoleStatus.FAILED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
         .equals(HostRoleStatus.FAILED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
       scheduler.doWork();
       scheduler.doWork();
       scheduler.getServerActionExecutor().doWork();
       scheduler.getServerActionExecutor().doWork();
     }
     }
-    assertEquals(stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION"),
+    assertEquals(stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION"),
         HostRoleStatus.FAILED);
         HostRoleStatus.FAILED);
     assertEquals("test", stages.get(0).getRequestContext());
     assertEquals("test", stages.get(0).getRequestContext());
   }
   }
@@ -831,14 +800,15 @@ public class TestActionScheduler {
   private Stage getStageWithServerAction(long requestId, long stageId,
   private Stage getStageWithServerAction(long requestId, long stageId,
                                                 Map<String, String> payload, String requestContext,
                                                 Map<String, String> payload, String requestContext,
                                                 int timeout) {
                                                 int timeout) {
-    String serverHostname = StageUtils.getHostName();
+
     Stage stage = stageFactory.createNew(requestId, "/tmp", "cluster1", 1L, requestContext, CLUSTER_HOST_INFO,
     Stage stage = stageFactory.createNew(requestId, "/tmp", "cluster1", 1L, requestContext, CLUSTER_HOST_INFO,
       "{}", "{}");
       "{}", "{}");
     stage.setStageId(stageId);
     stage.setStageId(stageId);
 
 
-    stage.addServerActionCommand(MockServerAction.class.getName(), Role.AMBARI_SERVER_ACTION,
+    stage.addServerActionCommand(MockServerAction.class.getName(), null,
+        Role.AMBARI_SERVER_ACTION,
         RoleCommand.EXECUTE, "cluster1",
         RoleCommand.EXECUTE, "cluster1",
-        new ServiceComponentHostServerActionEvent(serverHostname, System.currentTimeMillis()),
+        new ServiceComponentHostServerActionEvent(null, System.currentTimeMillis()),
         payload,
         payload,
         null, null, timeout, false);
         null, null, timeout, false);
 
 
@@ -859,7 +829,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -917,10 +886,6 @@ public class TestActionScheduler {
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
             new HostsMap((String) null), unitOfWork, null, conf);
             new HostsMap((String) null), unitOfWork, null, conf);
 
 
-    ActionManager am = new ActionManager(
-        2, 2, aq, fsm, db, new HostsMap((String) null),
-        unitOfWork, requestFactory, conf, null);
-
     scheduler.doWork();
     scheduler.doWork();
 
 
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "DATANODE"));
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "DATANODE"));
@@ -943,7 +908,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -1011,11 +975,6 @@ public class TestActionScheduler {
             new HostsMap((String) null),
             new HostsMap((String) null),
             unitOfWork, null, conf);
             unitOfWork, null, conf);
 
 
-    ActionManager am = new ActionManager(
-        2, 2, aq, fsm, db, new HostsMap((String) null),
-        unitOfWork,
-        requestFactory, conf, null);
-
     scheduler.doWork();
     scheduler.doWork();
 
 
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "HIVE_CLIENT"));
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "HIVE_CLIENT"));
@@ -1038,7 +997,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -1088,11 +1046,6 @@ public class TestActionScheduler {
         new HostsMap((String) null),
         new HostsMap((String) null),
         unitOfWork, null, conf);
         unitOfWork, null, conf);
 
 
-    ActionManager am = new ActionManager(
-        2, 2, aq, fsm, db, new HostsMap((String) null),
-        unitOfWork,
-        requestFactory, conf, null);
-
     scheduler.doWork();
     scheduler.doWork();
 
 
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "NAMENODE"));
     Assert.assertEquals(HostRoleStatus.QUEUED, stages.get(0).getHostRoleStatus(hostname1, "NAMENODE"));
@@ -1143,16 +1096,15 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         for (CommandReport report : reports) {
         for (CommandReport report : reports) {
           String actionId = report.getActionId();
           String actionId = report.getActionId();
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           Long requestId = requestStageIds[0];
           Long requestId = requestStageIds[0];
           Long stageId = requestStageIds[1];
           Long stageId = requestStageIds[1];
-          String role = report.getRole();
           Long id = report.getTaskId();
           Long id = report.getTaskId();
           for (Stage stage : stages) {
           for (Stage stage : stages) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
@@ -1184,9 +1136,9 @@ public class TestActionScheduler {
         return null;
         return null;
       }
       }
     });
     });
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         Long requestId = (Long) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[0];
         for (Stage stage : stages) {
         for (Stage stage : stages) {
           if (requestId.equals(stage.getRequestId())) {
           if (requestId.equals(stage.getRequestId())) {
@@ -1251,7 +1203,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -1332,9 +1283,9 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[1];
         Long requestId = (Long) invocation.getArguments()[1];
         Long stageId = (Long) invocation.getArguments()[2];
         Long stageId = (Long) invocation.getArguments()[2];
@@ -1365,9 +1316,9 @@ public class TestActionScheduler {
         return null;
         return null;
       }
       }
     });
     });
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         Long requestId = (Long) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[0];
         for (Stage stage : stages) {
         for (Stage stage : stages) {
           if (requestId.equals(stage.getRequestId())) {
           if (requestId.equals(stage.getRequestId())) {
@@ -1390,8 +1341,6 @@ public class TestActionScheduler {
     ActionScheduler scheduler = new ActionScheduler(100, 10000, db, aq, fsm, 3,
     ActionScheduler scheduler = new ActionScheduler(100, 10000, db, aq, fsm, 3,
         new HostsMap((String) null),
         new HostsMap((String) null),
         unitOfWork, null, conf);
         unitOfWork, null, conf);
-    ActionManager am = new ActionManager(
-        2, 10000, aq, fsm, db, new HostsMap((String) null), unitOfWork, requestFactory, conf, null);
 
 
     scheduler.doWork();
     scheduler.doWork();
 
 
@@ -1511,16 +1460,15 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         for (CommandReport report : reports) {
         for (CommandReport report : reports) {
           String actionId = report.getActionId();
           String actionId = report.getActionId();
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           Long requestId = requestStageIds[0];
           Long requestId = requestStageIds[0];
           Long stageId = requestStageIds[1];
           Long stageId = requestStageIds[1];
-          String role = report.getRole();
           Long id = report.getTaskId();
           Long id = report.getTaskId();
           for (Stage stage : stages) {
           for (Stage stage : stages) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
@@ -1552,9 +1500,9 @@ public class TestActionScheduler {
         return null;
         return null;
       }
       }
     });
     });
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         Long requestId = (Long) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[0];
         for (Stage stage : stages) {
         for (Stage stage : stages) {
           if (requestId.equals(stage.getRequestId())) {
           if (requestId.equals(stage.getRequestId())) {
@@ -1882,26 +1830,11 @@ public class TestActionScheduler {
     Properties properties = new Properties();
     Properties properties = new Properties();
     Configuration conf = new Configuration(properties);
     Configuration conf = new Configuration(properties);
     Clusters fsm = mock(Clusters.class);
     Clusters fsm = mock(Clusters.class);
-    Cluster oneClusterMock = mock(Cluster.class);
-    Service serviceObj = mock(Service.class);
-    ServiceComponent scomp = mock(ServiceComponent.class);
-    ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
-    when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
-    when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
-    when(scomp.getServiceComponentHost(anyString())).thenReturn(sch);
-    when(serviceObj.getCluster()).thenReturn(oneClusterMock);
-
-    HashMap<String, ServiceComponentHost> hosts =
-            new HashMap<String, ServiceComponentHost>();
-    hosts.put(serverHostname, sch);
-    when(scomp.getServiceComponentHosts()).thenReturn(hosts);
 
 
     List<Stage> stages = new ArrayList<Stage>();
     List<Stage> stages = new ArrayList<Stage>();
     Map<String, String> payload = new HashMap<String, String>();
     Map<String, String> payload = new HashMap<String, String>();
     final Stage s = getStageWithServerAction(1, 977, payload, "test", 300);
     final Stage s = getStageWithServerAction(1, 977, payload, "test", 300);
-    s.getExecutionCommands().get(serverHostname).get(0).getExecutionCommand().setServiceName(null);
     stages.add(s);
     stages.add(s);
 
 
     ActionDBAccessor db = mock(ActionDBAccessor.class);
     ActionDBAccessor db = mock(ActionDBAccessor.class);
@@ -1912,37 +1845,45 @@ public class TestActionScheduler {
 
 
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getCommandsInProgressCount()).thenReturn(stages.size());
     when(db.getStagesInProgress()).thenReturn(stages);
     when(db.getStagesInProgress()).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         String host = (String) invocation.getArguments()[0];
         String host = (String) invocation.getArguments()[0];
         String role = (String) invocation.getArguments()[3];
         String role = (String) invocation.getArguments()[3];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
         CommandReport commandReport = (CommandReport) invocation.getArguments()[4];
-        HostRoleCommand command = s.getHostRoleCommand(host, role);
+
+        HostRoleCommand command = null;
+        if (null == host) {
+          command = s.getHostRoleCommand(null, role);
+        } else {
+          command = s.getHostRoleCommand(host, role);
+        }
+
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         command.setStatus(HostRoleStatus.valueOf(commandReport.getStatus()));
         return null;
         return null;
       }
       }
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
     }).when(db).updateHostRoleState(anyString(), anyLong(), anyLong(), anyString(), any(CommandReport.class));
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<List<HostRoleCommand>>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        String host = (String) invocation.getArguments()[0];
+      public List<HostRoleCommand> answer(InvocationOnMock invocation) throws Throwable {
         String role = (String) invocation.getArguments()[1];
         String role = (String) invocation.getArguments()[1];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
         HostRoleStatus status = (HostRoleStatus) invocation.getArguments()[2];
 
 
-        HostRoleCommand task = s.getHostRoleCommand(host, role);
+        HostRoleCommand task = s.getHostRoleCommand(null, role);
 
 
         if (task.getStatus() == status) {
         if (task.getStatus() == status) {
           return Arrays.asList(task);
           return Arrays.asList(task);
         } else {
         } else {
-          return null;
+          return Collections.emptyList();
         }
         }
       }
       }
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
     }).when(db).getTasksByHostRoleAndStatus(anyString(), anyString(), any(HostRoleStatus.class));
-    doAnswer(new Answer() {
+
+    doAnswer(new Answer<HostRoleCommand>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
-        return s.getHostRoleCommand(serverHostname, "AMBARI_SERVER_ACTION");
+      public HostRoleCommand answer(InvocationOnMock invocation) throws Throwable {
+        return s.getHostRoleCommand(null, "AMBARI_SERVER_ACTION");
       }
       }
     }).when(db).getTask(anyLong());
     }).when(db).getTask(anyLong());
 
 
@@ -1951,13 +1892,13 @@ public class TestActionScheduler {
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
     int cycleCount = 0;
     int cycleCount = 0;
-    while (!stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION")
+    while (!stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION")
         .equals(HostRoleStatus.COMPLETED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
         .equals(HostRoleStatus.COMPLETED) && cycleCount++ <= MAX_CYCLE_ITERATIONS) {
       scheduler.doWork();
       scheduler.doWork();
       scheduler.getServerActionExecutor().doWork();
       scheduler.getServerActionExecutor().doWork();
     }
     }
 
 
-    assertEquals(stages.get(0).getHostRoleStatus(serverHostname, "AMBARI_SERVER_ACTION"),
+    assertEquals(stages.get(0).getHostRoleStatus(null, "AMBARI_SERVER_ACTION"),
             HostRoleStatus.COMPLETED);
             HostRoleStatus.COMPLETED);
   }
   }
 
 
@@ -1970,7 +1911,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -2019,16 +1959,15 @@ public class TestActionScheduler {
     }
     }
     when(db.getRequestTasks(anyLong())).thenReturn(requestTasks);
     when(db.getRequestTasks(anyLong())).thenReturn(requestTasks);
     when(db.getAllStages(anyLong())).thenReturn(stages);
     when(db.getAllStages(anyLong())).thenReturn(stages);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         for (CommandReport report : reports) {
         for (CommandReport report : reports) {
           String actionId = report.getActionId();
           String actionId = report.getActionId();
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           Long requestId = requestStageIds[0];
           Long requestId = requestStageIds[0];
           Long stageId = requestStageIds[1];
           Long stageId = requestStageIds[1];
-          String role = report.getRole();
           Long id = report.getTaskId();
           Long id = report.getTaskId();
           for (Stage stage : stages) {
           for (Stage stage : stages) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
@@ -2060,9 +1999,9 @@ public class TestActionScheduler {
         return null;
         return null;
       }
       }
     });
     });
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         Long requestId = (Long) invocation.getArguments()[0];
         Long requestId = (Long) invocation.getArguments()[0];
         for (Stage stage : stages) {
         for (Stage stage : stages) {
           if (requestId.equals(stage.getRequestId())) {
           if (requestId.equals(stage.getRequestId())) {
@@ -2086,10 +2025,6 @@ public class TestActionScheduler {
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
-    ActionManager am = new ActionManager(
-        2, 2, aq, fsm, db, new HostsMap((String) null),
-        unitOfWork, requestFactory, conf, null);
-
     scheduler.doWork();
     scheduler.doWork();
 
 
     String reason = "Some reason";
     String reason = "Some reason";
@@ -2116,7 +2051,6 @@ public class TestActionScheduler {
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponent scomp = mock(ServiceComponent.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     ServiceComponentHost sch = mock(ServiceComponentHost.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
     UnitOfWork unitOfWork = mock(UnitOfWork.class);
-    RequestFactory requestFactory = mock(RequestFactory.class);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(fsm.getCluster(anyString())).thenReturn(oneClusterMock);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(oneClusterMock.getService(anyString())).thenReturn(serviceObj);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
     when(serviceObj.getServiceComponent(anyString())).thenReturn(scomp);
@@ -2184,16 +2118,15 @@ public class TestActionScheduler {
     }
     }
     when(db.getRequestTasks(anyLong())).thenReturn(requestTasks);
     when(db.getRequestTasks(anyLong())).thenReturn(requestTasks);
     when(db.getAllStages(anyLong())).thenReturn(stagesInProgress);
     when(db.getAllStages(anyLong())).thenReturn(stagesInProgress);
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         List<CommandReport> reports = (List<CommandReport>) invocation.getArguments()[0];
         for (CommandReport report : reports) {
         for (CommandReport report : reports) {
           String actionId = report.getActionId();
           String actionId = report.getActionId();
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           long[] requestStageIds = StageUtils.getRequestStage(actionId);
           Long requestId = requestStageIds[0];
           Long requestId = requestStageIds[0];
           Long stageId = requestStageIds[1];
           Long stageId = requestStageIds[1];
-          String role = report.getRole();
           Long id = report.getTaskId();
           Long id = report.getTaskId();
           for (Stage stage : stagesInProgress) {
           for (Stage stage : stagesInProgress) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
             if (requestId.equals(stage.getRequestId()) && stageId.equals(stage.getStageId())) {
@@ -2227,9 +2160,9 @@ public class TestActionScheduler {
     });
     });
 
 
     final Map<Long, Boolean> startedRequests = new HashMap<Long, Boolean>();
     final Map<Long, Boolean> startedRequests = new HashMap<Long, Boolean>();
-    doAnswer(new Answer() {
+    doAnswer(new Answer<Void>() {
       @Override
       @Override
-      public Object answer(InvocationOnMock invocation) throws Throwable {
+      public Void answer(InvocationOnMock invocation) throws Throwable {
         startedRequests.put((Long)invocation.getArguments()[0], true);
         startedRequests.put((Long)invocation.getArguments()[0], true);
         return null;
         return null;
       }
       }
@@ -2252,10 +2185,6 @@ public class TestActionScheduler {
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
     ActionScheduler scheduler = new ActionScheduler(100, 50, db, aq, fsm, 3,
         new HostsMap((String) null), unitOfWork, null, conf);
         new HostsMap((String) null), unitOfWork, null, conf);
 
 
-    ActionManager am = new ActionManager(
-        2, 2, aq, fsm, db, new HostsMap((String) null),
-        unitOfWork, requestFactory, conf, null);
-
     // Execution of request 1
     // Execution of request 1
 
 
     scheduler.doWork();
     scheduler.doWork();

+ 12 - 0
ambari-server/src/test/java/org/apache/ambari/server/controller/internal/UpgradeResourceProviderTest.java

@@ -23,6 +23,7 @@ import static org.easymock.EasyMock.replay;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertNotNull;
 import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
 import static org.junit.Assert.assertTrue;
 import static org.junit.Assert.assertTrue;
 
 
 import java.lang.reflect.Field;
 import java.lang.reflect.Field;
@@ -33,6 +34,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Map;
 import java.util.Set;
 import java.util.Set;
 
 
+import org.apache.ambari.server.Role;
 import org.apache.ambari.server.actionmanager.ActionManager;
 import org.apache.ambari.server.actionmanager.ActionManager;
 import org.apache.ambari.server.actionmanager.HostRoleCommand;
 import org.apache.ambari.server.actionmanager.HostRoleCommand;
 import org.apache.ambari.server.actionmanager.HostRoleStatus;
 import org.apache.ambari.server.actionmanager.HostRoleStatus;
@@ -568,6 +570,16 @@ public class UpgradeResourceProviderTest {
 
 
     List<HostRoleCommand> commands = am.getRequestTasks(id);
     List<HostRoleCommand> commands = am.getRequestTasks(id);
 
 
+    boolean foundOne = false;
+    for (HostRoleCommand hrc : commands) {
+      if (hrc.getRole().equals(Role.AMBARI_SERVER_ACTION)) {
+        assertEquals(-1L, hrc.getHostId());
+        assertNull(hrc.getHostName());
+        foundOne = true;
+      }
+    }
+    assertTrue("Expected at least one server-side action", foundOne);
+
     HostRoleCommand cmd = commands.get(commands.size()-1);
     HostRoleCommand cmd = commands.get(commands.size()-1);
 
 
     HostRoleCommandDAO dao = injector.getInstance(HostRoleCommandDAO.class);
     HostRoleCommandDAO dao = injector.getInstance(HostRoleCommandDAO.class);

+ 3 - 5
ambari-server/src/test/java/org/apache/ambari/server/serveraction/ServerActionExecutorTest.java

@@ -122,9 +122,7 @@ public class ServerActionExecutorTest {
         "{\"host_param\":\"param_value\"}", "{\"stage_param\":\"param_value\"}");
         "{\"host_param\":\"param_value\"}", "{\"stage_param\":\"param_value\"}");
 
 
     stage.addServerActionCommand(ManualStageAction.class.getName(),
     stage.addServerActionCommand(ManualStageAction.class.getName(),
-        null, Role.AMBARI_SERVER_ACTION,
-        RoleCommand.EXECUTE,
-        "cluster1", SERVER_HOST_NAME,
+        null, Role.AMBARI_SERVER_ACTION, RoleCommand.EXECUTE, "cluster1",
         new ServiceComponentHostServerActionEvent(StageUtils.getHostName(), System.currentTimeMillis()),
         new ServiceComponentHostServerActionEvent(StageUtils.getHostName(), System.currentTimeMillis()),
         Collections.<String, String>emptyMap(), null, null, 1200, false);
         Collections.<String, String>emptyMap(), null, null, 1200, false);
 
 
@@ -321,8 +319,8 @@ public class ServerActionExecutorTest {
                 1L, requestContext, CLUSTER_HOST_INFO, "{}", "{}");
                 1L, requestContext, CLUSTER_HOST_INFO, "{}", "{}");
 
 
             stage.setStageId(stageId);
             stage.setStageId(stageId);
-            stage.addServerActionCommand(MockServerAction.class.getName(), Role.AMBARI_SERVER_ACTION,
-                RoleCommand.EXECUTE, "cluster1",
+            stage.addServerActionCommand(MockServerAction.class.getName(), null,
+                Role.AMBARI_SERVER_ACTION, RoleCommand.EXECUTE, "cluster1",
                 new ServiceComponentHostServerActionEvent(SERVER_HOST_NAME, System.currentTimeMillis()),
                 new ServiceComponentHostServerActionEvent(SERVER_HOST_NAME, System.currentTimeMillis()),
                 payload, "command detail", null, timeout, false);
                 payload, "command detail", null, timeout, false);
 
 

+ 12 - 11
ambari-server/src/test/java/org/apache/ambari/server/stageplanner/TestStagePlanner.java

@@ -17,10 +17,12 @@
  */
  */
 package org.apache.ambari.server.stageplanner;
 package org.apache.ambari.server.stageplanner;
 
 
-import com.google.inject.Guice;
-import com.google.inject.Inject;
-import com.google.inject.Injector;
-import com.google.inject.persist.PersistService;
+import static org.junit.Assert.assertEquals;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
+
+import java.util.List;
+
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.AmbariException;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.Role;
 import org.apache.ambari.server.RoleCommand;
 import org.apache.ambari.server.RoleCommand;
@@ -38,11 +40,10 @@ import org.junit.After;
 import org.junit.Before;
 import org.junit.Before;
 import org.junit.Test;
 import org.junit.Test;
 
 
-import java.util.List;
-
-import static org.junit.Assert.assertEquals;
-import static org.mockito.Mockito.mock;
-import static org.mockito.Mockito.when;
+import com.google.inject.Guice;
+import com.google.inject.Inject;
+import com.google.inject.Injector;
+import com.google.inject.persist.PersistService;
 
 
 public class TestStagePlanner {
 public class TestStagePlanner {
 
 
@@ -126,11 +127,11 @@ public class TestStagePlanner {
     Stage stage = stageFactory.createNew(1, "/tmp", "cluster1", 1L,  "execution command wrapper test",
     Stage stage = stageFactory.createNew(1, "/tmp", "cluster1", 1L,  "execution command wrapper test",
             "clusterHostInfo", "commandParamsStage", "hostParamsStage");
             "clusterHostInfo", "commandParamsStage", "hostParamsStage");
     stage.setStageId(1);
     stage.setStageId(1);
-    stage.addServerActionCommand("RESTART", Role.HIVE_METASTORE,
+    stage.addServerActionCommand("RESTART", null, Role.HIVE_METASTORE,
             RoleCommand.CUSTOM_COMMAND, "cluster1",
             RoleCommand.CUSTOM_COMMAND, "cluster1",
             new ServiceComponentHostServerActionEvent("host2", System.currentTimeMillis()),
             new ServiceComponentHostServerActionEvent("host2", System.currentTimeMillis()),
             null, "command detail", null, null, false);
             null, "command detail", null, null, false);
-    stage.addServerActionCommand("RESTART", Role.MYSQL_SERVER,
+    stage.addServerActionCommand("RESTART", null, Role.MYSQL_SERVER,
             RoleCommand.CUSTOM_COMMAND, "cluster1",
             RoleCommand.CUSTOM_COMMAND, "cluster1",
             new ServiceComponentHostServerActionEvent("host2", System.currentTimeMillis()),
             new ServiceComponentHostServerActionEvent("host2", System.currentTimeMillis()),
             null, "command detail", null, null, false);
             null, "command detail", null, null, false);

+ 112 - 0
ambari-server/src/test/java/org/apache/ambari/server/upgrade/UpgradeCatalog220Test.java

@@ -0,0 +1,112 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.ambari.server.upgrade;
+
+import static org.easymock.EasyMock.capture;
+import static org.easymock.EasyMock.createNiceMock;
+import static org.easymock.EasyMock.eq;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.expectLastCall;
+import static org.easymock.EasyMock.replay;
+import static org.easymock.EasyMock.verify;
+import static org.junit.Assert.assertTrue;
+
+import java.lang.reflect.Field;
+
+import org.apache.ambari.server.configuration.Configuration;
+import org.apache.ambari.server.orm.DBAccessor;
+import org.apache.ambari.server.state.stack.OsFamily;
+import org.easymock.Capture;
+import org.junit.Assert;
+import org.junit.Test;
+
+import com.google.inject.Binder;
+import com.google.inject.Guice;
+import com.google.inject.Injector;
+import com.google.inject.Module;
+
+/**
+ * UpgradeCatalog220 tests.
+ */
+public class UpgradeCatalog220Test {
+
+
+  @Test
+  public void testExecuteDDLUpdates() throws Exception {
+
+    final DBAccessor dbAccessor = createNiceMock(DBAccessor.class);
+
+    Configuration configuration = createNiceMock(Configuration.class);
+    expect(configuration.getDatabaseUrl()).andReturn(Configuration.JDBC_IN_MEMORY_URL).anyTimes();
+
+    Capture<DBAccessor.DBColumnInfo> columnCapture = new Capture<DBAccessor.DBColumnInfo>();
+
+    dbAccessor.alterColumn(eq("host_role_command"), capture(columnCapture));
+    expectLastCall();
+
+
+    replay(dbAccessor, configuration);
+    AbstractUpgradeCatalog upgradeCatalog = getUpgradeCatalog(dbAccessor);
+    Class<?> c = AbstractUpgradeCatalog.class;
+    Field f = c.getDeclaredField("configuration");
+    f.setAccessible(true);
+    f.set(upgradeCatalog, configuration);
+
+    upgradeCatalog.executeDDLUpdates();
+    verify(dbAccessor, configuration);
+
+    assertTrue(columnCapture.getValue().isNullable());
+  }
+
+  @Test
+  public void testExecuteDMLUpdates() throws Exception {
+    final DBAccessor dbAccessor     = createNiceMock(DBAccessor.class);
+    UpgradeCatalog220 upgradeCatalog = (UpgradeCatalog220) getUpgradeCatalog(dbAccessor);
+
+    upgradeCatalog.executeDMLUpdates();
+  }
+
+  @Test
+  public void testGetTargetVersion() throws Exception {
+    final DBAccessor dbAccessor     = createNiceMock(DBAccessor.class);
+    UpgradeCatalog   upgradeCatalog = getUpgradeCatalog(dbAccessor);
+
+    Assert.assertEquals("2.2.0", upgradeCatalog.getTargetVersion());
+  }
+
+  @Test
+  public void testGetSourceVersion() {
+    final DBAccessor dbAccessor     = createNiceMock(DBAccessor.class);
+    UpgradeCatalog upgradeCatalog = getUpgradeCatalog(dbAccessor);
+    Assert.assertEquals("2.1.0", upgradeCatalog.getSourceVersion());
+  }
+
+  private AbstractUpgradeCatalog getUpgradeCatalog(final DBAccessor dbAccessor) {
+    Module module = new Module() {
+      @Override
+      public void configure(Binder binder) {
+        binder.bind(DBAccessor.class).toInstance(dbAccessor);
+        binder.bind(OsFamily.class).toInstance(createNiceMock(OsFamily.class));
+      }
+    };
+    Injector injector = Guice.createInjector(module);
+    return injector.getInstance(UpgradeCatalog220.class);
+  }
+
+
+}