You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@lucene.apache.org by sh...@apache.org on 2017/06/22 13:16:45 UTC

lucene-solr:jira/SOLR-10496: SOLR-10496: Initial patch for ComputePlanAction

Repository: lucene-solr
Updated Branches:
  refs/heads/jira/SOLR-10496 [created] ebf298329


SOLR-10496: Initial patch for ComputePlanAction


Project: http://git-wip-us.apache.org/repos/asf/lucene-solr/repo
Commit: http://git-wip-us.apache.org/repos/asf/lucene-solr/commit/ebf29832
Tree: http://git-wip-us.apache.org/repos/asf/lucene-solr/tree/ebf29832
Diff: http://git-wip-us.apache.org/repos/asf/lucene-solr/diff/ebf29832

Branch: refs/heads/jira/SOLR-10496
Commit: ebf298329360240014253daf58ab4699f3685033
Parents: 148865f
Author: Shalin Shekhar Mangar <sh...@apache.org>
Authored: Thu Jun 22 18:46:28 2017 +0530
Committer: Shalin Shekhar Mangar <sh...@apache.org>
Committed: Thu Jun 22 18:46:28 2017 +0530

----------------------------------------------------------------------
 .../solr/cloud/autoscaling/ActionContext.java   |  56 +++++
 .../solr/cloud/autoscaling/AutoScaling.java     |   2 +-
 .../cloud/autoscaling/ComputePlanAction.java    |  93 +++++++-
 .../cloud/autoscaling/ExecutePlanAction.java    |   7 +-
 .../solr/cloud/autoscaling/LogPlanAction.java   |   7 +-
 .../autoscaling/OverseerTriggerThread.java      |   2 +-
 .../cloud/autoscaling/ScheduledTriggers.java    |  13 +-
 .../solr/cloud/autoscaling/TriggerAction.java   |   6 +-
 .../autoscaling/ComputePlanActionTest.java      | 221 +++++++++++++++++++
 .../cloud/autoscaling/NodeAddedTriggerTest.java |   7 +-
 .../cloud/autoscaling/NodeLostTriggerTest.java  |   7 +-
 .../autoscaling/TriggerIntegrationTest.java     |  23 +-
 .../apache/solr/cloud/autoscaling/Policy.java   |   5 +
 .../org/apache/solr/cloud/autoscaling/Row.java  |   2 +
 14 files changed, 392 insertions(+), 59 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/ActionContext.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ActionContext.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ActionContext.java
