浏览代码

Remove `_shutdown` API

Thsi commit removes the `_shutdown` API entirely without any replacement.
Nodes should be managed from the operating system not via REST APIs
Simon Willnauer 10 年之前
父节点
当前提交
d164526d27
共有 19 个文件被更改,包括 9 次插入857 次删除
  1. 0 2
      docs/reference/cluster.asciidoc
  2. 0 57
      docs/reference/cluster/nodes-shutdown.asciidoc
  3. 5 0
      docs/reference/migration/migrate_2_0.asciidoc
  4. 0 10
      docs/reference/setup/upgrade.asciidoc
  5. 0 27
      rest-api-spec/api/nodes.shutdown.json
  6. 0 3
      src/main/java/org/elasticsearch/action/ActionModule.java
  7. 0 45
      src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownAction.java
  8. 0 110
      src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownRequest.java
  9. 0 72
      src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownRequestBuilder.java
  10. 0 73
      src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownResponse.java
  11. 0 306
      src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/TransportNodesShutdownAction.java
  12. 0 26
      src/main/java/org/elasticsearch/client/ClusterAdminClient.java
  13. 0 19
      src/main/java/org/elasticsearch/client/Requests.java
  14. 0 19
      src/main/java/org/elasticsearch/client/support/AbstractClusterAdminClient.java
  15. 0 2
      src/main/java/org/elasticsearch/rest/action/RestActionModule.java
  16. 0 73
      src/main/java/org/elasticsearch/rest/action/admin/cluster/node/shutdown/RestNodesShutdownAction.java
  17. 1 4
      src/test/java/org/elasticsearch/client/AbstractClientHeadersTests.java
  18. 3 1
      src/test/java/org/elasticsearch/stresstest/manyindices/ManyNodesManyIndicesRecoveryStressTest.java
  19. 0 8
      src/test/java/org/elasticsearch/test/ExternalNode.java

+ 0 - 2
docs/reference/cluster.asciidoc

@@ -46,5 +46,3 @@ include::cluster/nodes-stats.asciidoc[]
 include::cluster/nodes-info.asciidoc[]
 include::cluster/nodes-info.asciidoc[]
 
 
 include::cluster/nodes-hot-threads.asciidoc[]
 include::cluster/nodes-hot-threads.asciidoc[]
-
-include::cluster/nodes-shutdown.asciidoc[]

+ 0 - 57
docs/reference/cluster/nodes-shutdown.asciidoc

@@ -1,57 +0,0 @@
-[[cluster-nodes-shutdown]]
-== Nodes Shutdown
-
-The nodes shutdown API allows to shutdown one or more (or all) nodes in
-the cluster. Here is an example of shutting the `_local` node the
-request is directed to:
-
-[source,js]
---------------------------------------------------
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/_local/_shutdown'
---------------------------------------------------
-
-Specific node(s) can be shutdown as well using their respective node ids
-(or other selective options as explained
-<<cluster-nodes,here>> .):
-
-[source,js]
---------------------------------------------------
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/nodeId1,nodeId2/_shutdown'
---------------------------------------------------
-
-The master (of the cluster) can also be shutdown using:
-
-[source,js]
---------------------------------------------------
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/_master/_shutdown'
---------------------------------------------------
-
-Finally, all nodes can be shutdown using one of the options below:
-
-[source,js]
---------------------------------------------------
-$ curl -XPOST 'http://localhost:9200/_shutdown'
-
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/_shutdown'
-
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/_all/_shutdown'
---------------------------------------------------
-
-[float]
-[[delay]]
-=== Delay
-
-By default, the shutdown will be executed after a 1 second delay (`1s`).
-The delay can be customized by setting the `delay` parameter in a time
-value format. For example:
-
-[source,js]
---------------------------------------------------
-$ curl -XPOST 'http://localhost:9200/_cluster/nodes/_local/_shutdown?delay=10s'
---------------------------------------------------
-
-[float]
-=== Disable Shutdown
-
-The shutdown API can be disabled by setting `action.disable_shutdown` in
-the node configuration.

+ 5 - 0
docs/reference/migration/migrate_2_0.asciidoc

@@ -405,3 +405,8 @@ The obsolete parameters `expand_wildcards_open` and `expand_wildcards_close` are
 supported by the snapshot and restore operations. These parameters have been replaced by
 supported by the snapshot and restore operations. These parameters have been replaced by
 a single `expand_wildcards` parameter. See <<multi-index,the multi-index docs>> for more.
 a single `expand_wildcards` parameter. See <<multi-index,the multi-index docs>> for more.
 
 
+=== `_shutdown` API
+
+The `_shutdown` API has been removed without a replacement. Nodes should be managed via operating
+systems and the provided start/stop scripts.
+

+ 0 - 10
docs/reference/setup/upgrade.asciidoc

@@ -94,11 +94,6 @@ This syntax applies to Elasticsearch 1.0 and later:
 
 
 * Shut down a single node within the cluster.
 * Shut down a single node within the cluster.
 
 
-[source,sh]
---------------------------------------------
-curl -XPOST 'http://localhost:9200/_cluster/nodes/_local/_shutdown'
---------------------------------------------
-
 * Confirm that all shards are correctly reallocated to the remaining running nodes.
 * Confirm that all shards are correctly reallocated to the remaining running nodes.
 
 
 * Upgrade the stopped node.  To upgrade using a zip or compressed tarball from elasticsearch.org:
 * Upgrade the stopped node.  To upgrade using a zip or compressed tarball from elasticsearch.org:
