Skip to content

Commit

Permalink
Expose ?master_timeout on get-shutdown API (#108886)
Browse files Browse the repository at this point in the history
Relates #107862 Relates #107984
  • Loading branch information
DaveCTurner committed May 23, 2024
1 parent 1073c8b commit 4878509
Show file tree
Hide file tree
Showing 11 changed files with 86 additions and 14 deletions.
5 changes: 5 additions & 0 deletions docs/changelog/108886.yaml
Original file line number Diff line number Diff line change
@@ -0,0 +1,5 @@
pr: 108886
summary: Expose `?master_timeout` on get-shutdown API
area: Infra/Node Lifecycle
type: bug
issues: []
Original file line number Diff line number Diff line change
Expand Up @@ -29,6 +29,11 @@
}
]
},
"params":{}
"params": {
"master_timeout": {
"type": "time",
"description": "Timeout for processing on master node"
}
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -175,6 +175,7 @@ static TransportVersion def(int id) {
public static final TransportVersion RULE_QUERY_RENAME = def(8_666_00_0);
public static final TransportVersion SPARSE_VECTOR_QUERY_ADDED = def(8_667_00_0);
public static final TransportVersion ESQL_ADD_INDEX_MODE_TO_SOURCE = def(8_668_00_0);
public static final TransportVersion GET_SHUTDOWN_STATUS_TIMEOUT = def(8_669_00_0);

/*
* STOP! READ THIS FIRST! No, really,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@
import org.elasticsearch.common.util.concurrent.ThreadContext;
import org.elasticsearch.core.Nullable;
import org.elasticsearch.core.TimeValue;
import org.elasticsearch.rest.RestUtils;
import org.elasticsearch.test.rest.ESRestTestCase;
import org.elasticsearch.xcontent.ObjectPath;
import org.elasticsearch.xcontent.XContentBuilder;
Expand Down Expand Up @@ -110,6 +111,12 @@ public void testPutShutdownIsIdempotentForRemove() throws Exception {
checkPutShutdownIdempotency("REMOVE");
}

private static void maybeAddMasterNodeTimeout(Request request) {
if (randomBoolean()) {
request.addParameter(RestUtils.REST_MASTER_TIMEOUT_PARAM, TEST_REQUEST_TIMEOUT.getStringRep());
}
}

@SuppressWarnings("unchecked")
private void checkPutShutdownIdempotency(String type) throws Exception {
String nodeIdToShutdown = getRandomNodeId();
Expand All @@ -122,12 +129,14 @@ private void checkPutShutdownIdempotency(String type) throws Exception {

// Put a shutdown request
Request putShutdown = new Request("PUT", "_nodes/" + nodeIdToShutdown + "/shutdown");
maybeAddMasterNodeTimeout(putShutdown);
putShutdown.setJsonEntity("{\"type\": \"" + type + "\", \"reason\": \"" + newReason + "\"}");
assertOK(client().performRequest(putShutdown));

// Ensure we can read it back and it has the new reason
{
Request getShutdownStatus = new Request("GET", "_nodes/" + nodeIdToShutdown + "/shutdown");
maybeAddMasterNodeTimeout(getShutdownStatus);
Map<String, Object> statusResponse = responseAsMap(client().performRequest(getShutdownStatus));
List<Map<String, Object>> nodesArray = (List<Map<String, Object>>) statusResponse.get("nodes");
assertThat(nodesArray, hasSize(1));
Expand Down Expand Up @@ -410,6 +419,7 @@ private void putNodeShutdown(

// Put a shutdown request
Request putShutdown = new Request("PUT", "_nodes/" + nodeIdToShutdown + "/shutdown");
maybeAddMasterNodeTimeout(putShutdown);

try (XContentBuilder putBody = JsonXContent.contentBuilder()) {
putBody.startObject();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -83,8 +83,10 @@ public Settings onNodeStopped(String newNodeName) {
logger.info("--> waiting for replacement to complete");

assertBusy(() -> {
final var getShutdownResponse = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request())
.actionGet(10, TimeUnit.SECONDS);
final var getShutdownResponse = client().execute(
GetShutdownStatusAction.INSTANCE,
new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT)
).actionGet(10, TimeUnit.SECONDS);
assertTrue(
Strings.toString(getShutdownResponse, true, true),
getShutdownResponse.getShutdownStatuses()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -73,13 +73,16 @@ public void testShutdownAwarePlugin() throws Exception {

GetShutdownStatusAction.Response getResp = client().execute(
GetShutdownStatusAction.INSTANCE,
new GetShutdownStatusAction.Request(remainNode)
new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, remainNode)
).get();

assertTrue(getResp.getShutdownStatuses().isEmpty());

// The plugin should be in progress
getResp = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(shutdownNode)).get();
getResp = client().execute(
GetShutdownStatusAction.INSTANCE,
new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, shutdownNode)
).get();
assertThat(
getResp.getShutdownStatuses().get(0).pluginsStatus().getStatus(),
equalTo(SingleNodeShutdownMetadata.Status.IN_PROGRESS)
Expand All @@ -89,7 +92,10 @@ public void testShutdownAwarePlugin() throws Exception {
safe.set(true);

// The plugin should be complete
getResp = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(shutdownNode)).get();
getResp = client().execute(
GetShutdownStatusAction.INSTANCE,
new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, shutdownNode)
).get();
assertThat(getResp.getShutdownStatuses().get(0).pluginsStatus().getStatus(), equalTo(SingleNodeShutdownMetadata.Status.COMPLETE));

// The shutdown node should be in the triggered list
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -105,7 +105,8 @@ private void deleteNodeShutdown(String nodeId) {
}

private void assertNoShuttingDownNodes(String nodeId) throws ExecutionException, InterruptedException {
var response = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(nodeId)).get();
var response = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, nodeId))
.get();
assertThat(response.getShutdownStatuses(), empty());
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -473,7 +473,8 @@ private void putNodeShutdown(String nodeId, SingleNodeShutdownMetadata.Type type
}

private void assertNodeShutdownStatus(String nodeId, SingleNodeShutdownMetadata.Status status) throws Exception {
var response = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(nodeId)).get();
var response = client().execute(GetShutdownStatusAction.INSTANCE, new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, nodeId))
.get();
assertThat(response.getShutdownStatuses().get(0).migrationStatus().getStatus(), equalTo(status));
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -7,6 +7,7 @@

package org.elasticsearch.xpack.shutdown;

import org.elasticsearch.TransportVersions;
import org.elasticsearch.action.ActionRequestValidationException;
import org.elasticsearch.action.ActionResponse;
import org.elasticsearch.action.ActionType;
Expand All @@ -17,6 +18,8 @@
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.xcontent.ChunkedToXContentHelper;
import org.elasticsearch.common.xcontent.ChunkedToXContentObject;
import org.elasticsearch.core.TimeValue;
import org.elasticsearch.core.UpdateForV9;
import org.elasticsearch.tasks.CancellableTask;
import org.elasticsearch.tasks.Task;
import org.elasticsearch.tasks.TaskId;
Expand All @@ -42,17 +45,50 @@ public static class Request extends MasterNodeRequest<Request> {

private final String[] nodeIds;

public Request(String... nodeIds) {
super(TRAPPY_IMPLICIT_DEFAULT_MASTER_NODE_TIMEOUT);
@Deprecated(forRemoval = true) // temporary compatibility shim
public Request() {
super(MasterNodeRequest.TRAPPY_IMPLICIT_DEFAULT_MASTER_NODE_TIMEOUT);
nodeIds = Strings.EMPTY_ARRAY;
}

@Deprecated(forRemoval = true) // temporary compatibility shim
public Request(String nodeId) {
super(MasterNodeRequest.TRAPPY_IMPLICIT_DEFAULT_MASTER_NODE_TIMEOUT);
nodeIds = new String[] { nodeId };
}

public Request(TimeValue masterNodeTimeout, String... nodeIds) {
super(masterNodeTimeout);
this.nodeIds = nodeIds;
}

@UpdateForV9 // only needed for bwc, inline in v9
public static Request readFrom(StreamInput in) throws IOException {
return new Request(in.readStringArray());
if (in.getTransportVersion().onOrAfter(TransportVersions.GET_SHUTDOWN_STATUS_TIMEOUT)) {
return new Request(in);
} else {
return new Request(TimeValue.THIRTY_SECONDS, in);
}
}

private Request(StreamInput in) throws IOException {
super(in);
assert in.getTransportVersion().onOrAfter(TransportVersions.GET_SHUTDOWN_STATUS_TIMEOUT);
nodeIds = in.readStringArray();
}

@UpdateForV9 // only needed for bwc, remove in v9
private Request(TimeValue masterNodeTimeout, StreamInput in) throws IOException {
super(masterNodeTimeout);
assert in.getTransportVersion().before(TransportVersions.GET_SHUTDOWN_STATUS_TIMEOUT);
nodeIds = in.readStringArray();
}

@Override
public void writeTo(StreamOutput out) throws IOException {
if (out.getTransportVersion().onOrAfter(TransportVersions.GET_SHUTDOWN_STATUS_TIMEOUT)) {
super.writeTo(out);
}
out.writeStringArray(this.nodeIds);
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -11,6 +11,7 @@
import org.elasticsearch.common.Strings;
import org.elasticsearch.rest.BaseRestHandler;
import org.elasticsearch.rest.RestRequest;
import org.elasticsearch.rest.RestUtils;
import org.elasticsearch.rest.Scope;
import org.elasticsearch.rest.ServerlessScope;
import org.elasticsearch.rest.action.RestCancellableNodeClient;
Expand All @@ -36,10 +37,13 @@ public List<Route> routes() {

@Override
protected RestChannelConsumer prepareRequest(RestRequest request, NodeClient client) {
String[] nodeIds = Strings.commaDelimitedListToStringArray(request.param("nodeId"));
final var actionRequest = new GetShutdownStatusAction.Request(
RestUtils.getMasterNodeTimeout(request),
Strings.commaDelimitedListToStringArray(request.param("nodeId"))
);
return channel -> new RestCancellableNodeClient(client, request.getHttpChannel()).execute(
GetShutdownStatusAction.INSTANCE,
new GetShutdownStatusAction.Request(nodeIds),
actionRequest,
new RestRefCountedChunkedToXContentListener<>(channel)
);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@ protected Writeable.Reader<GetShutdownStatusAction.Request> instanceReader() {
@Override
protected GetShutdownStatusAction.Request createTestInstance() {
return new GetShutdownStatusAction.Request(
TEST_REQUEST_TIMEOUT,
randomList(0, 20, () -> randomAlphaOfLengthBetween(15, 25)).toArray(Strings.EMPTY_ARRAY)
);
}
Expand All @@ -35,6 +36,6 @@ protected GetShutdownStatusAction.Request mutateInstance(GetShutdownStatusAction
String[] newNodeIds = randomList(1, 20, () -> randomValueOtherThanMany(oldIds::contains, () -> randomAlphaOfLengthBetween(15, 25)))
.toArray(Strings.EMPTY_ARRAY);

return new GetShutdownStatusAction.Request(newNodeIds);
return new GetShutdownStatusAction.Request(TEST_REQUEST_TIMEOUT, newNodeIds);
}
}

0 comments on commit 4878509

Please sign in to comment.