mirror of https://github.com/apache/lucene.git
SOLR-8560: Add RequestStatusState enum
git-svn-id: https://svn.apache.org/repos/asf/lucene/dev/trunk@1726144 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
7213e7cfc0
commit
2311c7fee1
|
@ -330,6 +330,9 @@ New Features
|
|||
|
||||
* SOLR-4619: Improve PreAnalyzedField query analysis. (Andrzej Bialecki, Steve Rowe)
|
||||
|
||||
* SOLR-8560: Added RequestStatusState enum which can be used when comparing states of
|
||||
asynchronous requests. (Shai Erera)
|
||||
|
||||
Bug Fixes
|
||||
----------------------
|
||||
|
||||
|
|
|
@ -1,7 +1,3 @@
|
|||
package org.apache.solr.handler.admin;
|
||||
|
||||
import java.lang.invoke.MethodHandles;
|
||||
|
||||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
|
@ -18,67 +14,9 @@ import java.lang.invoke.MethodHandles;
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.solr.handler.admin;
|
||||
|
||||
import java.nio.charset.StandardCharsets;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Arrays;
|
||||
import java.util.Iterator;
|
||||
import java.util.LinkedHashMap;
|
||||
import java.util.List;
|
||||
import java.util.Locale;
|
||||
import java.util.Map;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.TimeUnit;
|
||||
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.google.common.collect.ImmutableSet;
|
||||
import org.apache.commons.lang.StringUtils;
|
||||
import org.apache.solr.client.solrj.SolrResponse;
|
||||
import org.apache.solr.client.solrj.impl.HttpSolrClient;
|
||||
import org.apache.solr.client.solrj.request.CoreAdminRequest;
|
||||
import org.apache.solr.client.solrj.request.CoreAdminRequest.RequestSyncShard;
|
||||
import org.apache.solr.cloud.DistributedMap;
|
||||
import org.apache.solr.cloud.OverseerCollectionMessageHandler;
|
||||
import org.apache.solr.cloud.OverseerTaskQueue;
|
||||
import org.apache.solr.cloud.OverseerTaskQueue.QueueEvent;
|
||||
import org.apache.solr.cloud.Overseer;
|
||||
import org.apache.solr.cloud.OverseerSolrResponse;
|
||||
import org.apache.solr.cloud.overseer.SliceMutator;
|
||||
import org.apache.solr.cloud.rule.ReplicaAssigner;
|
||||
import org.apache.solr.cloud.rule.Rule;
|
||||
import org.apache.solr.common.SolrException;
|
||||
import org.apache.solr.common.SolrException.ErrorCode;
|
||||
import org.apache.solr.common.cloud.ClusterState;
|
||||
import org.apache.solr.common.cloud.DocCollection;
|
||||
import org.apache.solr.common.cloud.ImplicitDocRouter;
|
||||
import org.apache.solr.common.cloud.Replica;
|
||||
import org.apache.solr.common.cloud.Slice;
|
||||
import org.apache.solr.common.cloud.SolrZkClient;
|
||||
import org.apache.solr.common.cloud.ZkCmdExecutor;
|
||||
import org.apache.solr.common.cloud.ZkCoreNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.cloud.Replica.State;
|
||||
import org.apache.solr.common.params.CollectionParams.CollectionAction;
|
||||
import org.apache.solr.common.params.CoreAdminParams.CoreAdminAction;
|
||||
import org.apache.solr.common.params.CommonParams;
|
||||
import org.apache.solr.common.params.CoreAdminParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.params.SolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.handler.BlobHandler;
|
||||
import org.apache.solr.handler.RequestHandlerBase;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.request.SolrQueryRequest;
|
||||
import org.apache.solr.response.SolrQueryResponse;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
import static org.apache.solr.client.solrj.response.RequestStatusState.*;
|
||||
import static org.apache.solr.cloud.Overseer.QUEUE_OPERATION;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.COLL_CONF;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.COLL_PROP_PREFIX;
|
||||
|
@ -102,7 +40,32 @@ import static org.apache.solr.common.cloud.ZkStateReader.PROPERTY_VALUE_PROP;
|
|||
import static org.apache.solr.common.cloud.ZkStateReader.REPLICATION_FACTOR;
|
||||
import static org.apache.solr.common.cloud.ZkStateReader.REPLICA_PROP;
|
||||
import static org.apache.solr.common.cloud.ZkStateReader.SHARD_ID_PROP;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.*;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.ADDREPLICA;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.ADDREPLICAPROP;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.ADDROLE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.BALANCESHARDUNIQUE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.CLUSTERPROP;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.CLUSTERSTATUS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.CREATE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.CREATEALIAS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.CREATESHARD;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.DELETE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.DELETEALIAS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.DELETEREPLICA;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.DELETEREPLICAPROP;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.DELETESHARD;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.FORCELEADER;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.LIST;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.MIGRATE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.MIGRATESTATEFORMAT;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.MODIFYCOLLECTION;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.OVERSEERSTATUS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.REBALANCELEADERS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.RELOAD;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.REMOVEROLE;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.REQUESTSTATUS;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.SPLITSHARD;
|
||||
import static org.apache.solr.common.params.CollectionParams.CollectionAction.SYNCSHARD;
|
||||
import static org.apache.solr.common.params.CommonAdminParams.ASYNC;
|
||||
import static org.apache.solr.common.params.CommonParams.NAME;
|
||||
import static org.apache.solr.common.params.CommonParams.VALUE_LONG;
|
||||
|
@ -111,6 +74,71 @@ import static org.apache.solr.common.params.CoreAdminParams.INSTANCE_DIR;
|
|||
import static org.apache.solr.common.params.ShardParams._ROUTE_;
|
||||
import static org.apache.solr.common.util.StrUtils.formatString;
|
||||
|
||||
import java.lang.invoke.MethodHandles;
|
||||
|
||||
import java.nio.charset.StandardCharsets;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Arrays;
|
||||
import java.util.Iterator;
|
||||
import java.util.LinkedHashMap;
|
||||
import java.util.List;
|
||||
import java.util.Locale;
|
||||
import java.util.Map;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.TimeUnit;
|
||||
|
||||
import org.apache.commons.lang.StringUtils;
|
||||
import org.apache.solr.client.solrj.SolrResponse;
|
||||
import org.apache.solr.client.solrj.impl.HttpSolrClient;
|
||||
import org.apache.solr.client.solrj.request.CoreAdminRequest;
|
||||
import org.apache.solr.client.solrj.request.CoreAdminRequest.RequestSyncShard;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.cloud.DistributedMap;
|
||||
import org.apache.solr.cloud.Overseer;
|
||||
import org.apache.solr.cloud.OverseerCollectionMessageHandler;
|
||||
import org.apache.solr.cloud.OverseerSolrResponse;
|
||||
import org.apache.solr.cloud.OverseerTaskQueue;
|
||||
import org.apache.solr.cloud.OverseerTaskQueue.QueueEvent;
|
||||
import org.apache.solr.cloud.ZkController;
|
||||
import org.apache.solr.cloud.overseer.SliceMutator;
|
||||
import org.apache.solr.cloud.rule.ReplicaAssigner;
|
||||
import org.apache.solr.cloud.rule.Rule;
|
||||
import org.apache.solr.common.SolrException;
|
||||
import org.apache.solr.common.SolrException.ErrorCode;
|
||||
import org.apache.solr.common.cloud.ClusterState;
|
||||
import org.apache.solr.common.cloud.DocCollection;
|
||||
import org.apache.solr.common.cloud.ImplicitDocRouter;
|
||||
import org.apache.solr.common.cloud.Replica;
|
||||
import org.apache.solr.common.cloud.Replica.State;
|
||||
import org.apache.solr.common.cloud.Slice;
|
||||
import org.apache.solr.common.cloud.SolrZkClient;
|
||||
import org.apache.solr.common.cloud.ZkCmdExecutor;
|
||||
import org.apache.solr.common.cloud.ZkCoreNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.params.CollectionParams.CollectionAction;
|
||||
import org.apache.solr.common.params.CommonParams;
|
||||
import org.apache.solr.common.params.CoreAdminParams;
|
||||
import org.apache.solr.common.params.CoreAdminParams.CoreAdminAction;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.params.SolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.handler.BlobHandler;
|
||||
import org.apache.solr.handler.RequestHandlerBase;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.request.SolrQueryRequest;
|
||||
import org.apache.solr.response.SolrQueryResponse;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.google.common.collect.ImmutableSet;
|
||||
|
||||
public class CollectionsHandler extends RequestHandlerBase {
|
||||
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
|
||||
protected final CoreContainer coreContainer;
|
||||
|
@ -514,63 +542,50 @@ public class CollectionsHandler extends RequestHandlerBase {
|
|||
}
|
||||
},
|
||||
REQUESTSTATUS_OP(REQUESTSTATUS) {
|
||||
@SuppressWarnings("unchecked")
|
||||
@Override
|
||||
Map<String, Object> call(SolrQueryRequest req, SolrQueryResponse rsp, CollectionsHandler h) throws Exception {
|
||||
CoreContainer coreContainer = h.coreContainer;
|
||||
req.getParams().required().check(REQUESTID);
|
||||
|
||||
String requestId = req.getParams().get(REQUESTID);
|
||||
|
||||
|
||||
final CoreContainer coreContainer = h.coreContainer;
|
||||
final String requestId = req.getParams().get(REQUESTID);
|
||||
final ZkController zkController = coreContainer.getZkController();
|
||||
|
||||
if (requestId.equals("-1")) {
|
||||
// Special taskId (-1), clears up the request state maps.
|
||||
if (requestId.equals("-1")) {
|
||||
coreContainer.getZkController().getOverseerCompletedMap().clear();
|
||||
coreContainer.getZkController().getOverseerFailureMap().clear();
|
||||
return null;
|
||||
}
|
||||
} else {
|
||||
NamedList<Object> results = new NamedList<>();
|
||||
if (coreContainer.getZkController().getOverseerCompletedMap().contains(requestId)) {
|
||||
DistributedMap.MapEvent mapEvent = coreContainer.getZkController().getOverseerCompletedMap().get(requestId);
|
||||
SimpleOrderedMap success = new SimpleOrderedMap();
|
||||
if(mapEvent != null) {
|
||||
rsp.getValues().addAll(SolrResponse.deserialize(mapEvent.getBytes()).getResponse());
|
||||
}
|
||||
success.add("state", "completed");
|
||||
success.add("msg", "found " + requestId + " in completed tasks");
|
||||
results.add("status", success);
|
||||
} else if (coreContainer.getZkController().getOverseerFailureMap().contains(requestId)) {
|
||||
SimpleOrderedMap success = new SimpleOrderedMap();
|
||||
DistributedMap.MapEvent mapEvent = coreContainer.getZkController().getOverseerFailureMap().get(requestId);
|
||||
if(mapEvent != null) {
|
||||
rsp.getValues().addAll(SolrResponse.deserialize(mapEvent.getBytes()).getResponse());
|
||||
}
|
||||
success.add("state", "failed");
|
||||
success.add("msg", "found " + requestId + " in failed tasks");
|
||||
results.add("status", success);
|
||||
} else if (coreContainer.getZkController().getOverseerRunningMap().contains(requestId)) {
|
||||
SimpleOrderedMap success = new SimpleOrderedMap();
|
||||
success.add("state", "running");
|
||||
success.add("msg", "found " + requestId + " in running tasks");
|
||||
results.add("status", success);
|
||||
} else if (h.overseerCollectionQueueContains(requestId)) {
|
||||
SimpleOrderedMap success = new SimpleOrderedMap();
|
||||
success.add("state", "submitted");
|
||||
success.add("msg", "found " + requestId + " in submitted tasks");
|
||||
results.add("status", success);
|
||||
} else {
|
||||
SimpleOrderedMap failure = new SimpleOrderedMap();
|
||||
failure.add("state", "notfound");
|
||||
failure.add("msg", "Did not find taskid [" + requestId + "] in any tasks queue");
|
||||
results.add("status", failure);
|
||||
}
|
||||
SolrResponse response = new OverseerSolrResponse(results);
|
||||
|
||||
rsp.getValues().addAll(response.getResponse());
|
||||
zkController.getOverseerCompletedMap().clear();
|
||||
zkController.getOverseerFailureMap().clear();
|
||||
return null;
|
||||
}
|
||||
|
||||
final NamedList<Object> results = new NamedList<>();
|
||||
if (zkController.getOverseerCompletedMap().contains(requestId)) {
|
||||
final DistributedMap.MapEvent mapEvent = zkController.getOverseerCompletedMap().get(requestId);
|
||||
rsp.getValues().addAll(SolrResponse.deserialize(mapEvent.getBytes()).getResponse());
|
||||
addStatusToResponse(results, COMPLETED, "found [" + requestId + "] in completed tasks");
|
||||
} else if (zkController.getOverseerFailureMap().contains(requestId)) {
|
||||
final DistributedMap.MapEvent mapEvent = zkController.getOverseerFailureMap().get(requestId);
|
||||
rsp.getValues().addAll(SolrResponse.deserialize(mapEvent.getBytes()).getResponse());
|
||||
addStatusToResponse(results, FAILED, "found [" + requestId + "] in failed tasks");
|
||||
} else if (zkController.getOverseerRunningMap().contains(requestId)) {
|
||||
addStatusToResponse(results, RUNNING, "found [" + requestId + "] in running tasks");
|
||||
} else if (h.overseerCollectionQueueContains(requestId)) {
|
||||
addStatusToResponse(results, SUBMITTED, "found [" + requestId + "] in submitted tasks");
|
||||
} else {
|
||||
addStatusToResponse(results, NOT_FOUND, "Did not find [" + requestId + "] in any tasks queue");
|
||||
}
|
||||
|
||||
final SolrResponse response = new OverseerSolrResponse(results);
|
||||
rsp.getValues().addAll(response.getResponse());
|
||||
return null;
|
||||
}
|
||||
|
||||
private void addStatusToResponse(NamedList<Object> results, RequestStatusState state, String msg) {
|
||||
SimpleOrderedMap<String> status = new SimpleOrderedMap<>();
|
||||
status.add("state", state.getKey());
|
||||
status.add("msg", msg);
|
||||
results.add("status", status);
|
||||
}
|
||||
},
|
||||
ADDREPLICA_OP(ADDREPLICA) {
|
||||
@Override
|
||||
|
@ -867,4 +882,4 @@ public class CollectionsHandler extends RequestHandlerBase {
|
|||
MAX_SHARDS_PER_NODE,
|
||||
AUTO_ADD_REPLICAS);
|
||||
|
||||
}
|
||||
}
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.solr.cloud;
|
|||
|
||||
import org.apache.solr.client.solrj.request.CollectionAdminRequest;
|
||||
import org.apache.solr.client.solrj.response.CollectionAdminResponse;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.junit.Test;
|
||||
|
||||
|
@ -34,12 +35,12 @@ public class AsyncCallRequestStatusResponseTest extends AbstractFullDistribZkTes
|
|||
create.setConfigName("conf1");
|
||||
create.process(cloudClient);
|
||||
waitForCollection(cloudClient.getZkStateReader(), "asynccall", 2);
|
||||
String state = getRequestStateAfterCompletion("1000", 30, cloudClient);
|
||||
assertTrue(state.equals("completed"));
|
||||
final RequestStatusState state = getRequestStateAfterCompletion("1000", 30, cloudClient);
|
||||
assertSame(RequestStatusState.COMPLETED, state);
|
||||
CollectionAdminRequest.RequestStatus requestStatus = new CollectionAdminRequest.RequestStatus();
|
||||
requestStatus.setRequestId("1000");
|
||||
CollectionAdminResponse rsp = requestStatus.process(cloudClient);
|
||||
NamedList r = rsp.getResponse();
|
||||
NamedList<?> r = rsp.getResponse();
|
||||
// Check that there's more response than the hardcoded status and states
|
||||
assertEquals("Assertion Failure" + r.toString(), 5, r.size());
|
||||
}
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.solr.client.solrj.SolrRequest;
|
|||
import org.apache.solr.client.solrj.SolrServerException;
|
||||
import org.apache.solr.client.solrj.impl.HttpSolrClient;
|
||||
import org.apache.solr.client.solrj.request.QueryRequest;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.params.CollectionParams;
|
||||
import org.apache.solr.common.params.CommonAdminParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
|
@ -53,11 +54,11 @@ public class AsyncMigrateRouteKeyTest extends MigrateRouteKeyTest {
|
|||
params.set(OverseerCollectionMessageHandler.REQUESTID, asyncId);
|
||||
// This task takes long enough to run. Also check for the current state of the task to be running.
|
||||
message = sendStatusRequestWithRetry(params, 5);
|
||||
assertEquals("found " + asyncId + " in running tasks", message);
|
||||
assertEquals("found [" + asyncId + "] in running tasks", message);
|
||||
// Now wait until the task actually completes successfully/fails.
|
||||
message = sendStatusRequestWithRetry(params, MAX_WAIT_SECONDS);
|
||||
assertEquals("Task " + asyncId + " not found in completed tasks.",
|
||||
"found " + asyncId + " in completed tasks", message);
|
||||
assertEquals("Task " + asyncId + " not found in completed tasks.",
|
||||
"found [" + asyncId + "] in completed tasks", message);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -83,17 +84,18 @@ public class AsyncMigrateRouteKeyTest extends MigrateRouteKeyTest {
|
|||
private String sendStatusRequestWithRetry(ModifiableSolrParams params, int maxCounter)
|
||||
throws SolrServerException, IOException {
|
||||
NamedList status = null;
|
||||
String state = null;
|
||||
RequestStatusState state = null;
|
||||
String message = null;
|
||||
NamedList r;
|
||||
while (maxCounter-- > 0) {
|
||||
r = sendRequest(params);
|
||||
status = (NamedList) r.get("status");
|
||||
state = (String) status.get("state");
|
||||
state = RequestStatusState.fromKey((String) status.get("state"));
|
||||
message = (String) status.get("msg");
|
||||
|
||||
if (state.equals("completed") || state.equals("failed"))
|
||||
if (state == RequestStatusState.COMPLETED || state == RequestStatusState.FAILED) {
|
||||
return (String) status.get("msg");
|
||||
}
|
||||
try {
|
||||
Thread.sleep(1000);
|
||||
} catch (InterruptedException e) {
|
||||
|
@ -106,11 +108,10 @@ public class AsyncMigrateRouteKeyTest extends MigrateRouteKeyTest {
|
|||
}
|
||||
|
||||
protected NamedList sendRequest(ModifiableSolrParams params) throws SolrServerException, IOException {
|
||||
SolrRequest request = new QueryRequest(params);
|
||||
final SolrRequest request = new QueryRequest(params);
|
||||
request.setPath("/admin/collections");
|
||||
|
||||
String baseUrl = ((HttpSolrClient) shardToJetty.get(SHARD1).get(0).client.solrClient)
|
||||
.getBaseURL();
|
||||
String baseUrl = ((HttpSolrClient) shardToJetty.get(SHARD1).get(0).client.solrClient).getBaseURL();
|
||||
baseUrl = baseUrl.substring(0, baseUrl.length() - "collection1".length());
|
||||
|
||||
try (HttpSolrClient baseServer = new HttpSolrClient(baseUrl)) {
|
||||
|
|
|
@ -28,11 +28,11 @@ import org.apache.solr.client.solrj.impl.HttpSolrClient;
|
|||
import org.apache.solr.client.solrj.request.CollectionAdminRequest;
|
||||
import org.apache.solr.client.solrj.request.CollectionAdminRequest.Create;
|
||||
import org.apache.solr.client.solrj.request.CollectionAdminRequest.SplitShard;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.client.solrj.response.CollectionAdminResponse;
|
||||
import org.apache.solr.common.SolrException;
|
||||
import org.apache.solr.common.SolrInputDocument;
|
||||
import org.apache.solr.common.cloud.Slice;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.junit.Test;
|
||||
|
||||
/**
|
||||
|
@ -57,10 +57,9 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
.setAsyncId("1001");
|
||||
createCollectionRequest.process(client);
|
||||
|
||||
String state = getRequestStateAfterCompletion("1001", MAX_TIMEOUT_SECONDS, client);
|
||||
|
||||
assertEquals("CreateCollection task did not complete!", "completed", state);
|
||||
RequestStatusState state = getRequestStateAfterCompletion("1001", MAX_TIMEOUT_SECONDS, client);
|
||||
|
||||
assertSame("CreateCollection task did not complete!", RequestStatusState.COMPLETED, state);
|
||||
|
||||
createCollectionRequest = new Create()
|
||||
.setCollectionName("testasynccollectioncreation")
|
||||
|
@ -71,7 +70,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
|
||||
state = getRequestStateAfterCompletion("1002", MAX_TIMEOUT_SECONDS, client);
|
||||
|
||||
assertEquals("Recreating a collection with the same name didn't fail, should have.", "failed", state);
|
||||
assertSame("Recreating a collection with the same should have failed.", RequestStatusState.FAILED, state);
|
||||
|
||||
CollectionAdminRequest.AddReplica addReplica = new CollectionAdminRequest.AddReplica()
|
||||
.setCollectionName("testasynccollectioncreation")
|
||||
|
@ -79,8 +78,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
.setAsyncId("1003");
|
||||
client.request(addReplica);
|
||||
state = getRequestStateAfterCompletion("1003", MAX_TIMEOUT_SECONDS, client);
|
||||
assertEquals("Add replica did not complete", "completed", state);
|
||||
|
||||
assertSame("Add replica did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
SplitShard splitShardRequest = new SplitShard()
|
||||
.setCollectionName("testasynccollectioncreation")
|
||||
|
@ -90,7 +88,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
|
||||
state = getRequestStateAfterCompletion("1004", MAX_TIMEOUT_SECONDS * 2, client);
|
||||
|
||||
assertEquals("Shard split did not complete. Last recorded state: " + state, "completed", state);
|
||||
assertEquals("Shard split did not complete. Last recorded state: " + state, RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -107,8 +105,8 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
.setAsyncId("42");
|
||||
CollectionAdminResponse response = createCollectionRequest.process(cloudClient);
|
||||
assertEquals("42", response.getResponse().get("requestid"));
|
||||
String state = getRequestStateAfterCompletion("42", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("CreateCollection task did not complete!", "completed", state);
|
||||
RequestStatusState state = getRequestStateAfterCompletion("42", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertSame("CreateCollection task did not complete!", RequestStatusState.COMPLETED, state);
|
||||
|
||||
//Add a few documents to shard1
|
||||
int numDocs = TestUtil.nextInt(random(), 10, 100);
|
||||
|
@ -131,7 +129,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = reloadCollection.process(cloudClient);
|
||||
assertEquals("43", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("43", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("ReloadCollection did not complete", "completed", state);
|
||||
assertSame("ReloadCollection did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
CollectionAdminRequest.CreateShard createShard = new CollectionAdminRequest.CreateShard()
|
||||
.setCollectionName(collection)
|
||||
|
@ -140,7 +138,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = createShard.process(cloudClient);
|
||||
assertEquals("44", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("44", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("CreateShard did not complete", "completed", state);
|
||||
assertSame("CreateShard did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
//Add a doc to shard2 to make sure shard2 was created properly
|
||||
SolrInputDocument doc = new SolrInputDocument();
|
||||
|
@ -159,7 +157,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = deleteShard.process(cloudClient);
|
||||
assertEquals("45", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("45", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("DeleteShard did not complete", "completed", state);
|
||||
assertSame("DeleteShard did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
CollectionAdminRequest.AddReplica addReplica = new CollectionAdminRequest.AddReplica()
|
||||
.setCollectionName(collection)
|
||||
|
@ -168,7 +166,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = addReplica.process(cloudClient);
|
||||
assertEquals("46", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("46", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("AddReplica did not complete", "completed", state);
|
||||
assertSame("AddReplica did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
//cloudClient watch might take a couple of seconds to reflect it
|
||||
Slice shard1 = cloudClient.getZkStateReader().getClusterState().getSlice(collection, "shard1");
|
||||
|
@ -187,7 +185,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = createAlias.process(cloudClient);
|
||||
assertEquals("47", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("47", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("CreateAlias did not complete", "completed", state);
|
||||
assertSame("CreateAlias did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
query = new SolrQuery("*:*");
|
||||
query.set("shards", "shard1");
|
||||
|
@ -199,7 +197,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = deleteAlias.process(cloudClient);
|
||||
assertEquals("48", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("48", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("DeleteAlias did not complete", "completed", state);
|
||||
assertSame("DeleteAlias did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
try {
|
||||
cloudClient.query("myalias", query);
|
||||
|
@ -217,7 +215,7 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = deleteReplica.process(cloudClient);
|
||||
assertEquals("47", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("47", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("DeleteReplica did not complete", "completed", state);
|
||||
assertSame("DeleteReplica did not complete", RequestStatusState.COMPLETED, state);
|
||||
|
||||
CollectionAdminRequest.Delete deleteCollection = new CollectionAdminRequest.Delete()
|
||||
.setCollectionName(collection)
|
||||
|
@ -225,6 +223,6 @@ public class CollectionsAPIAsyncDistributedZkTest extends AbstractFullDistribZkT
|
|||
response = deleteCollection.process(cloudClient);
|
||||
assertEquals("48", response.getResponse().get("requestid"));
|
||||
state = getRequestStateAfterCompletion("48", MAX_TIMEOUT_SECONDS, cloudClient);
|
||||
assertEquals("DeleteCollection did not complete", "completed", state);
|
||||
assertSame("DeleteCollection did not complete", RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
}
|
||||
|
|
|
@ -29,6 +29,7 @@ import org.apache.solr.client.solrj.request.CollectionAdminRequest.Create;
|
|||
import org.apache.solr.client.solrj.request.CollectionAdminRequest.SplitShard;
|
||||
import org.apache.solr.client.solrj.request.QueryRequest;
|
||||
import org.apache.solr.client.solrj.response.CollectionAdminResponse;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.params.CollectionParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
|
@ -76,13 +77,15 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
while(true) {
|
||||
int numRunningTasks = 0;
|
||||
for (int i = 1; i <= NUM_COLLECTIONS; i++)
|
||||
if (getRequestState(i + "", client).equals("running"))
|
||||
if (getRequestState(i + "", client) == RequestStatusState.RUNNING) {
|
||||
numRunningTasks++;
|
||||
if(numRunningTasks > 1) {
|
||||
}
|
||||
if (numRunningTasks > 1) {
|
||||
pass = true;
|
||||
break;
|
||||
} else if(counter++ > 100)
|
||||
} else if (counter++ > 100) {
|
||||
break;
|
||||
}
|
||||
try {
|
||||
Thread.sleep(100);
|
||||
} catch (InterruptedException e) {
|
||||
|
@ -90,9 +93,9 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
}
|
||||
}
|
||||
assertTrue("More than one tasks were supposed to be running in parallel but they weren't.", pass);
|
||||
for(int i=1;i<=NUM_COLLECTIONS;i++) {
|
||||
String state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertTrue("Task " + i + " did not complete, final state: " + state,state.equals("completed"));
|
||||
for (int i = 1; i <= NUM_COLLECTIONS; i++) {
|
||||
final RequestStatusState state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertSame("Task " + i + " did not complete, final state: " + state, RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -122,13 +125,14 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
while(true) {
|
||||
int runningTasks = 0;
|
||||
int completedTasks = 0;
|
||||
for (int i=1001;i<=1002;i++) {
|
||||
String state = getRequestState(i, client);
|
||||
if (state.equals("running"))
|
||||
for (int i = 1001; i <= 1002; i++) {
|
||||
final RequestStatusState state = getRequestState(i, client);
|
||||
if (state == RequestStatusState.RUNNING) {
|
||||
runningTasks++;
|
||||
if (state.equals("completed"))
|
||||
} else if (state == RequestStatusState.COMPLETED) {
|
||||
completedTasks++;
|
||||
assertTrue("We have a failed SPLITSHARD task", !state.equals("failed"));
|
||||
}
|
||||
assertNotSame("We have a failed SPLITSHARD task", RequestStatusState.FAILED, state);
|
||||
}
|
||||
// TODO: REQUESTSTATUS might come back with more than 1 running tasks over multiple calls.
|
||||
// The only way to fix this is to support checking of multiple requestids in a single REQUESTSTATUS task.
|
||||
|
@ -145,9 +149,9 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
return;
|
||||
}
|
||||
}
|
||||
for (int i=1001;i<=1002;i++) {
|
||||
String state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertTrue("Task " + i + " did not complete, final state: " + state,state.equals("completed"));
|
||||
for (int i = 1001; i <= 1002; i++) {
|
||||
final RequestStatusState state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertSame("Task " + i + " did not complete, final state: " + state, RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -184,9 +188,9 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
assertEquals("Duplicate request was supposed to exist but wasn't found. De-duplication of submitted task failed.",
|
||||
"Task with the same requestid already exists.", r.get("error"));
|
||||
|
||||
for (int i=3001;i<=3002;i++) {
|
||||
String state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertTrue("Task " + i + " did not complete, final state: " + state,state.equals("completed"));
|
||||
for (int i = 3001; i <= 3002; i++) {
|
||||
final RequestStatusState state = getRequestStateAfterCompletion(i + "", REQUEST_STATUS_TIMEOUT, client);
|
||||
assertSame("Task " + i + " did not complete, final state: " + state, RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -216,12 +220,13 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
.setAsyncId("2000");
|
||||
splitShardRequest.process(client);
|
||||
|
||||
String state = getRequestState("2000", client);
|
||||
while (state.equals("submitted")) {
|
||||
RequestStatusState state = getRequestState("2000", client);
|
||||
while (state == RequestStatusState.SUBMITTED) {
|
||||
state = getRequestState("2000", client);
|
||||
Thread.sleep(10);
|
||||
}
|
||||
assertTrue("SplitShard task [2000] was supposed to be in [running] but isn't. It is [" + state + "]", state.equals("running"));
|
||||
assertSame("SplitShard task [2000] was supposed to be in [running] but isn't. It is [" + state + "]",
|
||||
RequestStatusState.RUNNING, state);
|
||||
|
||||
// CLUSTERSTATE is always mutually exclusive, it should return with a response before the split completes
|
||||
|
||||
|
@ -235,8 +240,8 @@ public class MultiThreadedOCPTest extends AbstractFullDistribZkTestBase {
|
|||
|
||||
state = getRequestState("2000", client);
|
||||
|
||||
assertTrue("After invoking OVERSEERSTATUS, SplitShard task [2000] was still supposed to be in [running] but isn't." +
|
||||
"It is [" + state + "]", state.equals("running"));
|
||||
assertSame("After invoking OVERSEERSTATUS, SplitShard task [2000] was still supposed to be in [running] but "
|
||||
+ "isn't. It is [" + state + "]", RequestStatusState.RUNNING, state);
|
||||
|
||||
} finally {
|
||||
try {
|
||||
|
|
|
@ -40,6 +40,7 @@ import org.apache.solr.client.solrj.embedded.JettyConfig.Builder;
|
|||
import org.apache.solr.client.solrj.embedded.JettySolrRunner;
|
||||
import org.apache.solr.client.solrj.impl.CloudSolrClient;
|
||||
import org.apache.solr.client.solrj.response.QueryResponse;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.SolrInputDocument;
|
||||
import org.apache.solr.common.cloud.ClusterState;
|
||||
import org.apache.solr.common.cloud.Replica;
|
||||
|
@ -144,7 +145,9 @@ public class TestMiniSolrCloudCluster extends LuceneTestCase {
|
|||
final String asyncId = (random().nextBoolean() ? null : "asyncId("+collectionName+".create)="+random().nextInt());
|
||||
createCollection(miniCluster, collectionName, null, asyncId, null, null);
|
||||
if (asyncId != null) {
|
||||
assertEquals("did not see async createCollection completion", "completed", AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330, cloudSolrClient));
|
||||
final RequestStatusState state = AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330,
|
||||
cloudSolrClient);
|
||||
assertSame("did not see async createCollection completion", RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
|
||||
ZkStateReader zkStateReader = miniCluster.getSolrClient().getZkStateReader();
|
||||
|
@ -204,7 +207,9 @@ public class TestMiniSolrCloudCluster extends LuceneTestCase {
|
|||
String asyncId2 = (random().nextBoolean() ? null : "asyncId("+collectionName+".create)="+random().nextInt());
|
||||
createCollection(miniCluster, collectionName, null, asyncId2, null, null);
|
||||
if (asyncId2 != null) {
|
||||
assertEquals("did not see async createCollection completion", "completed", AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId2, 330, cloudSolrClient));
|
||||
final RequestStatusState state = AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId2, 330,
|
||||
cloudSolrClient);
|
||||
assertSame("did not see async createCollection completion", RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
AbstractDistribZkTestBase.waitForRecoveriesToFinish(collectionName, zkStateReader, true, true, 330);
|
||||
|
||||
|
@ -299,7 +304,8 @@ public class TestMiniSolrCloudCluster extends LuceneTestCase {
|
|||
final String asyncId = (random().nextBoolean() ? null : "asyncId("+collectionName+".create)="+random().nextInt());
|
||||
createCollection(miniCluster, collectionName, OverseerCollectionMessageHandler.CREATE_NODE_SET_EMPTY, asyncId, null, null);
|
||||
if (asyncId != null) {
|
||||
assertEquals("did not see async createCollection completion", "completed", AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330, cloudSolrClient));
|
||||
final RequestStatusState state = AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330, cloudSolrClient);
|
||||
assertSame("did not see async createCollection completion", RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
|
||||
try (SolrZkClient zkClient = new SolrZkClient
|
||||
|
|
|
@ -33,6 +33,7 @@ import org.apache.solr.client.solrj.embedded.JettyConfig;
|
|||
import org.apache.solr.client.solrj.embedded.JettySolrRunner;
|
||||
import org.apache.solr.client.solrj.impl.CloudSolrClient;
|
||||
import org.apache.solr.client.solrj.response.QueryResponse;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.SolrInputDocument;
|
||||
import org.apache.solr.common.cloud.ClusterState;
|
||||
import org.apache.solr.common.cloud.Replica;
|
||||
|
@ -129,7 +130,9 @@ public class TestMiniSolrCloudClusterBase extends LuceneTestCase {
|
|||
final String asyncId = (random().nextBoolean() ? null : "asyncId("+collectionName+".create)="+random().nextInt());
|
||||
createCollection(miniCluster, collectionName, null, asyncId);
|
||||
if (asyncId != null) {
|
||||
assertEquals("did not see async createCollection completion", "completed", AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330, cloudSolrClient));
|
||||
final RequestStatusState state = AbstractFullDistribZkTestBase.getRequestStateAfterCompletion(asyncId, 330,
|
||||
cloudSolrClient);
|
||||
assertSame("did not see async createCollection completion", RequestStatusState.COMPLETED, state);
|
||||
}
|
||||
|
||||
try (SolrZkClient zkClient = new SolrZkClient
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.solr.client.solrj.SolrRequest;
|
|||
import org.apache.solr.client.solrj.SolrServerException;
|
||||
import org.apache.solr.client.solrj.impl.HttpSolrClient;
|
||||
import org.apache.solr.client.solrj.request.QueryRequest;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.params.CollectionParams;
|
||||
import org.apache.solr.common.params.CommonAdminParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
|
@ -71,7 +72,7 @@ public class TestRequestStatusCollectionAPI extends BasicDistributedZkTest {
|
|||
e.printStackTrace();
|
||||
}
|
||||
|
||||
assertEquals("Task 1000 not found in completed tasks.", "found 1000 in completed tasks", message);
|
||||
assertEquals("found [1000] in completed tasks", message);
|
||||
|
||||
// Check for a random (hopefully non-existent request id
|
||||
params = new ModifiableSolrParams();
|
||||
|
@ -85,7 +86,7 @@ public class TestRequestStatusCollectionAPI extends BasicDistributedZkTest {
|
|||
e.printStackTrace();
|
||||
}
|
||||
|
||||
assertEquals("Task 9999999 found in tasks queue.", "Did not find taskid [9999999] in any tasks queue", message);
|
||||
assertEquals("Did not find [9999999] in any tasks queue", message);
|
||||
|
||||
params = new ModifiableSolrParams();
|
||||
params.set(CollectionParams.ACTION, CollectionParams.CollectionAction.SPLITSHARD.toString());
|
||||
|
@ -99,36 +100,35 @@ public class TestRequestStatusCollectionAPI extends BasicDistributedZkTest {
|
|||
}
|
||||
|
||||
// Check for the request to be completed.
|
||||
params = new ModifiableSolrParams();
|
||||
params.set("action", CollectionParams.CollectionAction.REQUESTSTATUS.toString());
|
||||
params.set(OverseerCollectionMessageHandler.REQUESTID, "1001");
|
||||
params = new ModifiableSolrParams();
|
||||
params.set("action", CollectionParams.CollectionAction.REQUESTSTATUS.toString());
|
||||
params.set(OverseerCollectionMessageHandler.REQUESTID, "1001");
|
||||
try {
|
||||
message = sendStatusRequestWithRetry(params, MAX_WAIT_TIMEOUT_SECONDS);
|
||||
} catch (SolrServerException | IOException e) {
|
||||
e.printStackTrace();
|
||||
}
|
||||
|
||||
assertEquals("Task 1001 not found in completed tasks.", "found 1001 in completed tasks", message);
|
||||
assertEquals("found [1001] in completed tasks", message);
|
||||
|
||||
params = new ModifiableSolrParams();
|
||||
params.set(CollectionParams.ACTION, CollectionParams.CollectionAction.CREATE.toString());
|
||||
params.set("name", "collection2");
|
||||
params.set("numShards", 2);
|
||||
params.set("replicationFactor", 1);
|
||||
params.set("maxShardsPerNode", 100);
|
||||
params.set("collection.configName", "conf1");
|
||||
params.set(CommonAdminParams.ASYNC, "1002");
|
||||
params = new ModifiableSolrParams();
|
||||
params.set(CollectionParams.ACTION, CollectionParams.CollectionAction.CREATE.toString());
|
||||
params.set("name", "collection2");
|
||||
params.set("numShards", 2);
|
||||
params.set("replicationFactor", 1);
|
||||
params.set("maxShardsPerNode", 100);
|
||||
params.set("collection.configName", "conf1");
|
||||
params.set(CommonAdminParams.ASYNC, "1002");
|
||||
try {
|
||||
sendRequest(params);
|
||||
} catch (SolrServerException | IOException e) {
|
||||
e.printStackTrace();
|
||||
}
|
||||
|
||||
|
||||
params = new ModifiableSolrParams();
|
||||
|
||||
params.set("action", CollectionParams.CollectionAction.REQUESTSTATUS.toString());
|
||||
params.set(OverseerCollectionMessageHandler.REQUESTID, "1002");
|
||||
params.set("action", CollectionParams.CollectionAction.REQUESTSTATUS.toString());
|
||||
params.set(OverseerCollectionMessageHandler.REQUESTID, "1002");
|
||||
|
||||
try {
|
||||
message = sendStatusRequestWithRetry(params, MAX_WAIT_TIMEOUT_SECONDS);
|
||||
|
@ -137,51 +137,45 @@ public class TestRequestStatusCollectionAPI extends BasicDistributedZkTest {
|
|||
}
|
||||
|
||||
|
||||
assertEquals("Task 1002 not found in completed tasks.", "found 1002 in failed tasks", message);
|
||||
assertEquals("found [1002] in failed tasks", message);
|
||||
|
||||
params = new ModifiableSolrParams();
|
||||
params.set(CollectionParams.ACTION, CollectionParams.CollectionAction.CREATE.toString());
|
||||
params.set("name", "collection3");
|
||||
params.set("numShards", 1);
|
||||
params.set("replicationFactor", 1);
|
||||
params.set("maxShardsPerNode", 100);
|
||||
params.set("collection.configName", "conf1");
|
||||
params.set(CommonAdminParams.ASYNC, "1002");
|
||||
params = new ModifiableSolrParams();
|
||||
params.set(CollectionParams.ACTION, CollectionParams.CollectionAction.CREATE.toString());
|
||||
params.set("name", "collection3");
|
||||
params.set("numShards", 1);
|
||||
params.set("replicationFactor", 1);
|
||||
params.set("maxShardsPerNode", 100);
|
||||
params.set("collection.configName", "conf1");
|
||||
params.set(CommonAdminParams.ASYNC, "1002");
|
||||
try {
|
||||
r = sendRequest(params);
|
||||
} catch (SolrServerException | IOException e) {
|
||||
e.printStackTrace();
|
||||
}
|
||||
|
||||
assertEquals("Did not error out on duplicate requests (same request id)",
|
||||
"Task with the same requestid already exists.", r.get("error"));
|
||||
|
||||
assertEquals("Task with the same requestid already exists.", r.get("error"));
|
||||
}
|
||||
|
||||
|
||||
/**
|
||||
* Helper method to send a status request with specific retry limit and return
|
||||
* the message/null from the success response.
|
||||
*/
|
||||
private String sendStatusRequestWithRetry(ModifiableSolrParams params, int maxCounter)
|
||||
throws SolrServerException, IOException{
|
||||
NamedList status = null;
|
||||
String state = null;
|
||||
String message = null;
|
||||
NamedList r;
|
||||
while(maxCounter-- > 0) {
|
||||
r = sendRequest(params);
|
||||
status = (NamedList) r.get("status");
|
||||
state = (String) status.get("state");
|
||||
while (maxCounter-- > 0) {
|
||||
final NamedList r = sendRequest(params);
|
||||
final NamedList status = (NamedList) r.get("status");
|
||||
final RequestStatusState state = RequestStatusState.fromKey((String) status.get("state"));
|
||||
message = (String) status.get("msg");
|
||||
|
||||
if(state.equals("completed") || state.equals("failed"))
|
||||
return (String) status.get("msg");
|
||||
if (state == RequestStatusState.COMPLETED || state == RequestStatusState.FAILED) {
|
||||
return message;
|
||||
}
|
||||
|
||||
try {
|
||||
Thread.sleep(1000);
|
||||
} catch (InterruptedException e) {
|
||||
|
||||
}
|
||||
|
||||
}
|
||||
|
@ -193,8 +187,7 @@ public class TestRequestStatusCollectionAPI extends BasicDistributedZkTest {
|
|||
SolrRequest request = new QueryRequest(params);
|
||||
request.setPath("/admin/collections");
|
||||
|
||||
String baseUrl = ((HttpSolrClient) shardToJetty.get(SHARD1).get(0).client.solrClient)
|
||||
.getBaseURL();
|
||||
String baseUrl = ((HttpSolrClient) shardToJetty.get(SHARD1).get(0).client.solrClient).getBaseURL();
|
||||
baseUrl = baseUrl.substring(0, baseUrl.length() - "collection1".length());
|
||||
|
||||
try (HttpSolrClient baseServer = new HttpSolrClient(baseUrl)) {
|
||||
|
|
|
@ -0,0 +1,73 @@
|
|||
/*
|
||||
* 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.client.solrj.response;
|
||||
|
||||
import java.util.Locale;
|
||||
|
||||
/**
|
||||
* Represents the state of an asynchronous request.
|
||||
*
|
||||
* @see org.apache.solr.client.solrj.request.CollectionAdminRequest.RequestStatus
|
||||
*/
|
||||
public enum RequestStatusState {
|
||||
|
||||
/** The request was completed. */
|
||||
COMPLETED("completed"),
|
||||
|
||||
/** The request has failed. */
|
||||
FAILED("failed"),
|
||||
|
||||
/** The request is in progress. */
|
||||
RUNNING("running"),
|
||||
|
||||
/** The request was submitted, but has not yet started. */
|
||||
SUBMITTED("submitted"),
|
||||
|
||||
/** The request Id was not found. */
|
||||
NOT_FOUND("notfound");
|
||||
|
||||
private final String key;
|
||||
|
||||
private RequestStatusState(String key) {
|
||||
this.key = key;
|
||||
}
|
||||
|
||||
/**
|
||||
* Returns the string representation of this state, for using as a key. For backward compatibility, it returns the
|
||||
* lowercase form of the state's name.
|
||||
*/
|
||||
public String getKey() {
|
||||
return key;
|
||||
}
|
||||
|
||||
/**
|
||||
* Resolves a key that was returned from {@link #getKey()} to a {@link RequestStatusState}. For backwards
|
||||
* compatibility, it resolves the key "notfound" to {@link #NOT_FOUND}.
|
||||
*/
|
||||
public static RequestStatusState fromKey(String key) {
|
||||
try {
|
||||
return RequestStatusState.valueOf(key.toUpperCase(Locale.ENGLISH));
|
||||
} catch (final IllegalArgumentException e) {
|
||||
if (key.equalsIgnoreCase(RequestStatusState.NOT_FOUND.getKey())) {
|
||||
return RequestStatusState.NOT_FOUND;
|
||||
} else {
|
||||
throw e;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
}
|
|
@ -1,5 +1,3 @@
|
|||
package org.apache.solr.cloud;
|
||||
|
||||
/*
|
||||
* Licensed to the Apache Software Foundation (ASF) under one or more
|
||||
* contributor license agreements. See the NOTICE file distributed with
|
||||
|
@ -16,6 +14,12 @@ package org.apache.solr.cloud;
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.solr.cloud;
|
||||
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.CREATE_NODE_SET;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.NUM_SLICES;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.SHARDS_PROP;
|
||||
import static org.apache.solr.common.util.Utils.makeMap;
|
||||
|
||||
import java.io.File;
|
||||
import java.io.IOException;
|
||||
|
@ -51,6 +55,7 @@ import org.apache.solr.client.solrj.request.QueryRequest;
|
|||
import org.apache.solr.client.solrj.request.UpdateRequest;
|
||||
import org.apache.solr.client.solrj.response.CollectionAdminResponse;
|
||||
import org.apache.solr.client.solrj.response.QueryResponse;
|
||||
import org.apache.solr.client.solrj.response.RequestStatusState;
|
||||
import org.apache.solr.common.SolrDocument;
|
||||
import org.apache.solr.common.SolrDocumentList;
|
||||
import org.apache.solr.common.SolrException;
|
||||
|
@ -73,7 +78,6 @@ import org.apache.solr.common.util.Utils;
|
|||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.core.Diagnostics;
|
||||
import org.apache.solr.core.SolrCore;
|
||||
import org.apache.solr.servlet.SolrDispatchFilter;
|
||||
import org.apache.solr.update.DirectUpdateHandler2;
|
||||
import org.apache.solr.update.SolrCmdDistributor;
|
||||
import org.apache.solr.util.RTimer;
|
||||
|
@ -86,11 +90,6 @@ import org.noggit.JSONWriter;
|
|||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.CREATE_NODE_SET;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.NUM_SLICES;
|
||||
import static org.apache.solr.cloud.OverseerCollectionMessageHandler.SHARDS_PROP;
|
||||
import static org.apache.solr.common.util.Utils.makeMap;
|
||||
|
||||
/**
|
||||
* TODO: we should still test this works as a custom update chain as well as
|
||||
* what we test now - the default update chain
|
||||
|
@ -1895,16 +1894,16 @@ public abstract class AbstractFullDistribZkTestBase extends AbstractDistribZkTes
|
|||
return cs;
|
||||
}
|
||||
|
||||
|
||||
static String getRequestStateAfterCompletion(String requestId, int waitForSeconds, SolrClient client)
|
||||
static RequestStatusState getRequestStateAfterCompletion(String requestId, int waitForSeconds, SolrClient client)
|
||||
throws IOException, SolrServerException {
|
||||
String state = null;
|
||||
RequestStatusState state = null;
|
||||
final TimeOut timeout = new TimeOut(waitForSeconds, TimeUnit.SECONDS);
|
||||
|
||||
while (! timeout.hasTimedOut()) {
|
||||
while (!timeout.hasTimedOut()) {
|
||||
state = getRequestState(requestId, client);
|
||||
if(state.equals("completed") || state.equals("failed"))
|
||||
if (state == RequestStatusState.COMPLETED || state == RequestStatusState.FAILED) {
|
||||
return state;
|
||||
}
|
||||
try {
|
||||
Thread.sleep(1000);
|
||||
} catch (InterruptedException e) {
|
||||
|
@ -1914,17 +1913,17 @@ public abstract class AbstractFullDistribZkTestBase extends AbstractDistribZkTes
|
|||
return state;
|
||||
}
|
||||
|
||||
static String getRequestState(int requestId, SolrClient client) throws IOException, SolrServerException {
|
||||
static RequestStatusState getRequestState(int requestId, SolrClient client) throws IOException, SolrServerException {
|
||||
return getRequestState(String.valueOf(requestId), client);
|
||||
}
|
||||
|
||||
static String getRequestState(String requestId, SolrClient client) throws IOException, SolrServerException {
|
||||
static RequestStatusState getRequestState(String requestId, SolrClient client) throws IOException, SolrServerException {
|
||||
CollectionAdminRequest.RequestStatus requestStatusRequest = new CollectionAdminRequest.RequestStatus();
|
||||
requestStatusRequest.setRequestId(requestId);
|
||||
CollectionAdminResponse response = requestStatusRequest.process(client);
|
||||
|
||||
NamedList innerResponse = (NamedList) response.getResponse().get("status");
|
||||
return (String) innerResponse.get("state");
|
||||
return RequestStatusState.fromKey((String) innerResponse.get("state"));
|
||||
}
|
||||
|
||||
}
|
||||
|
|
Loading…
Reference in New Issue