@@ -149,11 +144,6 @@ This syntax is from versions prior to 1.0:
 --------------------------------------------------
 --------------------------------------------------
 
 
 * Stop all Elasticsearch services on all nodes in the cluster.
 * Stop all Elasticsearch services on all nodes in the cluster.
-[source,sh]
-------------------------------------------------------
-	curl -XPOST 'http://localhost:9200/_shutdown'
-------------------------------------------------------
-
 * On the first node to be upgraded, extract the archive or install the new package as described above in the Rolling Upgrades section.  Repeat for all nodes.
 * On the first node to be upgraded, extract the archive or install the new package as described above in the Rolling Upgrades section.  Repeat for all nodes.
 
 
 * After upgrading Elasticsearch on all nodes is complete, the cluster can be started by starting each node individually.
 * After upgrading Elasticsearch on all nodes is complete, the cluster can be started by starting each node individually.

+ 0 - 27
rest-api-spec/api/nodes.shutdown.json

@@ -1,27 +0,0 @@
-{
-  "nodes.shutdown": {
-    "documentation": "http://www.elastic.co/guide/en/elasticsearch/reference/master/cluster-nodes-shutdown.html",
-    "methods": ["POST"],
-    "url": {
-      "path": "/_shutdown",
-      "paths": ["/_shutdown", "/_cluster/nodes/_shutdown", "/_cluster/nodes/{node_id}/_shutdown"],
-      "parts": {
-        "node_id": {
-          "type" : "list",
-          "description" : "A comma-separated list of node IDs or names to perform the operation on; use `_local` to perform the operation on the node you're connected to, leave empty to perform the operation on all nodes"
-        }
-      },
-      "params": {
-        "delay": {
-          "type" : "time",
-          "description" : "Set the delay for the operation (default: 1s)"
-        },
-        "exit": {
-          "type" : "boolean",
-          "description" : "Exit the JVM as well (default: true)"
-        }
-      }
-    },
-    "body": null
-  }
-}

+ 0 - 3
src/main/java/org/elasticsearch/action/ActionModule.java

@@ -28,8 +28,6 @@ import org.elasticsearch.action.admin.cluster.node.hotthreads.TransportNodesHotT
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoAction;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoAction;
 import org.elasticsearch.action.admin.cluster.node.info.TransportNodesInfoAction;
 import org.elasticsearch.action.admin.cluster.node.info.TransportNodesInfoAction;
 import org.elasticsearch.action.admin.cluster.node.liveness.TransportLivenessAction;
 import org.elasticsearch.action.admin.cluster.node.liveness.TransportLivenessAction;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownAction;
-import org.elasticsearch.action.admin.cluster.node.shutdown.TransportNodesShutdownAction;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsAction;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsAction;
 import org.elasticsearch.action.admin.cluster.node.stats.TransportNodesStatsAction;
 import org.elasticsearch.action.admin.cluster.node.stats.TransportNodesStatsAction;
 import org.elasticsearch.action.admin.cluster.repositories.delete.DeleteRepositoryAction;
 import org.elasticsearch.action.admin.cluster.repositories.delete.DeleteRepositoryAction;