new file mode 100644
index 0000000..ecc339c
--- /dev/null
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ActionContext.java
@@ -0,0 +1,56 @@
+/*
+ * 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.solr.cloud.autoscaling;
+
+import java.util.Map;
+
+import org.apache.solr.core.CoreContainer;
+
+/**
+ * Provides additional context for the TriggerAction such as the trigger instance on
+ * which the action is being executed as well as helper methods to pass computed information along
+ * to the next action
+ */
+public class ActionContext {
+
+  private final CoreContainer coreContainer;
+  private final AutoScaling.Trigger source;
+  private final Map<String, Object> properties;
+
+  public ActionContext(CoreContainer coreContainer, AutoScaling.Trigger source, Map<String, Object> properties) {
+    this.coreContainer = coreContainer;
+    this.source = source;
+    this.properties = properties;
+  }
+
+  public CoreContainer getCoreContainer() {
+    return coreContainer;
+  }
+
+  public AutoScaling.Trigger getSource() {
+    return source;
+  }
+
+  public Map<String, Object> getProperties()  {
+    return properties;
+  }
+
+  public Object getProperty(String name)  {
+    return properties != null ? properties.get(name) : null;
+  }
+}

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/AutoScaling.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/AutoScaling.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/AutoScaling.java
index f1f2a26..cd65090 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/AutoScaling.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/AutoScaling.java
@@ -129,7 +129,7 @@ public class AutoScaling {
      * Called before a trigger is scheduled. Any heavy object creation or initialisation should
      * be done in this method instead of the Trigger's constructor.
      */
-    public void init();
+    void init();
   }
 
   public static class TriggerFactory implements Closeable {

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/ComputePlanAction.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ComputePlanAction.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ComputePlanAction.java
index 1b8e680..253c231 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ComputePlanAction.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ComputePlanAction.java
@@ -18,12 +18,30 @@
 package org.apache.solr.cloud.autoscaling;
 
 import java.io.IOException;
+import java.lang.invoke.MethodHandles;
+import java.util.ArrayList;
+import java.util.List;
 import java.util.Map;
 
+import org.apache.solr.client.solrj.SolrRequest;
+import org.apache.solr.client.solrj.impl.CloudSolrClient;
+import org.apache.solr.client.solrj.impl.SolrClientDataProvider;
+import org.apache.solr.common.cloud.ZkStateReader;
+import org.apache.solr.common.params.CollectionParams;
+import org.apache.solr.common.util.Utils;
+import org.apache.solr.core.CoreContainer;
+import org.apache.zookeeper.KeeperException;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
 /**
  * todo nocommit
  */
 public class ComputePlanAction implements TriggerAction {
+  private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
+
+  private Map<String, String> initArgs;
+
   @Override
   public void close() throws IOException {
 
@@ -31,21 +49,82 @@ public class ComputePlanAction implements TriggerAction {
 
   @Override
   public void init(Map<String, String> args) {
-
+    this.initArgs = args;
   }
 
   @Override
   public String getName() {
-    return null;
+    return initArgs.get("name");
   }
 
   @Override
-  public String getClassName() {
-    return null;
+  public void process(TriggerEvent event, ActionContext context) {
+    log.debug("-- processing event: {} with context properties: {}", event, context.getProperties());
+    CoreContainer container = context.getCoreContainer();
+    try {
+      try (CloudSolrClient cloudSolrClient = new CloudSolrClient.Builder()
+          .withZkHost(container.getZkController().getZkServerAddress())
+          .withHttpClient(container.getUpdateShardHandler().getHttpClient())
+          .build()) {
+        ZkStateReader zkStateReader = cloudSolrClient.getZkStateReader();
+        zkStateReader.getZkClient().printLayoutToStdOut();
+        Map<String, Object> autoScalingConf = Utils.getJson(zkStateReader.getZkClient(), ZkStateReader.SOLR_AUTOSCALING_CONF_PATH, true);
+        if (autoScalingConf.isEmpty()) {
+          log.error("Action: " + getName() + " executed but no policy is configured");
+          return;
+        }
+        log.debug("Fetched autoscaling conf: {}", autoScalingConf); // todo nocommit
+        AutoScalingConfig config = new AutoScalingConfig(autoScalingConf);
+        Policy policy = config.getPolicy();
+        log.debug("created policy"); // todo nocommit
+        Policy.Session session = policy.createSession(new SolrClientDataProvider(cloudSolrClient));
+        Policy.Suggester suggester = getSuggester(session, event);
+        while (true) {
+          SolrRequest operation = suggester.getOperation();
+          if (operation == null) break;
+          log.info("Computed Plan: {}", operation);
+          Map<String, Object> props = context.getProperties();
+          props.compute("operations", (k, v) -> {
+            List<SolrRequest> operations = (List<SolrRequest>) v;
+            if (operations == null) operations = new ArrayList<>();
+            operations.add(operation);
+            return operations;
+          });
+          // todo nocommit following code is temporarily disabled until iterative calling of suggester is supported
+//          session = suggester.getSession();
+//          suggester = getSuggester(session, event);
+          break;
+        }
+      }
+    } catch (KeeperException e) {
+      log.error("ZooKeeperException while processing event: " + event, e);
+    } catch (InterruptedException e) {
+      log.error("Interrupted while processing event: " + event, e);
+    } catch (IOException e) {
+      log.error("IOException while processing event: " + event, e);
+    } catch (Exception e) {
+      log.error("Unexpected exception while processing event: " + event, e);
+    }
   }
 
-  @Override
-  public void process(TriggerEvent event) {
-
+  private Policy.Suggester getSuggester(Policy.Session session, TriggerEvent event) {
+    Policy.Suggester suggester;
+    switch (event.getEventType()) {
+      case NODEADDED:
+        NodeAddedTrigger.NodeAddedEvent nodeAddedEvent = (NodeAddedTrigger.NodeAddedEvent) event;
+        suggester = session.getSuggester(CollectionParams.CollectionAction.MOVEREPLICA)
+            .hint(Policy.Suggester.Hint.TARGET_NODE, nodeAddedEvent.getProperty(TriggerEvent.NODE_NAME));
+        log.debug("Created suggester with targetNode: {}", nodeAddedEvent.getProperty(TriggerEvent.NODE_NAME));
+        break;
+      case NODELOST:
+        NodeLostTrigger.NodeLostEvent nodeLostEvent = (NodeLostTrigger.NodeLostEvent) event;
+        suggester = session.getSuggester(CollectionParams.CollectionAction.MOVEREPLICA)
+            .hint(Policy.Suggester.Hint.SRC_NODE, nodeLostEvent.getProperty(TriggerEvent.NODE_NAME));
+        log.debug("Created suggester with srcNode: {}", nodeLostEvent.getProperty(TriggerEvent.NODE_NAME));
+        break;
+      default:
+        throw new UnsupportedOperationException("No support for events other than nodeAdded and nodeLost, received: " + event.getEventType());
+    }
+    return suggester;
   }
 }

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/ExecutePlanAction.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ExecutePlanAction.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ExecutePlanAction.java
index 90a7cf7..d6c288b 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ExecutePlanAction.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ExecutePlanAction.java
@@ -40,12 +40,7 @@ public class ExecutePlanAction implements TriggerAction {
   }
 
   @Override
-  public String getClassName() {
-    return null;
-  }
-
-  @Override
-  public void process(TriggerEvent event) {
+  public void process(TriggerEvent event, ActionContext actionContext) {
 
   }
 }

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/LogPlanAction.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/LogPlanAction.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/LogPlanAction.java
index f89e8d9..7b2de80 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/LogPlanAction.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/LogPlanAction.java
@@ -40,12 +40,7 @@ public class LogPlanAction implements TriggerAction {
   }
 
   @Override
-  public String getClassName() {
-    return null;
-  }
-
-  @Override
-  public void process(TriggerEvent event) {
+  public void process(TriggerEvent event, ActionContext actionContext) {
 
   }
 }

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/OverseerTriggerThread.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/OverseerTriggerThread.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/OverseerTriggerThread.java
index 91146b6..d7fc47e 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/OverseerTriggerThread.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/OverseerTriggerThread.java
@@ -79,7 +79,7 @@ public class OverseerTriggerThread implements Runnable, Closeable {
     this.zkController = zkController;
     zkStateReader = zkController.getZkStateReader();
     zkClient = zkController.getZkClient();
-    scheduledTriggers = new ScheduledTriggers(zkClient);
+    scheduledTriggers = new ScheduledTriggers(zkController);
     triggerFactory = new AutoScaling.TriggerFactory(zkController.getCoreContainer());
   }
 

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/ScheduledTriggers.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ScheduledTriggers.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ScheduledTriggers.java
index d595d4d..a15b2d1 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/ScheduledTriggers.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/ScheduledTriggers.java
@@ -22,6 +22,7 @@ import java.io.IOException;
 import java.lang.invoke.MethodHandles;
 import java.util.ArrayList;
 import java.util.Collections;
+import java.util.HashMap;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
@@ -37,10 +38,12 @@ import java.util.concurrent.atomic.AtomicBoolean;
 import org.apache.lucene.store.AlreadyClosedException;
 import org.apache.solr.cloud.ActionThrottle;
 import org.apache.solr.cloud.Overseer;
+import org.apache.solr.cloud.ZkController;
 import org.apache.solr.common.cloud.SolrZkClient;
 import org.apache.solr.common.cloud.ZkStateReader;
 import org.apache.solr.common.util.ExecutorUtil;
 import org.apache.solr.common.util.IOUtils;
+import org.apache.solr.core.CoreContainer;
 import org.apache.solr.util.DefaultSolrThreadFactory;
 import org.apache.zookeeper.KeeperException;
 import org.apache.zookeeper.Op;
@@ -80,7 +83,9 @@ public class ScheduledTriggers implements Closeable {
 
   private final Overseer.Stats queueStats;
 
-  public ScheduledTriggers(SolrZkClient zkClient) {
+  private final CoreContainer coreContainer;
+
+  public ScheduledTriggers(ZkController zkController) {
     // todo make the core pool size configurable
     // it is important to use more than one because a time taking trigger can starve other scheduled triggers
     // ideally we should have as many core threads as the number of triggers but firstly, we don't know beforehand
@@ -93,7 +98,8 @@ public class ScheduledTriggers implements Closeable {
     actionExecutor = ExecutorUtil.newMDCAwareSingleThreadExecutor(new DefaultSolrThreadFactory("AutoscalingActionExecutor"));
     // todo make the wait time configurable
     actionThrottle = new ActionThrottle("action", DEFAULT_MIN_MS_BETWEEN_ACTIONS);
-    this.zkClient = zkClient;
+    this.coreContainer = zkController.getCoreContainer();
+    this.zkClient = zkController.getZkClient();
     queueStats = new Overseer.Stats();
   }
 
@@ -150,9 +156,10 @@ public class ScheduledTriggers implements Closeable {
               // let the action executor thread wait instead of the trigger thread so we use the throttle here
               actionThrottle.minimumWaitBetweenActions();
               actionThrottle.markAttemptingAction();
+              ActionContext actionContext = new ActionContext(coreContainer, newTrigger, new HashMap<>());
               for (TriggerAction action : actions) {
                 try {
-                  action.process(event);
+                  action.process(event, actionContext);
                 } catch (Exception e) {
                   log.error("Error executing action: " + action.getName() + " for trigger event: " + event, e);
                   throw e;

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/java/org/apache/solr/cloud/autoscaling/TriggerAction.java
----------------------------------------------------------------------
diff --git a/solr/core/src/java/org/apache/solr/cloud/autoscaling/TriggerAction.java b/solr/core/src/java/org/apache/solr/cloud/autoscaling/TriggerAction.java
index b00dfd0..e67a217 100644
--- a/solr/core/src/java/org/apache/solr/cloud/autoscaling/TriggerAction.java
+++ b/solr/core/src/java/org/apache/solr/cloud/autoscaling/TriggerAction.java
@@ -26,9 +26,7 @@ import org.apache.solr.util.plugin.MapInitializedPlugin;
  */
 public interface TriggerAction extends MapInitializedPlugin, Closeable {
   // todo nocommit
-  public String getName();
+  String getName();
 
-  public String getClassName();
-
-  public void process(TriggerEvent event);
+  void process(TriggerEvent event, ActionContext context);
 }

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/test/org/apache/solr/cloud/autoscaling/ComputePlanActionTest.java
----------------------------------------------------------------------
diff --git a/solr/core/src/test/org/apache/solr/cloud/autoscaling/ComputePlanActionTest.java b/solr/core/src/test/org/apache/solr/cloud/autoscaling/ComputePlanActionTest.java
new file mode 100644
index 0000000..d2fcbc3
--- /dev/null
+++ b/solr/core/src/test/org/apache/solr/cloud/autoscaling/ComputePlanActionTest.java
@@ -0,0 +1,221 @@
+/*
+ * 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.solr.cloud.autoscaling;
+
+import java.io.IOException;
+import java.lang.invoke.MethodHandles;
+import java.util.List;
+import java.util.Map;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicReference;
+
+import com.google.common.base.Charsets;
+import org.apache.solr.client.solrj.SolrRequest;
+import org.apache.solr.client.solrj.embedded.JettySolrRunner;
+import org.apache.solr.client.solrj.impl.CloudSolrClient;
+import org.apache.solr.client.solrj.request.CollectionAdminRequest;
+import org.apache.solr.cloud.SolrCloudTestCase;
+import org.apache.solr.common.cloud.ZkStateReader;
+import org.apache.solr.common.params.CollectionParams;
+import org.apache.solr.common.params.SolrParams;
+import org.apache.solr.common.util.NamedList;
+import org.apache.solr.util.LogLevel;
+import org.junit.Before;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import static org.apache.solr.common.params.CollectionParams.CollectionAction.MOVEREPLICA;
+
+/**
+ * Test for {@link ComputePlanAction}
+ */
+@LogLevel("org.apache.solr.cloud.autoscaling=DEBUG")
+public class ComputePlanActionTest extends SolrCloudTestCase {
+  private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
+
+  private static final AtomicBoolean fired = new AtomicBoolean(false);
+  private static CountDownLatch triggerFiredLatch = new CountDownLatch(1);
+  private static final AtomicReference<Object> eventContextRef = new AtomicReference<>();
+
+  private String path = null;
+
+  @BeforeClass
+  public static void setupCluster() throws Exception {
+    configureCluster(1)
+        .addConfig("conf", configset("cloud-minimal"))
+        .configure();
+  }
+
+  @Before
+  public void setUp() throws Exception {
+    super.setUp();
+
+    fired.set(false);
+    triggerFiredLatch = new CountDownLatch(1);
+    eventContextRef.set(null);
+    this.path = "/admin/autoscaling";
+
+    // remove everything from autoscaling.json in ZK
+    zkClient().setData(ZkStateReader.SOLR_AUTOSCALING_CONF_PATH, "{}".getBytes(Charsets.UTF_8), true);
+
+    CloudSolrClient solrClient = cluster.getSolrClient();
+    String setClusterPolicyCommand = "{" +
+        " 'set-cluster-policy': [" +
+        "      {'cores':'<10', 'node':'#ANY'}," +
+        "      {'replica':'<2', 'shard': '#EACH', 'node': '#ANY'}," +
+        "      {'nodeRole':'overseer', 'replica':0}" +
+        "    ]" +
+        "}";
+    SolrRequest req = new AutoScalingHandlerTest.AutoScalingRequest(SolrRequest.METHOD.POST, path, setClusterPolicyCommand);
+    NamedList<Object> response = solrClient.request(req);
+    assertEquals(response.get("result").toString(), "success");
+
+    String setClusterPreferencesCommand = "{" +
+        "'set-cluster-preferences': [" +
+        "{'minimize': 'cores','precision': 3}," +
+        "{'maximize': 'freedisk','precision': 100}]" +
+        "}";
+    req = new AutoScalingHandlerTest.AutoScalingRequest(SolrRequest.METHOD.POST, path, setClusterPreferencesCommand);
+    response = solrClient.request(req);
+    assertEquals(response.get("result").toString(), "success");
+  }
+
+  @Test
+  public void testNodeLost() throws Exception  {
+    // let's start a node so that we have at least two
+    JettySolrRunner runner = cluster.startJettySolrRunner();
+
+    CloudSolrClient solrClient = cluster.getSolrClient();
+    String setTriggerCommand = "{" +
+        "'set-trigger' : {" +
+        "'name' : 'node_lost_trigger'," +
+        "'event' : 'nodeLost'," +
+        "'waitFor' : '1s'," +
+        "'enabled' : true," +
+        "'actions' : [{'name':'compute_plan', 'class' : 'solr.ComputePlanAction'}," +
+        "{'name':'test','class':'" + ComputePlanActionTest.AssertingTriggerAction.class.getName() + "'}]" +
+        "}}";
+    SolrRequest req = new AutoScalingHandlerTest.AutoScalingRequest(SolrRequest.METHOD.POST, path, setTriggerCommand);
+    NamedList<Object> response = solrClient.request(req);
+    assertEquals(response.get("result").toString(), "success");
+
+    CollectionAdminRequest.Create create = CollectionAdminRequest.createCollection("testNodeLost",
+        1, 2);
+    create.setMaxShardsPerNode(1);
+    create.process(solrClient);
+
+    waitForState("Timed out waiting for replicas of new collection to be active",
+        "testNodeLost", (liveNodes, collectionState) -> collectionState.getReplicas().stream().allMatch(replica -> replica.isActive(liveNodes)));
+
+    zkClient().printLayoutToStdOut();
+
+    cluster.startJettySolrRunner();
+    cluster.waitForAllNodes(30);
+
+    for (int i = 0; i < cluster.getJettySolrRunners().size(); i++) {
+      JettySolrRunner jettySolrRunner = cluster.getJettySolrRunners().get(i);
+      if (jettySolrRunner == runner)  {
+        cluster.stopJettySolrRunner(i);
+        break;
+      }
+    }
+    cluster.waitForAllNodes(30);
+
+    assertTrue("Trigger was not fired even after 5 seconds", triggerFiredLatch.await(180, TimeUnit.SECONDS));
+    assertTrue(fired.get());
+    Map context = (Map) eventContextRef.get();
+    assertNotNull(context);
+    List<SolrRequest> operations = (List<SolrRequest>) context.get("operations");
+    assertNotNull("The operations computed by ComputePlanAction should not be null", operations);
+    assertEquals("ComputePlanAction should have computed exactly 1 operation", 1, operations.size());
+    SolrRequest solrRequest = operations.get(0);
+    SolrParams params = solrRequest.getParams();
+    assertEquals("Expected MOVEREPLICA action after adding node", MOVEREPLICA, CollectionParams.CollectionAction.get(params.get("operation")));
+    String nodeAdded = params.get("srcNode");
+    assertEquals("Unexpected node in computed operation", runner.getNodeName(), nodeAdded);
+  }
+
+  @Test
+  public void testNodeAdded() throws Exception {
+    CloudSolrClient solrClient = cluster.getSolrClient();
+    String setTriggerCommand = "{" +
+        "'set-trigger' : {" +
+        "'name' : 'node_added_trigger'," +
+        "'event' : 'nodeAdded'," +
+        "'waitFor' : '1s'," +
+        "'enabled' : true," +
+        "'actions' : [{'name':'compute_plan', 'class' : 'solr.ComputePlanAction'}," +
+        "{'name':'test','class':'" + ComputePlanActionTest.AssertingTriggerAction.class.getName() + "'}]" +
+        "}}";
+    SolrRequest req = new AutoScalingHandlerTest.AutoScalingRequest(SolrRequest.METHOD.POST, path, setTriggerCommand);
+    NamedList<Object> response = solrClient.request(req);
+    assertEquals(response.get("result").toString(), "success");
+
+    CollectionAdminRequest.Create create = CollectionAdminRequest.createCollection("testNodeAdded",
+        1, 2);
+    create.setMaxShardsPerNode(2);
+    create.process(solrClient);
+
+    waitForState("Timed out waiting for replicas of new collection to be active",
+        "testNodeAdded", (liveNodes, collectionState) -> collectionState.getReplicas().stream().allMatch(replica -> replica.isActive(liveNodes)));
+
+    JettySolrRunner runner = cluster.startJettySolrRunner();
+    assertTrue("Trigger was not fired even after 5 seconds", triggerFiredLatch.await(5, TimeUnit.SECONDS));
+    assertTrue(fired.get());
+    Map context = (Map) eventContextRef.get();
+    assertNotNull(context);
+    List<SolrRequest> operations = (List<SolrRequest>) context.get("operations");
+    assertNotNull("The operations computed by ComputePlanAction should not be null", operations);
+    assertEquals("ComputePlanAction should have computed exactly 1 operation", 1, operations.size());
+    SolrRequest request = operations.get(0);
+    SolrParams params = request.getParams();
+    assertEquals("Expected MOVEREPLICA action after adding node", MOVEREPLICA, CollectionParams.CollectionAction.get(params.get("action")));
+    String nodeAdded = params.get("targetNode");
+    assertEquals("Unexpected node in computed operation", runner.getNodeName(), nodeAdded);
+  }
+
+  public static class AssertingTriggerAction implements TriggerAction {
+
+    @Override
+    public String getName() {
+      return null;
+    }
+
+    @Override
+    public void process(TriggerEvent event, ActionContext context) {
+      if (fired.compareAndSet(false, true)) {
+        eventContextRef.set(context.getProperties());
+        triggerFiredLatch.countDown();
+      }
+    }
+
+    @Override
+    public void close() throws IOException {
+
+    }
+
+    @Override
+    public void init(Map<String, String> args) {
+
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeAddedTriggerTest.java
----------------------------------------------------------------------
diff --git a/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeAddedTriggerTest.java b/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeAddedTriggerTest.java
index f874339..8e24a00 100644
--- a/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeAddedTriggerTest.java
+++ b/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeAddedTriggerTest.java
@@ -176,12 +176,7 @@ public class NodeAddedTriggerTest extends SolrCloudTestCase {
     }
 
     @Override
-    public String getClassName() {
-      return getClass().getName();
-    }
-
-    @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
 
     }
 

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeLostTriggerTest.java
----------------------------------------------------------------------
diff --git a/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeLostTriggerTest.java b/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeLostTriggerTest.java
index c5c3c47..82e1326 100644
--- a/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeLostTriggerTest.java
+++ b/solr/core/src/test/org/apache/solr/cloud/autoscaling/NodeLostTriggerTest.java
@@ -190,12 +190,7 @@ public class NodeLostTriggerTest extends SolrCloudTestCase {
     }
 
     @Override
-    public String getClassName() {
-      return getClass().getName();
-    }
-
-    @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
 
     }
 

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/core/src/test/org/apache/solr/cloud/autoscaling/TriggerIntegrationTest.java
----------------------------------------------------------------------
diff --git a/solr/core/src/test/org/apache/solr/cloud/autoscaling/TriggerIntegrationTest.java b/solr/core/src/test/org/apache/solr/cloud/autoscaling/TriggerIntegrationTest.java
index 70f0fdc..bc8417c 100644
--- a/solr/core/src/test/org/apache/solr/cloud/autoscaling/TriggerIntegrationTest.java
+++ b/solr/core/src/test/org/apache/solr/cloud/autoscaling/TriggerIntegrationTest.java
@@ -252,7 +252,7 @@ public class TriggerIntegrationTest extends SolrCloudTestCase {
     private final AtomicBoolean onlyOnce = new AtomicBoolean(false);
 
     @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
       boolean locked = lock.tryLock();
       if (!locked)  {
         log.info("We should never have a tryLock fail because actions are never supposed to be executed concurrently");
@@ -584,12 +584,7 @@ public class TriggerIntegrationTest extends SolrCloudTestCase {
     }
 
     @Override
-    public String getClassName() {
-      return this.getClass().getName();
-    }
-
-    @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
       try {
         if (triggerFired.compareAndSet(false, true))  {
           events.add(event);
@@ -630,12 +625,7 @@ public class TriggerIntegrationTest extends SolrCloudTestCase {
     }
 
     @Override
-    public String getClassName() {
-      return this.getClass().getName();
-    }
-
-    @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
       log.info("-- event: " + event);
       events.add(event);
       getActionStarted().countDown();
@@ -816,12 +806,7 @@ public class TriggerIntegrationTest extends SolrCloudTestCase {
     }
 
     @Override
-    public String getClassName() {
-      return this.getClass().getName();
-    }
-
-    @Override
-    public void process(TriggerEvent event) {
+    public void process(TriggerEvent event, ActionContext actionContext) {
       boolean locked = lock.tryLock();
       if (!locked)  {
         log.info("We should never have a tryLock fail because actions are never supposed to be executed concurrently");

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Policy.java
----------------------------------------------------------------------
diff --git a/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Policy.java b/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Policy.java
index dd9dfc5..5efb996 100644
--- a/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Policy.java
+++ b/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Policy.java
@@ -32,6 +32,7 @@ import java.util.Objects;
 import java.util.Set;
 import java.util.SortedSet;
 import java.util.TreeSet;
+import java.util.function.Predicate;
 import java.util.function.Supplier;
 import java.util.stream.Collectors;
 
@@ -378,6 +379,10 @@ public class Policy implements MapWriter {
             }
           }
         }
+        if(hints.get(Hint.SRC_NODE) != null && session.matrix.stream().noneMatch(row -> row.node.equals(hints.get(Hint.SRC_NODE)))){
+          // the source node is dead so live nodes may not have it
+          session.matrix.add(new Row((String) hints.get(Hint.SRC_NODE), session.getPolicy().params, session.dataProvider));
+        }
         session.applyRules();
         originalViolations.addAll(session.getViolations());
         this.operation = init();

http://git-wip-us.apache.org/repos/asf/lucene-solr/blob/ebf29832/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Row.java
----------------------------------------------------------------------
diff --git a/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Row.java b/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Row.java
index f7ab5ca..a13633d 100644
--- a/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Row.java
+++ b/solr/solrj/src/java/org/apache/solr/cloud/autoscaling/Row.java
@@ -39,12 +39,14 @@ class Row implements MapWriter {
   Map<String, Map<String, List<ReplicaInfo>>> collectionVsShardVsReplicas;
   List<Clause> violations = new ArrayList<>();
   boolean anyValueMissing = false;
+  boolean isLive = true;
 
   Row(String node, List<String> params, ClusterDataProvider dataProvider) {
     collectionVsShardVsReplicas = dataProvider.getReplicaInfo(node, params);
     if (collectionVsShardVsReplicas == null) collectionVsShardVsReplicas = new HashMap<>();
     this.node = node;
     cells = new Cell[params.size()];
+    isLive = dataProvider.getNodes().contains(node);
     Map<String, Object> vals = dataProvider.getNodeValues(node, params);
     for (int i = 0; i < params.size(); i++) {
       String s = params.get(i);