@@ -222,7 +220,6 @@ public class ActionModule extends AbstractModule {
         bind(ActionFilters.class).asEagerSingleton();
         bind(ActionFilters.class).asEagerSingleton();
         registerAction(NodesInfoAction.INSTANCE, TransportNodesInfoAction.class);
         registerAction(NodesInfoAction.INSTANCE, TransportNodesInfoAction.class);
         registerAction(NodesStatsAction.INSTANCE, TransportNodesStatsAction.class);
         registerAction(NodesStatsAction.INSTANCE, TransportNodesStatsAction.class);
-        registerAction(NodesShutdownAction.INSTANCE, TransportNodesShutdownAction.class);
         registerAction(NodesHotThreadsAction.INSTANCE, TransportNodesHotThreadsAction.class);
         registerAction(NodesHotThreadsAction.INSTANCE, TransportNodesHotThreadsAction.class);
 
 
         registerAction(ClusterStatsAction.INSTANCE, TransportClusterStatsAction.class);
         registerAction(ClusterStatsAction.INSTANCE, TransportClusterStatsAction.class);

+ 0 - 45
src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownAction.java

@@ -1,45 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.action.admin.cluster.node.shutdown;
-
-import org.elasticsearch.action.admin.cluster.ClusterAction;
-import org.elasticsearch.client.ClusterAdminClient;
-
-/**
- */
-public class NodesShutdownAction extends ClusterAction<NodesShutdownRequest, NodesShutdownResponse, NodesShutdownRequestBuilder> {
-
-    public static final NodesShutdownAction INSTANCE = new NodesShutdownAction();
-    public static final String NAME = "cluster:admin/nodes/shutdown";
-
-    private NodesShutdownAction() {
-        super(NAME);
-    }
-
-    @Override
-    public NodesShutdownResponse newResponse() {
-        return new NodesShutdownResponse();
-    }
-
-    @Override
-    public NodesShutdownRequestBuilder newRequestBuilder(ClusterAdminClient client) {
-        return new NodesShutdownRequestBuilder(client);
-    }
-}

+ 0 - 110
src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownRequest.java

@@ -1,110 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.action.admin.cluster.node.shutdown;
-
-import org.elasticsearch.action.ActionRequestValidationException;
-import org.elasticsearch.action.support.master.MasterNodeOperationRequest;
-import org.elasticsearch.common.Strings;
-import org.elasticsearch.common.io.stream.StreamInput;
-import org.elasticsearch.common.io.stream.StreamOutput;
-import org.elasticsearch.common.unit.TimeValue;
-
-import java.io.IOException;
-
-import static org.elasticsearch.common.unit.TimeValue.readTimeValue;
-
-/**
- *
- */
-public class NodesShutdownRequest extends MasterNodeOperationRequest<NodesShutdownRequest> {
-
-    String[] nodesIds = Strings.EMPTY_ARRAY;
-
-    TimeValue delay = TimeValue.timeValueSeconds(1);
-
-    boolean exit = true;
-
-    NodesShutdownRequest() {
-    }
-
-    public NodesShutdownRequest(String... nodesIds) {
-        this.nodesIds = nodesIds;
-    }
-
-    public NodesShutdownRequest nodesIds(String... nodesIds) {
-        this.nodesIds = nodesIds;
-        return this;
-    }
-
-    /**
-     * The delay for the shutdown to occur. Defaults to <tt>1s</tt>.
-     */
-    public NodesShutdownRequest delay(TimeValue delay) {
-        this.delay = delay;
-        return this;
-    }
-
-    public TimeValue delay() {
-        return this.delay;
-    }
-
-    /**
-     * The delay for the shutdown to occur. Defaults to <tt>1s</tt>.
-     */
-    public NodesShutdownRequest delay(String delay) {
-        return delay(TimeValue.parseTimeValue(delay, null));
-    }
-
-    /**
-     * Should the JVM be exited as well or not. Defaults to <tt>true</tt>.
-     */
-    public NodesShutdownRequest exit(boolean exit) {
-        this.exit = exit;
-        return this;
-    }
-
-    /**
-     * Should the JVM be exited as well or not. Defaults to <tt>true</tt>.
-     */
-    public boolean exit() {
-        return exit;
-    }
-
-    @Override
-    public ActionRequestValidationException validate() {
-        return null;
-    }
-
-    @Override
-    public void readFrom(StreamInput in) throws IOException {
-        super.readFrom(in);
-        delay = readTimeValue(in);
-        nodesIds = in.readStringArray();
-        exit = in.readBoolean();
-    }
-
-    @Override
-    public void writeTo(StreamOutput out) throws IOException {
-        super.writeTo(out);
-        delay.writeTo(out);
-        out.writeStringArrayNullable(nodesIds);
-        out.writeBoolean(exit);
-    }
-}

+ 0 - 72
src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownRequestBuilder.java

@@ -1,72 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.action.admin.cluster.node.shutdown;
-
-import org.elasticsearch.action.ActionListener;
-import org.elasticsearch.action.support.master.MasterNodeOperationRequestBuilder;
-import org.elasticsearch.client.ClusterAdminClient;
-import org.elasticsearch.common.unit.TimeValue;
-
-/**
- *
- */
-public class NodesShutdownRequestBuilder extends MasterNodeOperationRequestBuilder<NodesShutdownRequest, NodesShutdownResponse, NodesShutdownRequestBuilder, ClusterAdminClient> {
-
-    public NodesShutdownRequestBuilder(ClusterAdminClient clusterClient) {
-        super(clusterClient, new NodesShutdownRequest());
-    }
-
-    /**
-     * The nodes ids to restart.
-     */
-    public NodesShutdownRequestBuilder setNodesIds(String... nodesIds) {
-        request.nodesIds(nodesIds);
-        return this;
-    }
-
-    /**
-     * The delay for the restart to occur. Defaults to <tt>1s</tt>.
-     */
-    public NodesShutdownRequestBuilder setDelay(TimeValue delay) {
-        request.delay(delay);
-        return this;
-    }
-
-    /**
-     * The delay for the restart to occur. Defaults to <tt>1s</tt>.
-     */
-    public NodesShutdownRequestBuilder setDelay(String delay) {
-        request.delay(delay);
-        return this;
-    }
-
-    /**
-     * Should the JVM be exited as well or not. Defaults to <tt>true</tt>.
-     */
-    public NodesShutdownRequestBuilder setExit(boolean exit) {
-        request.exit(exit);
-        return this;
-    }
-
-    @Override
-    protected void doExecute(ActionListener<NodesShutdownResponse> listener) {
-        client.nodesShutdown(request, listener);
-    }
-}

+ 0 - 73
src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/NodesShutdownResponse.java

@@ -1,73 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.action.admin.cluster.node.shutdown;
-
-import org.elasticsearch.action.ActionResponse;
-import org.elasticsearch.cluster.ClusterName;
-import org.elasticsearch.cluster.node.DiscoveryNode;
-import org.elasticsearch.common.io.stream.StreamInput;
-import org.elasticsearch.common.io.stream.StreamOutput;
-
-import java.io.IOException;
-
-/**
- *
- */
-public class NodesShutdownResponse extends ActionResponse {
-
-    private ClusterName clusterName;
-    private DiscoveryNode[] nodes;
-
-    NodesShutdownResponse() {
-    }
-
-    public NodesShutdownResponse(ClusterName clusterName, DiscoveryNode[] nodes) {
-        this.clusterName = clusterName;
-        this.nodes = nodes;
-    }
-
-    public ClusterName getClusterName() {
-        return this.clusterName;
-    }
-
-    public DiscoveryNode[] getNodes() {
-        return this.nodes;
-    }
-
-    @Override
-    public void readFrom(StreamInput in) throws IOException {
-        super.readFrom(in);
-        clusterName = ClusterName.readClusterName(in);
-        nodes = new DiscoveryNode[in.readVInt()];
-        for (int i = 0; i < nodes.length; i++) {
-            nodes[i] = DiscoveryNode.readNode(in);
-        }
-    }
-
-    @Override
-    public void writeTo(StreamOutput out) throws IOException {
-        super.writeTo(out);
-        clusterName.writeTo(out);
-        out.writeVInt(nodes.length);
-        for (DiscoveryNode node : nodes) {
-            node.writeTo(out);
-        }
-    }
-}

+ 0 - 306
src/main/java/org/elasticsearch/action/admin/cluster/node/shutdown/TransportNodesShutdownAction.java

@@ -1,306 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.action.admin.cluster.node.shutdown;
-
-import com.carrotsearch.hppc.ObjectOpenHashSet;
-import com.carrotsearch.hppc.cursors.ObjectCursor;
-import org.elasticsearch.ElasticsearchException;
-import org.elasticsearch.ElasticsearchIllegalStateException;
-import org.elasticsearch.action.ActionListener;
-import org.elasticsearch.action.support.ActionFilters;
-import org.elasticsearch.action.support.master.TransportMasterNodeOperationAction;
-import org.elasticsearch.cluster.ClusterName;
-import org.elasticsearch.cluster.ClusterService;
-import org.elasticsearch.cluster.ClusterState;
-import org.elasticsearch.cluster.block.ClusterBlockException;
-import org.elasticsearch.cluster.block.ClusterBlockLevel;
-import org.elasticsearch.cluster.node.DiscoveryNode;
-import org.elasticsearch.common.inject.Inject;
-import org.elasticsearch.common.io.stream.StreamInput;
-import org.elasticsearch.common.io.stream.StreamOutput;
-import org.elasticsearch.common.settings.Settings;
-import org.elasticsearch.common.unit.TimeValue;
-import org.elasticsearch.node.Node;
-import org.elasticsearch.threadpool.ThreadPool;
-import org.elasticsearch.transport.*;
-
-import java.io.IOException;
-import java.util.concurrent.CountDownLatch;
-
-/**
- *
- */
-public class TransportNodesShutdownAction extends TransportMasterNodeOperationAction<NodesShutdownRequest, NodesShutdownResponse> {
-
-    public static final String SHUTDOWN_NODE_ACTION_NAME = NodesShutdownAction.NAME + "[n]";
-
-    private final Node node;
-    private final ClusterName clusterName;
-    private final boolean disabled;
-    private final TimeValue delay;
-
-    @Inject
-    public TransportNodesShutdownAction(Settings settings, TransportService transportService, ClusterService clusterService, ThreadPool threadPool,
-                                        Node node, ClusterName clusterName, ActionFilters actionFilters) {
-        super(settings, NodesShutdownAction.NAME, transportService, clusterService, threadPool, actionFilters, NodesShutdownRequest.class);
-        this.node = node;
-        this.clusterName = clusterName;
-        this.disabled = settings.getAsBoolean("action.disable_shutdown", this.settings.getAsBoolean("action.admin.cluster.node.shutdown.disabled", false));
-        this.delay = this.settings.getAsTime("action.admin.cluster.node.shutdown.delay", TimeValue.timeValueMillis(200));
-
-        this.transportService.registerRequestHandler(SHUTDOWN_NODE_ACTION_NAME, NodeShutdownRequest.class, ThreadPool.Names.SAME, new NodeShutdownRequestHandler());
-    }
-
-    @Override
-    protected String executor() {
-        return ThreadPool.Names.GENERIC;
-    }
-
-    @Override
-    protected ClusterBlockException checkBlock(NodesShutdownRequest request, ClusterState state) {
-        // Stopping a node impacts the cluster state, so we check for the METADATA_WRITE block here
-        return state.blocks().globalBlockedException(ClusterBlockLevel.METADATA_WRITE);
-    }
-
-    @Override
-    protected NodesShutdownResponse newResponse() {
-        return new NodesShutdownResponse();
-    }
-
-    @Override
-    protected void processBeforeDelegationToMaster(NodesShutdownRequest request, ClusterState state) {
-        String[] nodesIds = request.nodesIds;
-        if (nodesIds != null) {
-            for (int i = 0; i < nodesIds.length; i++) {
-                // replace the _local one, since it looses its meaning when going over to the master...
-                if ("_local".equals(nodesIds[i])) {
-                    nodesIds[i] = state.nodes().localNodeId();
-                }
-            }
-        }
-    }
-
-    @Override
-    protected void masterOperation(final NodesShutdownRequest request, final ClusterState state, final ActionListener<NodesShutdownResponse> listener) throws ElasticsearchException {
-        if (disabled) {
-            throw new ElasticsearchIllegalStateException("Shutdown is disabled");
-        }
-        final ObjectOpenHashSet<DiscoveryNode> nodes = new ObjectOpenHashSet<>();
-        if (state.nodes().isAllNodes(request.nodesIds)) {
-            logger.info("[cluster_shutdown]: requested, shutting down in [{}]", request.delay);
-            nodes.addAll(state.nodes().dataNodes().values());
-            nodes.addAll(state.nodes().masterNodes().values());
-            Thread t = new Thread(new Runnable() {
-                @Override
-                public void run() {
-                    try {
-                        Thread.sleep(request.delay.millis());
-                    } catch (InterruptedException e) {
-                        // ignore
-                    }
-                    // first, stop the cluster service
-                    logger.trace("[cluster_shutdown]: stopping the cluster service so no re-routing will occur");
-                    clusterService.stop();
-
-                    final CountDownLatch latch = new CountDownLatch(nodes.size());
-                    for (ObjectCursor<DiscoveryNode> cursor : nodes) {
-                        final DiscoveryNode node = cursor.value;
-                        if (node.id().equals(state.nodes().masterNodeId())) {
-                            // don't shutdown the master yet...
-                            latch.countDown();
-                        } else {
-                            logger.trace("[cluster_shutdown]: sending shutdown request to [{}]", node);
-                            transportService.sendRequest(node, SHUTDOWN_NODE_ACTION_NAME, new NodeShutdownRequest(request), new EmptyTransportResponseHandler(ThreadPool.Names.SAME) {
-                                @Override
-                                public void handleResponse(TransportResponse.Empty response) {
-                                    logger.trace("[cluster_shutdown]: received shutdown response from [{}]", node);
-                                    latch.countDown();
-                                }
-
-                                @Override
-                                public void handleException(TransportException exp) {
-                                    logger.warn("[cluster_shutdown]: received failed shutdown response from [{}]", exp, node);
-                                    latch.countDown();
-                                }
-                            });
-                        }
-                    }
-                    try {
-                        latch.await();
-                    } catch (InterruptedException e) {
-                        // ignore
-                    }
-                    logger.info("[cluster_shutdown]: done shutting down all nodes except master, proceeding to master");
-
-                    // now, kill the master
-                    logger.trace("[cluster_shutdown]: shutting down the master [{}]", state.nodes().masterNode());
-                    transportService.sendRequest(state.nodes().masterNode(), SHUTDOWN_NODE_ACTION_NAME, new NodeShutdownRequest(request), new EmptyTransportResponseHandler(ThreadPool.Names.SAME) {
-                        @Override
-                        public void handleResponse(TransportResponse.Empty response) {
-                            logger.trace("[cluster_shutdown]: received shutdown response from master");
-                        }
-
-                        @Override
-                        public void handleException(TransportException exp) {
-                            logger.warn("[cluster_shutdown]: received failed shutdown response master", exp);
-                        }
-                    });
-                }
-            });
-            t.start();
-        } else {
-            final String[] nodesIds = state.nodes().resolveNodesIds(request.nodesIds);
-            logger.info("[partial_cluster_shutdown]: requested, shutting down [{}] in [{}]", nodesIds, request.delay);
-
-            for (String nodeId : nodesIds) {
-                final DiscoveryNode node = state.nodes().get(nodeId);
-                if (node != null) {
-                    nodes.add(node);
-                }
-            }
-
-            Thread t = new Thread(new Runnable() {
-                @Override
-                public void run() {
-                    try {
-                        Thread.sleep(request.delay.millis());
-                    } catch (InterruptedException e) {
-                        // ignore
-                    }
-
-                    final CountDownLatch latch = new CountDownLatch(nodesIds.length);
-                    for (String nodeId : nodesIds) {
-                        final DiscoveryNode node = state.nodes().get(nodeId);
-                        if (node == null) {
-                            logger.warn("[partial_cluster_shutdown]: no node to shutdown for node_id [{}]", nodeId);
-                            latch.countDown();
-                            continue;
-                        }
-
-                        logger.trace("[partial_cluster_shutdown]: sending shutdown request to [{}]", node);
-                        transportService.sendRequest(node, SHUTDOWN_NODE_ACTION_NAME, new NodeShutdownRequest(request), new EmptyTransportResponseHandler(ThreadPool.Names.SAME) {
-                            @Override
-                            public void handleResponse(TransportResponse.Empty response) {
-                                logger.trace("[partial_cluster_shutdown]: received shutdown response from [{}]", node);
-                                latch.countDown();
-                            }
-
-                            @Override
-                            public void handleException(TransportException exp) {
-                                logger.warn("[partial_cluster_shutdown]: received failed shutdown response from [{}]", exp, node);
-                                latch.countDown();
-                            }
-                        });
-                    }
-
-                    try {
-                        latch.await();
-                    } catch (InterruptedException e) {
-                        // ignore
-                    }
-
-                    logger.info("[partial_cluster_shutdown]: done shutting down [{}]", ((Object) nodesIds));
-                }
-            });
-            t.start();
-        }
-        listener.onResponse(new NodesShutdownResponse(clusterName, nodes.toArray(DiscoveryNode.class)));
-    }
-
-    private class NodeShutdownRequestHandler implements TransportRequestHandler<NodeShutdownRequest> {
-
-        @Override
-        public void messageReceived(final NodeShutdownRequest request, TransportChannel channel) throws Exception {
-            if (disabled) {
-                throw new ElasticsearchIllegalStateException("Shutdown is disabled");
-            }
-            logger.info("shutting down in [{}]", delay);
-            Thread t = new Thread(new Runnable() {
-                @Override
-                public void run() {
-                    try {
-                        Thread.sleep(delay.millis());
-                    } catch (InterruptedException e) {
-                        // ignore
-                    }
-                    if (!request.exit) {
-                        logger.info("initiating requested shutdown (no exit)...");
-                        try {
-                            node.close();
-                        } catch (Exception e) {
-                            logger.warn("Failed to shutdown", e);
-                        }
-                        return;
-                    }
-                    boolean shutdownWithWrapper = false;
-                    if (System.getProperty("elasticsearch-service") != null) {
-                        try {
-                            Class wrapperManager = settings.getClassLoader().loadClass("org.tanukisoftware.wrapper.WrapperManager");
-                            logger.info("initiating requested shutdown (using service)");
-                            wrapperManager.getMethod("stopAndReturn", int.class).invoke(null, 0);
-                            shutdownWithWrapper = true;
-                        } catch (Throwable e) {
-                            logger.error("failed to initial shutdown on service wrapper", e);
-                        }
-                    }
-                    if (!shutdownWithWrapper) {
-                        logger.info("initiating requested shutdown...");
-                        try {
-                            node.close();
-                        } catch (Exception e) {
-                            logger.warn("Failed to shutdown", e);
-                        } finally {
-                            // make sure we initiate the shutdown hooks, so the Bootstrap#main thread will exit
-                            System.exit(0);
-                        }
-                    }
-                }
-            });
-            t.start();
-
-            channel.sendResponse(TransportResponse.Empty.INSTANCE);
-        }
-    }
-
-    static class NodeShutdownRequest extends TransportRequest {
-
-        boolean exit;
-
-        NodeShutdownRequest() {
-        }
-
-        NodeShutdownRequest(NodesShutdownRequest request) {
-            super(request);
-            this.exit = request.exit();
-        }
-
-        @Override
-        public void readFrom(StreamInput in) throws IOException {
-            super.readFrom(in);
-            exit = in.readBoolean();
-        }
-
-        @Override
-        public void writeTo(StreamOutput out) throws IOException {
-            super.writeTo(out);
-            out.writeBoolean(exit);
-        }
-    }
-}

+ 0 - 26
src/main/java/org/elasticsearch/client/ClusterAdminClient.java

@@ -29,9 +29,6 @@ import org.elasticsearch.action.admin.cluster.node.hotthreads.NodesHotThreadsRes
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoResponse;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoResponse;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequest;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequestBuilder;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownResponse;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsResponse;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsResponse;
@@ -248,29 +245,6 @@ public interface ClusterAdminClient extends ElasticsearchClient<ClusterAdminClie
      */
      */
     NodesHotThreadsRequestBuilder prepareNodesHotThreads(String... nodesIds);
     NodesHotThreadsRequestBuilder prepareNodesHotThreads(String... nodesIds);
 
 
-    /**
-     * Shutdown nodes in the cluster.
-     *
-     * @param request The nodes shutdown request
-     * @return The result future
-     * @see org.elasticsearch.client.Requests#nodesShutdownRequest(String...)
-     */
-    ActionFuture<NodesShutdownResponse> nodesShutdown(NodesShutdownRequest request);
-
-    /**
-     * Shutdown nodes in the cluster.
-     *
-     * @param request  The nodes shutdown request
-     * @param listener A listener to be notified with a result
-     * @see org.elasticsearch.client.Requests#nodesShutdownRequest(String...)
-     */
-    void nodesShutdown(NodesShutdownRequest request, ActionListener<NodesShutdownResponse> listener);
-
-    /**
-     * Shutdown nodes in the cluster.
-     */
-    NodesShutdownRequestBuilder prepareNodesShutdown(String... nodesIds);
-
     /**
     /**
      * Returns list of shards the given search would be executed on.
      * Returns list of shards the given search would be executed on.
      */
      */

+ 0 - 19
src/main/java/org/elasticsearch/client/Requests.java

@@ -21,7 +21,6 @@ package org.elasticsearch.client;
 
 
 import org.elasticsearch.action.admin.cluster.health.ClusterHealthRequest;
 import org.elasticsearch.action.admin.cluster.health.ClusterHealthRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.repositories.delete.DeleteRepositoryRequest;
 import org.elasticsearch.action.admin.cluster.repositories.delete.DeleteRepositoryRequest;
 import org.elasticsearch.action.admin.cluster.repositories.get.GetRepositoriesRequest;
 import org.elasticsearch.action.admin.cluster.repositories.get.GetRepositoriesRequest;
@@ -423,24 +422,6 @@ public class Requests {
         return new ClusterStatsRequest();
         return new ClusterStatsRequest();
     }
     }
 
 
-    /**
-     * Shuts down all nodes in the cluster.
-     */
-    public static NodesShutdownRequest nodesShutdownRequest() {
-        return new NodesShutdownRequest();
-    }
-
-    /**
-     * Shuts down the specified nodes in the cluster.
-     *
-     * @param nodesIds The nodes ids to get the status for
-     * @return The nodes info request
-     * @see org.elasticsearch.client.ClusterAdminClient#nodesShutdown(org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequest)
-     */
-    public static NodesShutdownRequest nodesShutdownRequest(String... nodesIds) {
-        return new NodesShutdownRequest(nodesIds);
-    }
-
     /**
     /**
      * Registers snapshot repository
      * Registers snapshot repository
      *
      *

+ 0 - 19
src/main/java/org/elasticsearch/client/support/AbstractClusterAdminClient.java

@@ -33,10 +33,6 @@ import org.elasticsearch.action.admin.cluster.node.info.NodesInfoAction;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequest;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoResponse;
 import org.elasticsearch.action.admin.cluster.node.info.NodesInfoResponse;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownAction;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequest;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequestBuilder;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownResponse;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsAction;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsAction;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequest;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequestBuilder;
 import org.elasticsearch.action.admin.cluster.node.stats.NodesStatsRequestBuilder;
@@ -233,21 +229,6 @@ public abstract class AbstractClusterAdminClient implements ClusterAdminClient {
         return new NodesHotThreadsRequestBuilder(this).setNodesIds(nodesIds);
         return new NodesHotThreadsRequestBuilder(this).setNodesIds(nodesIds);
     }
     }
 
 
-    @Override
-    public ActionFuture<NodesShutdownResponse> nodesShutdown(final NodesShutdownRequest request) {
-        return execute(NodesShutdownAction.INSTANCE, request);
-    }
-
-    @Override
-    public void nodesShutdown(final NodesShutdownRequest request, final ActionListener<NodesShutdownResponse> listener) {
-        execute(NodesShutdownAction.INSTANCE, request, listener);
-    }
-
-    @Override
-    public NodesShutdownRequestBuilder prepareNodesShutdown(String... nodesIds) {
-        return new NodesShutdownRequestBuilder(this).setNodesIds(nodesIds);
-    }
-
     @Override
     @Override
     public ActionFuture<ClusterSearchShardsResponse> searchShards(final ClusterSearchShardsRequest request) {
     public ActionFuture<ClusterSearchShardsResponse> searchShards(final ClusterSearchShardsRequest request) {
         return execute(ClusterSearchShardsAction.INSTANCE, request);
         return execute(ClusterSearchShardsAction.INSTANCE, request);

+ 0 - 2
src/main/java/org/elasticsearch/rest/action/RestActionModule.java

@@ -28,7 +28,6 @@ import org.elasticsearch.rest.action.admin.cluster.repositories.verify.RestVerif
 import org.elasticsearch.rest.action.admin.cluster.health.RestClusterHealthAction;
 import org.elasticsearch.rest.action.admin.cluster.health.RestClusterHealthAction;
 import org.elasticsearch.rest.action.admin.cluster.node.hotthreads.RestNodesHotThreadsAction;
 import org.elasticsearch.rest.action.admin.cluster.node.hotthreads.RestNodesHotThreadsAction;
 import org.elasticsearch.rest.action.admin.cluster.node.info.RestNodesInfoAction;
 import org.elasticsearch.rest.action.admin.cluster.node.info.RestNodesInfoAction;
-import org.elasticsearch.rest.action.admin.cluster.node.shutdown.RestNodesShutdownAction;
 import org.elasticsearch.rest.action.admin.cluster.node.stats.RestNodesStatsAction;
 import org.elasticsearch.rest.action.admin.cluster.node.stats.RestNodesStatsAction;
 import org.elasticsearch.rest.action.admin.cluster.repositories.delete.RestDeleteRepositoryAction;
 import org.elasticsearch.rest.action.admin.cluster.repositories.delete.RestDeleteRepositoryAction;
 import org.elasticsearch.rest.action.admin.cluster.repositories.get.RestGetRepositoriesAction;
 import org.elasticsearch.rest.action.admin.cluster.repositories.get.RestGetRepositoriesAction;
@@ -132,7 +131,6 @@ public class RestActionModule extends AbstractModule {
         bind(RestNodesInfoAction.class).asEagerSingleton();
         bind(RestNodesInfoAction.class).asEagerSingleton();
         bind(RestNodesStatsAction.class).asEagerSingleton();
         bind(RestNodesStatsAction.class).asEagerSingleton();
         bind(RestNodesHotThreadsAction.class).asEagerSingleton();
         bind(RestNodesHotThreadsAction.class).asEagerSingleton();
-        bind(RestNodesShutdownAction.class).asEagerSingleton();
         bind(RestClusterStatsAction.class).asEagerSingleton();
         bind(RestClusterStatsAction.class).asEagerSingleton();
         bind(RestClusterStateAction.class).asEagerSingleton();
         bind(RestClusterStateAction.class).asEagerSingleton();
         bind(RestClusterHealthAction.class).asEagerSingleton();
         bind(RestClusterHealthAction.class).asEagerSingleton();

+ 0 - 73
src/main/java/org/elasticsearch/rest/action/admin/cluster/node/shutdown/RestNodesShutdownAction.java

@@ -1,73 +0,0 @@
-/*
- * Licensed to Elasticsearch under one or more contributor
- * license agreements. See the NOTICE file distributed with
- * this work for additional information regarding copyright
- * ownership. Elasticsearch 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.elasticsearch.rest.action.admin.cluster.node.shutdown;
-
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownRequest;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownResponse;
-import org.elasticsearch.client.Client;
-import org.elasticsearch.cluster.node.DiscoveryNode;
-import org.elasticsearch.common.Strings;
-import org.elasticsearch.common.inject.Inject;
-import org.elasticsearch.common.settings.Settings;
-import org.elasticsearch.common.xcontent.XContentBuilder;
-import org.elasticsearch.rest.*;
-import org.elasticsearch.rest.action.support.RestBuilderListener;
-
-/**
- *
- */
-public class RestNodesShutdownAction extends BaseRestHandler {
-
-    @Inject
-    public RestNodesShutdownAction(Settings settings, RestController controller, Client client) {
-        super(settings, controller, client);
-
-        controller.registerHandler(RestRequest.Method.POST, "/_shutdown", this);
-        controller.registerHandler(RestRequest.Method.POST, "/_cluster/nodes/_shutdown", this);
-        controller.registerHandler(RestRequest.Method.POST, "/_cluster/nodes/{nodeId}/_shutdown", this);
-    }
-
-    @Override
-    public void handleRequest(final RestRequest request, final RestChannel channel, final Client client) {
-        String[] nodesIds = Strings.splitStringByCommaToArray(request.param("nodeId"));
-        NodesShutdownRequest nodesShutdownRequest = new NodesShutdownRequest(nodesIds);
-        nodesShutdownRequest.listenerThreaded(false);
-        nodesShutdownRequest.delay(request.paramAsTime("delay", nodesShutdownRequest.delay()));
-        nodesShutdownRequest.exit(request.paramAsBoolean("exit", nodesShutdownRequest.exit()));
-        client.admin().cluster().nodesShutdown(nodesShutdownRequest, new RestBuilderListener<NodesShutdownResponse>(channel) {
-            @Override
-            public RestResponse buildResponse(NodesShutdownResponse response, XContentBuilder builder) throws Exception {
-                builder.startObject();
-                builder.field("cluster_name", response.getClusterName().value());
-
-                builder.startObject("nodes");
-                for (DiscoveryNode node : response.getNodes()) {
-                    builder.startObject(node.id(), XContentBuilder.FieldCaseConversion.NONE);
-                    builder.field("name", node.name(), XContentBuilder.FieldCaseConversion.NONE);
-                    builder.endObject();
-                }
-                builder.endObject();
-
-                builder.endObject();
-                return new BytesRestResponse(RestStatus.OK, builder);
-            }
-        });
-    }
-}

+ 1 - 4
src/test/java/org/elasticsearch/client/AbstractClientHeadersTests.java

@@ -23,8 +23,6 @@ import com.google.common.base.Throwables;
 import com.google.common.collect.ImmutableMap;
 import com.google.common.collect.ImmutableMap;
 import org.elasticsearch.action.ActionListener;
 import org.elasticsearch.action.ActionListener;
 import org.elasticsearch.action.GenericAction;
 import org.elasticsearch.action.GenericAction;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownAction;
-import org.elasticsearch.action.admin.cluster.node.shutdown.NodesShutdownResponse;
 import org.elasticsearch.action.admin.cluster.reroute.ClusterRerouteAction;
 import org.elasticsearch.action.admin.cluster.reroute.ClusterRerouteAction;
 import org.elasticsearch.action.admin.cluster.reroute.ClusterRerouteResponse;
 import org.elasticsearch.action.admin.cluster.reroute.ClusterRerouteResponse;
 import org.elasticsearch.action.admin.cluster.snapshots.create.CreateSnapshotAction;
 import org.elasticsearch.action.admin.cluster.snapshots.create.CreateSnapshotAction;
@@ -80,7 +78,7 @@ public abstract class AbstractClientHeadersTests extends ElasticsearchTestCase {
                 IndexAction.INSTANCE,
                 IndexAction.INSTANCE,
 
 
                 // cluster admin actions
                 // cluster admin actions
-                ClusterStatsAction.INSTANCE, CreateSnapshotAction.INSTANCE, NodesShutdownAction.INSTANCE, ClusterRerouteAction.INSTANCE,
+                ClusterStatsAction.INSTANCE, CreateSnapshotAction.INSTANCE, ClusterRerouteAction.INSTANCE,
 
 
                 // indices admin actions
                 // indices admin actions
                 CreateIndexAction.INSTANCE, IndicesStatsAction.INSTANCE, ClearIndicesCacheAction.INSTANCE, FlushAction.INSTANCE
                 CreateIndexAction.INSTANCE, IndicesStatsAction.INSTANCE, ClearIndicesCacheAction.INSTANCE, FlushAction.INSTANCE
@@ -119,7 +117,6 @@ public abstract class AbstractClientHeadersTests extends ElasticsearchTestCase {
         // choosing arbitrary cluster admin actions to test
         // choosing arbitrary cluster admin actions to test
         client.admin().cluster().prepareClusterStats().execute().addListener(new AssertingActionListener<ClusterStatsResponse>(ClusterStatsAction.NAME));
         client.admin().cluster().prepareClusterStats().execute().addListener(new AssertingActionListener<ClusterStatsResponse>(ClusterStatsAction.NAME));
         client.admin().cluster().prepareCreateSnapshot("repo", "bck").execute().addListener(new AssertingActionListener<CreateSnapshotResponse>(CreateSnapshotAction.NAME));
         client.admin().cluster().prepareCreateSnapshot("repo", "bck").execute().addListener(new AssertingActionListener<CreateSnapshotResponse>(CreateSnapshotAction.NAME));
-        client.admin().cluster().prepareNodesShutdown("n1", "n2").execute().addListener(new AssertingActionListener<NodesShutdownResponse>(NodesShutdownAction.NAME));
         client.admin().cluster().prepareReroute().execute().addListener(new AssertingActionListener<ClusterRerouteResponse>(ClusterRerouteAction.NAME));
         client.admin().cluster().prepareReroute().execute().addListener(new AssertingActionListener<ClusterRerouteResponse>(ClusterRerouteAction.NAME));
 
 
         // choosing arbitrary indices admin actions to test
         // choosing arbitrary indices admin actions to test

+ 3 - 1
src/test/java/org/elasticsearch/stresstest/manyindices/ManyNodesManyIndicesRecoveryStressTest.java

@@ -72,7 +72,9 @@ public class ManyNodesManyIndicesRecoveryStressTest {
         }
         }
 
 
         System.out.println("--> Initiating shutdown");
         System.out.println("--> Initiating shutdown");
-        client.admin().cluster().prepareNodesShutdown().setExit(false).execute().actionGet();
+        for (Node node : nodes) {
+            node.close();
+        }
 
 
         System.out.println("--> Waiting for all nodes to be closed...");
         System.out.println("--> Waiting for all nodes to be closed...");
         while (true) {
         while (true) {

+ 0 - 8
src/test/java/org/elasticsearch/test/ExternalNode.java

@@ -212,16 +212,8 @@ final class ExternalNode implements Closeable {
     }
     }
 
 
     synchronized void stop() {
     synchronized void stop() {
-        stop(false);
-    }
-
-    synchronized void stop(boolean forceKill) {
         if (running()) {
         if (running()) {
             try {
             try {
-                if (forceKill == false && nodeInfo != null && random.nextBoolean()) {
-                    // sometimes shut down gracefully
-                    getClient().admin().cluster().prepareNodesShutdown(this.nodeInfo.getNode().id()).setExit(random.nextBoolean()).setDelay("0s").get();
-                }
                 if (this.client != null) {
                 if (this.client != null) {
                     client.close();
                     client.close();
                 }
                 }