mirror of https://github.com/apache/lucene.git
SOLR-14794: revert commit passing CloudConfig everywhere (a3b3ba10e3
) and remove tab in solr/bin/solr.cmd
This commit is contained in:
parent
b421878b77
commit
825c18c59b
|
@ -2051,7 +2051,7 @@ FOR /f "usebackq tokens=3" %%a IN (`^""%JAVA%" -version 2^>^&1 ^| findstr "versi
|
|||
echo "java version info is !JAVA_VERSION_INFO!"
|
||||
REM Extract the major Java version, e.g. 7, 8, 9, 10 ...
|
||||
for /f "tokens=1,2 delims=._-" %%a in ("!JAVA_VERSION_INFO!") do (
|
||||
echo "Extracted major version is %%a"
|
||||
echo "Extracted major version is %%a"
|
||||
if %%a GEQ 9 (
|
||||
set JAVA_MAJOR_VERSION=%%a
|
||||
) else (
|
||||
|
|
|
@ -603,7 +603,7 @@ public class Overseer implements SolrCloseable {
|
|||
private volatile boolean closed;
|
||||
private volatile boolean systemCollCompatCheck = true;
|
||||
|
||||
private final CloudConfig config;
|
||||
private CloudConfig config;
|
||||
|
||||
// overseer not responsible for closing reader
|
||||
public Overseer(HttpShardHandler shardHandler,
|
||||
|
@ -637,7 +637,7 @@ public class Overseer implements SolrCloseable {
|
|||
ThreadGroup ccTg = new ThreadGroup("Overseer collection creation process.");
|
||||
|
||||
OverseerNodePrioritizer overseerPrioritizer = new OverseerNodePrioritizer(reader, getStateUpdateQueue(), adminPath, shardHandler.getShardHandlerFactory());
|
||||
overseerCollectionConfigSetProcessor = new OverseerCollectionConfigSetProcessor(config, reader, id, shardHandler, adminPath, stats, Overseer.this, overseerPrioritizer);
|
||||
overseerCollectionConfigSetProcessor = new OverseerCollectionConfigSetProcessor(reader, id, shardHandler, adminPath, stats, Overseer.this, overseerPrioritizer);
|
||||
ccThread = new OverseerThread(ccTg, overseerCollectionConfigSetProcessor, "OverseerCollectionConfigSetProcessor-" + id);
|
||||
ccThread.setDaemon(true);
|
||||
|
||||
|
|
|
@ -24,7 +24,6 @@ import org.apache.commons.io.IOUtils;
|
|||
import org.apache.solr.cloud.api.collections.OverseerCollectionMessageHandler;
|
||||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.component.HttpShardHandler;
|
||||
import org.apache.solr.handler.component.HttpShardHandlerFactory;
|
||||
|
||||
|
@ -35,12 +34,11 @@ import org.apache.solr.handler.component.HttpShardHandlerFactory;
|
|||
*/
|
||||
public class OverseerCollectionConfigSetProcessor extends OverseerTaskProcessor {
|
||||
|
||||
public OverseerCollectionConfigSetProcessor(CloudConfig cloudConfig, ZkStateReader zkStateReader, String myId,
|
||||
public OverseerCollectionConfigSetProcessor(ZkStateReader zkStateReader, String myId,
|
||||
final HttpShardHandler shardHandler,
|
||||
String adminPath, Stats stats, Overseer overseer,
|
||||
OverseerNodePrioritizer overseerNodePrioritizer) {
|
||||
this(
|
||||
cloudConfig,
|
||||
zkStateReader,
|
||||
myId,
|
||||
(HttpShardHandlerFactory) shardHandler.getShardHandlerFactory(),
|
||||
|
@ -55,7 +53,7 @@ public class OverseerCollectionConfigSetProcessor extends OverseerTaskProcessor
|
|||
);
|
||||
}
|
||||
|
||||
protected OverseerCollectionConfigSetProcessor(CloudConfig cloudConfig, ZkStateReader zkStateReader, String myId,
|
||||
protected OverseerCollectionConfigSetProcessor(ZkStateReader zkStateReader, String myId,
|
||||
final HttpShardHandlerFactory shardHandlerFactory,
|
||||
String adminPath,
|
||||
Stats stats,
|
||||
|
@ -66,7 +64,6 @@ public class OverseerCollectionConfigSetProcessor extends OverseerTaskProcessor
|
|||
DistributedMap completedMap,
|
||||
DistributedMap failureMap) {
|
||||
super(
|
||||
cloudConfig,
|
||||
zkStateReader,
|
||||
myId,
|
||||
stats,
|
||||
|
|
|
@ -26,7 +26,6 @@ import org.apache.solr.common.cloud.ZkStateReader;
|
|||
import org.apache.solr.common.params.ConfigSetParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.ConfigSetProperties;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
|
@ -93,7 +92,7 @@ public class OverseerConfigSetMessageHandler implements OverseerMessageHandler {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public OverseerSolrResponse processMessage(ZkNodeProps message, CloudConfig cloudConfig, String operation) {
|
||||
public OverseerSolrResponse processMessage(ZkNodeProps message, String operation) {
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
NamedList results = new NamedList();
|
||||
try {
|
||||
|
|
|
@ -17,7 +17,6 @@
|
|||
package org.apache.solr.cloud;
|
||||
|
||||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
|
||||
/**
|
||||
* Interface for processing messages received by an {@link OverseerTaskProcessor}
|
||||
|
@ -30,7 +29,7 @@ public interface OverseerMessageHandler {
|
|||
*
|
||||
* @return response
|
||||
*/
|
||||
OverseerSolrResponse processMessage(ZkNodeProps message, CloudConfig cloudConfig, String operation);
|
||||
OverseerSolrResponse processMessage(ZkNodeProps message, String operation);
|
||||
|
||||
/**
|
||||
* @return the name of the OverseerMessageHandler
|
||||
|
|
|
@ -44,7 +44,6 @@ import org.apache.solr.common.cloud.ZkStateReader;
|
|||
import org.apache.solr.common.util.ExecutorUtil;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.logging.MDCLoggingContext;
|
||||
import org.apache.solr.common.util.SolrNamedThreadFactory;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
|
@ -134,9 +133,7 @@ public class OverseerTaskProcessor implements Runnable, Closeable {
|
|||
|
||||
private String thisNode;
|
||||
|
||||
private final CloudConfig cloudConfig;
|
||||
|
||||
public OverseerTaskProcessor(CloudConfig cloudConfig, ZkStateReader zkStateReader, String myId,
|
||||
public OverseerTaskProcessor(ZkStateReader zkStateReader, String myId,
|
||||
Stats stats,
|
||||
OverseerMessageHandlerSelector selector,
|
||||
OverseerNodePrioritizer prioritizer,
|
||||
|
@ -144,7 +141,6 @@ public class OverseerTaskProcessor implements Runnable, Closeable {
|
|||
DistributedMap runningMap,
|
||||
DistributedMap completedMap,
|
||||
DistributedMap failureMap) {
|
||||
this.cloudConfig = cloudConfig;
|
||||
this.zkStateReader = zkStateReader;
|
||||
this.myId = myId;
|
||||
this.stats = stats;
|
||||
|
@ -345,7 +341,7 @@ public class OverseerTaskProcessor implements Runnable, Closeable {
|
|||
if (log.isDebugEnabled()) {
|
||||
log.debug("{}: Get the message id: {} message: {}", messageHandler.getName(), head.getId(), message);
|
||||
}
|
||||
Runner runner = new Runner(messageHandler, cloudConfig, message, operation, head, lock);
|
||||
Runner runner = new Runner(messageHandler, message, operation, head, lock);
|
||||
tpe.execute(runner);
|
||||
}
|
||||
|
||||
|
@ -501,16 +497,14 @@ public class OverseerTaskProcessor implements Runnable, Closeable {
|
|||
QueueEvent head;
|
||||
OverseerMessageHandler messageHandler;
|
||||
private final OverseerMessageHandler.Lock lock;
|
||||
private final CloudConfig cloudConfig;
|
||||
|
||||
public Runner(OverseerMessageHandler messageHandler, CloudConfig cloudConfig, ZkNodeProps message, String operation, QueueEvent head, OverseerMessageHandler.Lock lock) {
|
||||
public Runner(OverseerMessageHandler messageHandler, ZkNodeProps message, String operation, QueueEvent head, OverseerMessageHandler.Lock lock) {
|
||||
this.message = message;
|
||||
this.operation = operation;
|
||||
this.head = head;
|
||||
this.messageHandler = messageHandler;
|
||||
this.lock = lock;
|
||||
response = null;
|
||||
this.cloudConfig = cloudConfig;
|
||||
}
|
||||
|
||||
|
||||
|
@ -527,7 +521,7 @@ public class OverseerTaskProcessor implements Runnable, Closeable {
|
|||
if (log.isDebugEnabled()) {
|
||||
log.debug("Runner processing {}", head.getId());
|
||||
}
|
||||
response = messageHandler.processMessage(message, cloudConfig, operation);
|
||||
response = messageHandler.processMessage(message, operation);
|
||||
} finally {
|
||||
timerContext.stop();
|
||||
updateStats(statsName);
|
||||
|
|
|
@ -67,7 +67,6 @@ import org.apache.solr.common.params.ModifiableSolrParams;
|
|||
import org.apache.solr.common.params.ShardParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -83,12 +82,12 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
addReplica(state, cloudConfig, message, results, null);
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
addReplica(state, message, results, null);
|
||||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
List<ZkNodeProps> addReplica(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, Runnable onComplete)
|
||||
List<ZkNodeProps> addReplica(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, Runnable onComplete)
|
||||
throws IOException, InterruptedException, KeeperException {
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("addReplica() : {}", Utils.toJSONString(message));
|
||||
|
@ -147,7 +146,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
}
|
||||
|
||||
List<CreateReplica> createReplicas = buildReplicaPositions(ocmh.cloudManager, clusterState, cloudConfig, collectionName, message, replicaTypesVsCount)
|
||||
List<CreateReplica> createReplicas = buildReplicaPositions(ocmh.cloudManager, clusterState, collectionName, message, replicaTypesVsCount)
|
||||
.stream()
|
||||
.map(replicaPosition -> assignReplicaDetails(ocmh.cloudManager, clusterState, message, replicaPosition))
|
||||
.collect(Collectors.toList());
|
||||
|
@ -159,7 +158,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
final ShardRequestTracker shardRequestTracker = ocmh.asyncRequestTracker(asyncId);
|
||||
for (CreateReplica createReplica : createReplicas) {
|
||||
assert createReplica.coreName != null;
|
||||
ModifiableSolrParams params = getReplicaParams(clusterState, cloudConfig, message, results, collectionName, coll, skipCreateReplicaInClusterState, asyncId, shardHandler, createReplica);
|
||||
ModifiableSolrParams params = getReplicaParams(clusterState, message, results, collectionName, coll, skipCreateReplicaInClusterState, asyncId, shardHandler, createReplica);
|
||||
shardRequestTracker.sendShardRequest(createReplica.node, params, shardHandler);
|
||||
}
|
||||
|
||||
|
@ -202,7 +201,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
.collect(Collectors.toList());
|
||||
}
|
||||
|
||||
private ModifiableSolrParams getReplicaParams(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, String collectionName, DocCollection coll, boolean skipCreateReplicaInClusterState, String asyncId, ShardHandler shardHandler, CreateReplica createReplica) throws IOException, InterruptedException, KeeperException {
|
||||
private ModifiableSolrParams getReplicaParams(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, String collectionName, DocCollection coll, boolean skipCreateReplicaInClusterState, String asyncId, ShardHandler shardHandler, CreateReplica createReplica) throws IOException, InterruptedException, KeeperException {
|
||||
if (coll.getStr(WITH_COLLECTION) != null) {
|
||||
String withCollectionName = coll.getStr(WITH_COLLECTION);
|
||||
DocCollection withCollection = clusterState.getCollection(withCollectionName);
|
||||
|
@ -222,7 +221,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
// since we already computed node assignments (which include assigning a node for this withCollection replica) we want to skip the assignment step
|
||||
CollectionAdminParams.SKIP_NODE_ASSIGNMENT, "true",
|
||||
CommonAdminParams.WAIT_FOR_FINAL_STATE, Boolean.TRUE.toString()); // set to true because we want `withCollection` to be ready after this collection is created
|
||||
addReplica(clusterState, cloudConfig, props, results, null);
|
||||
addReplica(clusterState, props, results, null);
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -329,7 +328,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
return new CreateReplica(collection, shard, node, replicaType, coreName, coreNodeName);
|
||||
}
|
||||
|
||||
public static List<ReplicaPosition> buildReplicaPositions(SolrCloudManager cloudManager, ClusterState clusterState, CloudConfig cloudConfig,
|
||||
public static List<ReplicaPosition> buildReplicaPositions(SolrCloudManager cloudManager, ClusterState clusterState,
|
||||
String collectionName, ZkNodeProps message,
|
||||
EnumMap<Replica.Type, Integer> replicaTypeVsCount) throws IOException, InterruptedException {
|
||||
boolean skipCreateReplicaInClusterState = message.getBool(SKIP_CREATE_REPLICA_IN_CLUSTER_STATE, false);
|
||||
|
@ -354,7 +353,7 @@ public class AddReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
List<ReplicaPosition> positions = null;
|
||||
if (!skipCreateReplicaInClusterState && !skipNodeAssignment) {
|
||||
|
||||
positions = Assign.getNodesForNewReplicas(clusterState, cloudConfig, collection.getName(), sliceName, numNrtReplicas,
|
||||
positions = Assign.getNodesForNewReplicas(clusterState, collection.getName(), sliceName, numNrtReplicas,
|
||||
numTlogReplicas, numPullReplicas, createNodeSetStr, cloudManager);
|
||||
}
|
||||
|
||||
|
|
|
@ -28,7 +28,6 @@ import org.apache.solr.common.cloud.ZkNodeProps;
|
|||
import org.apache.solr.common.params.CollectionParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.admin.CollectionsHandler;
|
||||
import org.apache.solr.request.LocalSolrQueryRequest;
|
||||
|
||||
|
@ -55,8 +54,8 @@ abstract class AliasCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
* If the collection already exists then this is not an error.<p>
|
||||
*/
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
static NamedList createCollectionAndWait(ClusterState clusterState, CloudConfig cloudConfig, String aliasName, Map<String, String> aliasMetadata,
|
||||
String createCollName, OverseerCollectionMessageHandler ocmh) throws Exception {
|
||||
static NamedList createCollectionAndWait(ClusterState clusterState, String aliasName, Map<String, String> aliasMetadata,
|
||||
String createCollName, OverseerCollectionMessageHandler ocmh) throws Exception {
|
||||
// Map alias metadata starting with a prefix to a create-collection API request
|
||||
final ModifiableSolrParams createReqParams = new ModifiableSolrParams();
|
||||
for (Map.Entry<String, String> e : aliasMetadata.entrySet()) {
|
||||
|
@ -82,7 +81,7 @@ abstract class AliasCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
// Since we are running in the Overseer here, send the message directly to the Overseer CreateCollectionCmd.
|
||||
// note: there's doesn't seem to be any point in locking on the collection name, so we don't. We currently should
|
||||
// already have a lock on the alias name which should be sufficient.
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, cloudConfig, new ZkNodeProps(createMsgMap), results);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, new ZkNodeProps(createMsgMap), results);
|
||||
} catch (SolrException e) {
|
||||
// The collection might already exist, and that's okay -- we can adopt it.
|
||||
if (!e.getMessage().contains("collection already exists")) {
|
||||
|
|
|
@ -53,7 +53,6 @@ import org.apache.solr.common.cloud.Slice;
|
|||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.util.NumberUtils;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
|
@ -270,7 +269,7 @@ public class Assign {
|
|||
// Gets a list of candidate nodes to put the required replica(s) on. Throws errors if the AssignStrategy
|
||||
// can't allocate valid positions.
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public static List<ReplicaPosition> getNodesForNewReplicas(ClusterState clusterState, CloudConfig cloudConfig, String collectionName,
|
||||
public static List<ReplicaPosition> getNodesForNewReplicas(ClusterState clusterState, String collectionName,
|
||||
String shard, int nrtReplicas, int tlogReplicas, int pullReplicas,
|
||||
Object createNodeSet, SolrCloudManager cloudManager) throws IOException, InterruptedException, AssignmentException {
|
||||
log.debug("getNodesForNewReplicas() shard: {} , nrtReplicas : {} , tlogReplicas: {} , pullReplicas: {} , createNodeSet {}"
|
||||
|
@ -299,7 +298,7 @@ public class Assign {
|
|||
.onNodes(createNodeList)
|
||||
.build();
|
||||
AssignStrategyFactory assignStrategyFactory = new AssignStrategyFactory(cloudManager);
|
||||
AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, cloudConfig, coll);
|
||||
AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, coll);
|
||||
return assignStrategy.assign(cloudManager, assignRequest);
|
||||
}
|
||||
|
||||
|
@ -551,7 +550,7 @@ public class Assign {
|
|||
this.solrCloudManager = solrCloudManager;
|
||||
}
|
||||
|
||||
public AssignStrategy create(ClusterState clusterState, CloudConfig cloudConfig, DocCollection collection) throws IOException, InterruptedException {
|
||||
public AssignStrategy create(ClusterState clusterState, DocCollection collection) throws IOException, InterruptedException {
|
||||
@SuppressWarnings({"unchecked", "rawtypes"})
|
||||
List<Map> ruleMaps = (List<Map>) collection.get("rule");
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
|
|
|
@ -45,7 +45,6 @@ import org.apache.solr.common.params.CollectionAdminParams;
|
|||
import org.apache.solr.common.params.CoreAdminParams;
|
||||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.core.backup.BackupManager;
|
||||
import org.apache.solr.core.backup.repository.BackupRepository;
|
||||
|
@ -67,7 +66,7 @@ public class BackupCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(COLLECTION_PROP);
|
||||
boolean followAliases = message.getBool(FOLLOW_ALIASES, false);
|
||||
String collectionName;
|
||||
|
|
|
@ -35,7 +35,6 @@ import org.apache.solr.common.params.CollectionAdminParams;
|
|||
import org.apache.solr.common.params.CommonParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
|
||||
import static org.apache.solr.common.SolrException.ErrorCode.BAD_REQUEST;
|
||||
import static org.apache.solr.common.SolrException.ErrorCode.SERVER_ERROR;
|
||||
|
@ -53,7 +52,7 @@ public class CreateAliasCmd extends AliasCmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
throws Exception {
|
||||
final String aliasName = message.getStr(CommonParams.NAME);
|
||||
ZkStateReader zkStateReader = ocmh.zkStateReader;
|
||||
|
@ -65,7 +64,7 @@ public class CreateAliasCmd extends AliasCmd {
|
|||
if (!anyRoutingParams(message)) {
|
||||
callCreatePlainAlias(message, aliasName, zkStateReader);
|
||||
} else {
|
||||
callCreateRoutedAlias(message, aliasName, zkStateReader, state, cloudConfig);
|
||||
callCreateRoutedAlias(message, aliasName, zkStateReader, state);
|
||||
}
|
||||
|
||||
// Sleep a bit to allow ZooKeeper state propagation.
|
||||
|
@ -109,7 +108,7 @@ public class CreateAliasCmd extends AliasCmd {
|
|||
}
|
||||
|
||||
@SuppressWarnings("unchecked")
|
||||
private void callCreateRoutedAlias(ZkNodeProps message, String aliasName, ZkStateReader zkStateReader, ClusterState state, CloudConfig cloudConfig) throws Exception {
|
||||
private void callCreateRoutedAlias(ZkNodeProps message, String aliasName, ZkStateReader zkStateReader, ClusterState state) throws Exception {
|
||||
// Validate we got a basic minimum
|
||||
if (!message.getProperties().keySet().containsAll(RoutedAlias.MINIMAL_REQUIRED_PARAMS)) {
|
||||
throw new SolrException(BAD_REQUEST, "A routed alias requires these params: " + RoutedAlias.MINIMAL_REQUIRED_PARAMS
|
||||
|
@ -134,16 +133,16 @@ public class CreateAliasCmd extends AliasCmd {
|
|||
|
||||
// Create the first collection.
|
||||
String initialColl = routedAlias.computeInitialCollectionName();
|
||||
ensureAliasCollection(aliasName, zkStateReader, state, cloudConfig, routedAlias.getAliasMetadata(), initialColl);
|
||||
ensureAliasCollection(aliasName, zkStateReader, state, routedAlias.getAliasMetadata(), initialColl);
|
||||
// Create/update the alias
|
||||
zkStateReader.aliasesManager.applyModificationAndExportToZk(aliases -> aliases
|
||||
.cloneWithCollectionAlias(aliasName, initialColl)
|
||||
.cloneWithCollectionAliasProperties(aliasName, routedAlias.getAliasMetadata()));
|
||||
}
|
||||
|
||||
private void ensureAliasCollection(String aliasName, ZkStateReader zkStateReader, ClusterState state, CloudConfig cloudConfig, Map<String, String> aliasProperties, String initialCollectionName) throws Exception {
|
||||
private void ensureAliasCollection(String aliasName, ZkStateReader zkStateReader, ClusterState state, Map<String, String> aliasProperties, String initialCollectionName) throws Exception {
|
||||
// Create the collection
|
||||
createCollectionAndWait(state, cloudConfig, aliasName, aliasProperties, initialCollectionName, ocmh);
|
||||
createCollectionAndWait(state, aliasName, aliasProperties, initialCollectionName, ocmh);
|
||||
validateAllCollectionsExistAndNoDuplicates(Collections.singletonList(initialCollectionName), zkStateReader);
|
||||
}
|
||||
|
||||
|
|
|
@ -63,7 +63,6 @@ import org.apache.solr.common.util.NamedList;
|
|||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.admin.ConfigSetsHandlerApi;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.handler.component.ShardRequest;
|
||||
|
@ -102,7 +101,7 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
if (ocmh.zkStateReader.aliasesManager != null) { // not a mock ZkStateReader
|
||||
ocmh.zkStateReader.aliasesManager.update();
|
||||
}
|
||||
|
@ -186,10 +185,10 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
|
||||
List<ReplicaPosition> replicaPositions = null;
|
||||
try {
|
||||
replicaPositions = buildReplicaPositions(ocmh.cloudManager, clusterState, cloudConfig, clusterState.getCollection(collectionName), message, shardNames);
|
||||
replicaPositions = buildReplicaPositions(ocmh.cloudManager, clusterState, clusterState.getCollection(collectionName), message, shardNames);
|
||||
} catch (Assign.AssignmentException e) {
|
||||
ZkNodeProps deleteMessage = new ZkNodeProps("name", collectionName);
|
||||
new DeleteCollectionCmd(ocmh).call(clusterState, cloudConfig, deleteMessage, results);
|
||||
new DeleteCollectionCmd(ocmh).call(clusterState, deleteMessage, results);
|
||||
// unwrap the exception
|
||||
throw new SolrException(ErrorCode.BAD_REQUEST, e.getMessage(), e.getCause());
|
||||
}
|
||||
|
@ -220,7 +219,7 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
ZkStateReader.SHARD_ID_PROP, withCollectionShard,
|
||||
"node", nodeName,
|
||||
CommonAdminParams.WAIT_FOR_FINAL_STATE, Boolean.TRUE.toString()); // set to true because we want `withCollection` to be ready after this collection is created
|
||||
new AddReplicaCmd(ocmh).call(clusterState, cloudConfig, props, results);
|
||||
new AddReplicaCmd(ocmh).call(clusterState, props, results);
|
||||
clusterState = zkStateReader.getClusterState(); // refresh
|
||||
}
|
||||
}
|
||||
|
@ -293,7 +292,7 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
// Let's cleanup as we hit an exception
|
||||
// We shouldn't be passing 'results' here for the cleanup as the response would then contain 'success'
|
||||
// element, which may be interpreted by the user as a positive ack
|
||||
ocmh.cleanupCollection(cloudConfig, collectionName, new NamedList<Object>());
|
||||
ocmh.cleanupCollection(collectionName, new NamedList<Object>());
|
||||
log.info("Cleaned up artifacts for failed create collection for [{}]", collectionName);
|
||||
throw new SolrException(ErrorCode.BAD_REQUEST, "Underlying core creation failed while creating collection: " + collectionName);
|
||||
} else {
|
||||
|
@ -337,7 +336,7 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
}
|
||||
}
|
||||
|
||||
public static List<ReplicaPosition> buildReplicaPositions(SolrCloudManager cloudManager, ClusterState clusterState, CloudConfig cloudConfig,
|
||||
public static List<ReplicaPosition> buildReplicaPositions(SolrCloudManager cloudManager, ClusterState clusterState,
|
||||
DocCollection docCollection,
|
||||
ZkNodeProps message,
|
||||
List<String> shardNames) throws IOException, InterruptedException, Assign.AssignmentException {
|
||||
|
@ -380,7 +379,7 @@ public class CreateCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
.onNodes(nodeList)
|
||||
.build();
|
||||
Assign.AssignStrategyFactory assignStrategyFactory = new Assign.AssignStrategyFactory(cloudManager);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, cloudConfig, docCollection);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, docCollection);
|
||||
replicaPositions = assignStrategy.assign(cloudManager, assignRequest);
|
||||
}
|
||||
return replicaPositions;
|
||||
|
|
|
@ -30,7 +30,6 @@ import org.apache.solr.common.params.CommonAdminParams;
|
|||
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.CloudConfig;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
@ -53,7 +52,7 @@ public class CreateShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked", "rawtypes"})
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(COLLECTION_PROP);
|
||||
String sliceName = message.getStr(SHARD_ID_PROP);
|
||||
boolean waitForFinalState = message.getBool(CommonAdminParams.WAIT_FOR_FINAL_STATE, false);
|
||||
|
@ -103,7 +102,7 @@ public class CreateShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
final NamedList addResult = new NamedList();
|
||||
try {
|
||||
//ocmh.addReplica(zkStateReader.getClusterState(), addReplicasProps, addResult, () -> {
|
||||
ocmh.addReplica(clusterState, cloudConfig, addReplicasProps, addResult, () -> {
|
||||
ocmh.addReplica(clusterState, addReplicasProps, addResult, () -> {
|
||||
Object addResultFailure = addResult.get("failure");
|
||||
if (addResultFailure != null) {
|
||||
SimpleOrderedMap failure = (SimpleOrderedMap) results.get("failure");
|
||||
|
@ -124,7 +123,7 @@ public class CreateShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
} catch (Assign.AssignmentException e) {
|
||||
// clean up the slice that we created
|
||||
ZkNodeProps deleteShard = new ZkNodeProps(COLLECTION_PROP, collectionName, SHARD_ID_PROP, sliceName, ASYNC, async);
|
||||
new DeleteShardCmd(ocmh).call(clusterState, cloudConfig, deleteShard, results);
|
||||
new DeleteShardCmd(ocmh).call(clusterState, deleteShard, results);
|
||||
throw e;
|
||||
}
|
||||
|
||||
|
|
|
@ -45,7 +45,6 @@ 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.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData.CoreSnapshotMetaData;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData.SnapshotStatus;
|
||||
|
@ -67,7 +66,7 @@ public class CreateSnapshotCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(COLLECTION_PROP);
|
||||
boolean followAliases = message.getBool(FOLLOW_ALIASES, false);
|
||||
|
||||
|
|
|
@ -22,7 +22,6 @@ import org.apache.solr.common.cloud.ClusterState;
|
|||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
|
||||
import static org.apache.solr.common.params.CommonParams.NAME;
|
||||
|
||||
|
@ -34,7 +33,7 @@ public class DeleteAliasCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String aliasName = message.getStr(NAME);
|
||||
|
||||
ZkStateReader zkStateReader = ocmh.zkStateReader;
|
||||
|
|
|
@ -43,7 +43,6 @@ import org.apache.solr.common.params.ModifiableSolrParams;
|
|||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.SolrInfoBean;
|
||||
import org.apache.solr.core.snapshots.SolrSnapshotManager;
|
||||
import org.apache.solr.handler.admin.ConfigSetsHandlerApi;
|
||||
|
@ -71,7 +70,7 @@ public class DeleteCollectionCmd implements OverseerCollectionMessageHandler.Cmd
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
Object o = message.get(MaintainRoutedAliasCmd.INVOKED_BY_ROUTED_ALIAS);
|
||||
if (o != null) {
|
||||
((Runnable)o).run(); // this will ensure the collection is removed from the alias before it disappears.
|
||||
|
|
|
@ -32,7 +32,6 @@ import org.apache.solr.common.cloud.Slice;
|
|||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
@ -53,7 +52,7 @@ public class DeleteNodeCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
ocmh.checkRequired(message, "node");
|
||||
String node = message.getStr("node");
|
||||
List<ZkNodeProps> sourceReplicas = ReplaceNodeCmd.getReplicasOfNode(node, state);
|
||||
|
|
|
@ -46,7 +46,6 @@ import org.apache.solr.common.params.ModifiableSolrParams;
|
|||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -63,7 +62,7 @@ public class DeleteReplicaCmd implements Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings("unchecked")
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
deleteReplica(clusterState, message, results,null);
|
||||
}
|
||||
|
||||
|
|
|
@ -47,7 +47,6 @@ import org.apache.solr.common.util.NamedList;
|
|||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
@ -64,7 +63,7 @@ public class DeleteShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(ZkStateReader.COLLECTION_PROP);
|
||||
String sliceId = message.getStr(ZkStateReader.SHARD_ID_PROP);
|
||||
|
||||
|
|
|
@ -43,7 +43,6 @@ import org.apache.solr.common.params.CoreAdminParams.CoreAdminAction;
|
|||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData.CoreSnapshotMetaData;
|
||||
import org.apache.solr.core.snapshots.CollectionSnapshotMetaData.SnapshotStatus;
|
||||
|
@ -65,7 +64,7 @@ public class DeleteSnapshotCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(COLLECTION_PROP);
|
||||
boolean followAliases = message.getBool(FOLLOW_ALIASES, false);
|
||||
String collectionName;
|
||||
|
|
|
@ -34,7 +34,6 @@ import org.apache.solr.common.cloud.ZkStateReader;
|
|||
import org.apache.solr.common.params.CollectionParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.admin.CollectionsHandler;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
@ -100,7 +99,7 @@ public class MaintainRoutedAliasCmd extends AliasCmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
//---- PARSE PRIMARY MESSAGE PARAMS
|
||||
// important that we use NAME for the alias as that is what the Overseer will get a lock on before calling us
|
||||
final String aliasName = message.getStr(NAME);
|
||||
|
@ -127,7 +126,7 @@ public class MaintainRoutedAliasCmd extends AliasCmd {
|
|||
if (exists) {
|
||||
ocmh.tpe.submit(() -> {
|
||||
try {
|
||||
deleteTargetCollection(clusterState, cloudConfig, results, aliasName, aliasesManager, action);
|
||||
deleteTargetCollection(clusterState, results, aliasName, aliasesManager, action);
|
||||
} catch (Exception e) {
|
||||
log.warn("Deletion of {} by {} {} failed (this might be ok if two clients were"
|
||||
, action.targetCollection, ra.getAliasName()
|
||||
|
@ -139,7 +138,7 @@ public class MaintainRoutedAliasCmd extends AliasCmd {
|
|||
break;
|
||||
case ENSURE_EXISTS:
|
||||
if (!exists) {
|
||||
addTargetCollection(clusterState, cloudConfig, results, aliasName, aliasesManager, aliasMetadata, action);
|
||||
addTargetCollection(clusterState, results, aliasName, aliasesManager, aliasMetadata, action);
|
||||
} else {
|
||||
// check that the collection is properly integrated into the alias (see
|
||||
// TimeRoutedAliasUpdateProcessorTest.java:141). Presently we need to ensure inclusion in the alias
|
||||
|
@ -164,9 +163,9 @@ public class MaintainRoutedAliasCmd extends AliasCmd {
|
|||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void addTargetCollection(ClusterState clusterState, CloudConfig cloudConfig, @SuppressWarnings({"rawtypes"})NamedList results, String aliasName, ZkStateReader.AliasesManager aliasesManager, Map<String, String> aliasMetadata, RoutedAlias.Action action) throws Exception {
|
||||
public void addTargetCollection(ClusterState clusterState, @SuppressWarnings({"rawtypes"})NamedList results, String aliasName, ZkStateReader.AliasesManager aliasesManager, Map<String, String> aliasMetadata, RoutedAlias.Action action) throws Exception {
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
NamedList createResults = createCollectionAndWait(clusterState, cloudConfig, aliasName, aliasMetadata,
|
||||
NamedList createResults = createCollectionAndWait(clusterState, aliasName, aliasMetadata,
|
||||
action.targetCollection, ocmh);
|
||||
if (createResults != null) {
|
||||
results.add("create", createResults);
|
||||
|
@ -174,12 +173,12 @@ public class MaintainRoutedAliasCmd extends AliasCmd {
|
|||
addCollectionToAlias(aliasName, aliasesManager, action.targetCollection);
|
||||
}
|
||||
|
||||
public void deleteTargetCollection(ClusterState clusterState, CloudConfig cloudConfig, @SuppressWarnings({"rawtypes"})NamedList results, String aliasName, ZkStateReader.AliasesManager aliasesManager, RoutedAlias.Action action) throws Exception {
|
||||
public void deleteTargetCollection(ClusterState clusterState, @SuppressWarnings({"rawtypes"})NamedList results, String aliasName, ZkStateReader.AliasesManager aliasesManager, RoutedAlias.Action action) throws Exception {
|
||||
Map<String, Object> delProps = new HashMap<>();
|
||||
delProps.put(INVOKED_BY_ROUTED_ALIAS,
|
||||
(Runnable) () -> removeCollectionFromAlias(aliasName, aliasesManager, action.targetCollection));
|
||||
delProps.put(NAME, action.targetCollection);
|
||||
ZkNodeProps messageDelete = new ZkNodeProps(delProps);
|
||||
new DeleteCollectionCmd(ocmh).call(clusterState, cloudConfig, messageDelete, results);
|
||||
new DeleteCollectionCmd(ocmh).call(clusterState, messageDelete, results);
|
||||
}
|
||||
}
|
||||
|
|
|
@ -43,7 +43,6 @@ import org.apache.solr.common.params.ModifiableSolrParams;
|
|||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.handler.component.ShardHandlerFactory;
|
||||
import org.apache.solr.update.SolrIndexSplitter;
|
||||
|
@ -74,7 +73,7 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
|
||||
@Override
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extSourceCollectionName = message.getStr("collection");
|
||||
String splitKey = message.getStr("split.key");
|
||||
String extTargetCollectionName = message.getStr("target.collection");
|
||||
|
@ -130,14 +129,14 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
for (Slice sourceSlice : sourceSlices) {
|
||||
for (Slice targetSlice : targetSlices) {
|
||||
log.info("Migrating source shard: {} to target shard: {} for split.key = {}", sourceSlice, targetSlice, splitKey);
|
||||
migrateKey(clusterState, cloudConfig, sourceCollection, sourceSlice, targetCollection, targetSlice, splitKey,
|
||||
migrateKey(clusterState, sourceCollection, sourceSlice, targetCollection, targetSlice, splitKey,
|
||||
timeout, results, asyncId, message);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
private void migrateKey(ClusterState clusterState, CloudConfig cloudConfig, DocCollection sourceCollection, Slice sourceSlice,
|
||||
private void migrateKey(ClusterState clusterState, DocCollection sourceCollection, Slice sourceSlice,
|
||||
DocCollection targetCollection, Slice targetSlice,
|
||||
String splitKey, int timeout,
|
||||
@SuppressWarnings({"rawtypes"})NamedList results, String asyncId, ZkNodeProps message) throws Exception {
|
||||
|
@ -150,7 +149,7 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
NAME, tempSourceCollectionName);
|
||||
|
||||
try {
|
||||
ocmh.commandMap.get(DELETE).call(zkStateReader.getClusterState(), cloudConfig, new ZkNodeProps(props), results);
|
||||
ocmh.commandMap.get(DELETE).call(zkStateReader.getClusterState(), new ZkNodeProps(props), results);
|
||||
clusterState = zkStateReader.getClusterState();
|
||||
} catch (Exception e) {
|
||||
log.warn("Unable to clean up existing temporary collection: {}", tempSourceCollectionName, e);
|
||||
|
@ -245,7 +244,7 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
log.info("Creating temporary collection: {}", props);
|
||||
ocmh.commandMap.get(CREATE).call(clusterState, cloudConfig, new ZkNodeProps(props), results);
|
||||
ocmh.commandMap.get(CREATE).call(clusterState, new ZkNodeProps(props), results);
|
||||
// refresh cluster state
|
||||
clusterState = zkStateReader.getClusterState();
|
||||
Slice tempSourceSlice = clusterState.getCollection(tempSourceCollectionName).getSlices().iterator().next();
|
||||
|
@ -313,7 +312,7 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
if (asyncId != null) {
|
||||
props.put(ASYNC, asyncId);
|
||||
}
|
||||
((AddReplicaCmd)ocmh.commandMap.get(ADDREPLICA)).addReplica(clusterState, cloudConfig, new ZkNodeProps(props), results, null);
|
||||
((AddReplicaCmd)ocmh.commandMap.get(ADDREPLICA)).addReplica(clusterState, new ZkNodeProps(props), results, null);
|
||||
|
||||
{
|
||||
final ShardRequestTracker syncRequestTracker = ocmh.syncRequestTracker();
|
||||
|
@ -373,7 +372,7 @@ public class MigrateCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
props = makeMap(
|
||||
Overseer.QUEUE_OPERATION, DELETE.toLower(),
|
||||
NAME, tempSourceCollectionName);
|
||||
ocmh.commandMap.get(DELETE). call(zkStateReader.getClusterState(), cloudConfig, new ZkNodeProps(props), results);
|
||||
ocmh.commandMap.get(DELETE). call(zkStateReader.getClusterState(), new ZkNodeProps(props), results);
|
||||
} catch (Exception e) {
|
||||
log.error("Unable to delete temporary collection: {}. Please remove it manually", tempSourceCollectionName, e);
|
||||
}
|
||||
|
|
|
@ -39,7 +39,6 @@ import org.apache.solr.common.params.CoreAdminParams;
|
|||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.update.UpdateLog;
|
||||
import org.apache.solr.util.TimeOut;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -67,11 +66,11 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
moveReplica(ocmh.zkStateReader.getClusterState(), cloudConfig, message, results);
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
moveReplica(ocmh.zkStateReader.getClusterState(), message, results);
|
||||
}
|
||||
|
||||
private void moveReplica(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
private void moveReplica(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
if (log.isDebugEnabled()) {
|
||||
log.debug("moveReplica() : {}", Utils.toJSONString(message));
|
||||
}
|
||||
|
@ -160,15 +159,15 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
if (isSharedFS && inPlaceMove) {
|
||||
log.debug("-- moveHdfsReplica");
|
||||
moveHdfsReplica(clusterState, cloudConfig, results, dataDir.toString(), targetNode, async, coll, replica, slice, timeout, waitForFinalState);
|
||||
moveHdfsReplica(clusterState, results, dataDir.toString(), targetNode, async, coll, replica, slice, timeout, waitForFinalState);
|
||||
} else {
|
||||
log.debug("-- moveNormalReplica (inPlaceMove={}, isSharedFS={}", inPlaceMove, isSharedFS);
|
||||
moveNormalReplica(clusterState, cloudConfig, results, targetNode, async, coll, replica, slice, timeout, waitForFinalState);
|
||||
moveNormalReplica(clusterState, results, targetNode, async, coll, replica, slice, timeout, waitForFinalState);
|
||||
}
|
||||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
private void moveHdfsReplica(ClusterState clusterState, CloudConfig cloudConfig, @SuppressWarnings({"rawtypes"})NamedList results, String dataDir, String targetNode, String async,
|
||||
private void moveHdfsReplica(ClusterState clusterState, @SuppressWarnings({"rawtypes"})NamedList results, String dataDir, String targetNode, String async,
|
||||
DocCollection coll, Replica replica, Slice slice, int timeout, boolean waitForFinalState) throws Exception {
|
||||
String skipCreateReplicaInClusterState = "true";
|
||||
if (clusterState.getLiveNodes().contains(replica.getNodeName())) {
|
||||
|
@ -230,7 +229,7 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
@SuppressWarnings({"rawtypes"})
|
||||
NamedList addResult = new NamedList();
|
||||
try {
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), cloudConfig, addReplicasProps, addResult, null);
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), addReplicasProps, addResult, null);
|
||||
} catch (Exception e) {
|
||||
// fatal error - try rolling back
|
||||
String errorString = String.format(Locale.ROOT, "Failed to create replica for collection=%s shard=%s" +
|
||||
|
@ -240,7 +239,7 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
addReplicasProps = addReplicasProps.plus(CoreAdminParams.NODE, replica.getNodeName());
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
NamedList rollback = new NamedList();
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), cloudConfig, addReplicasProps, rollback, null);
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), addReplicasProps, rollback, null);
|
||||
if (rollback.get("failure") != null) {
|
||||
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR, "Fatal error during MOVEREPLICA of " + replica
|
||||
+ ", collection may be inconsistent: " + rollback.get("failure"));
|
||||
|
@ -258,7 +257,7 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
@SuppressWarnings({"rawtypes"})
|
||||
NamedList rollback = new NamedList();
|
||||
try {
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), cloudConfig, addReplicasProps, rollback, null);
|
||||
ocmh.addReplica(ocmh.zkStateReader.getClusterState(), addReplicasProps, rollback, null);
|
||||
} catch (Exception e) {
|
||||
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR, "Fatal error during MOVEREPLICA of " + replica
|
||||
+ ", collection may be inconsistent!", e);
|
||||
|
@ -276,7 +275,7 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
private void moveNormalReplica(ClusterState clusterState, CloudConfig cloudConfig, @SuppressWarnings({"rawtypes"})NamedList results, String targetNode, String async,
|
||||
private void moveNormalReplica(ClusterState clusterState, @SuppressWarnings({"rawtypes"})NamedList results, String targetNode, String async,
|
||||
DocCollection coll, Replica replica, Slice slice, int timeout, boolean waitForFinalState) throws Exception {
|
||||
String newCoreName = Assign.buildSolrCoreName(ocmh.overseer.getSolrCloudManager().getDistribStateManager(), coll, slice.getName(), replica.getType());
|
||||
ZkNodeProps addReplicasProps = new ZkNodeProps(
|
||||
|
@ -291,7 +290,7 @@ public class MoveReplicaCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
NamedList addResult = new NamedList();
|
||||
SolrCloseableLatch countDownLatch = new SolrCloseableLatch(1, ocmh);
|
||||
ActiveReplicaWatcher watcher = null;
|
||||
ZkNodeProps props = ocmh.addReplica(clusterState, cloudConfig, addReplicasProps, addResult, null).get(0);
|
||||
ZkNodeProps props = ocmh.addReplica(clusterState, addReplicasProps, addResult, null).get(0);
|
||||
log.debug("props {}", props);
|
||||
if (replica.equals(slice.getLeader()) || waitForFinalState) {
|
||||
watcher = new ActiveReplicaWatcher(coll.getName(), null, Collections.singletonList(newCoreName), countDownLatch);
|
||||
|
|
|
@ -78,7 +78,6 @@ import org.apache.solr.common.util.StrUtils;
|
|||
import org.apache.solr.common.util.SuppressForbidden;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.handler.component.HttpShardHandlerFactory;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.handler.component.ShardRequest;
|
||||
|
@ -243,7 +242,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
|
||||
@Override
|
||||
@SuppressWarnings("unchecked")
|
||||
public OverseerSolrResponse processMessage(ZkNodeProps message, CloudConfig cloudConfig, String operation) {
|
||||
public OverseerSolrResponse processMessage(ZkNodeProps message, String operation) {
|
||||
MDCLoggingContext.setCollection(message.getStr(COLLECTION));
|
||||
MDCLoggingContext.setShard(message.getStr(SHARD_ID_PROP));
|
||||
MDCLoggingContext.setReplica(message.getStr(REPLICA_PROP));
|
||||
|
@ -255,7 +254,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
CollectionAction action = getCollectionAction(operation);
|
||||
Cmd command = commandMap.get(action);
|
||||
if (command != null) {
|
||||
command.call(cloudManager.getClusterStateProvider().getClusterState(), cloudConfig, message, results);
|
||||
command.call(cloudManager.getClusterStateProvider().getClusterState(), message, results);
|
||||
} else {
|
||||
throw new SolrException(ErrorCode.BAD_REQUEST, "Unknown operation:"
|
||||
+ operation);
|
||||
|
@ -282,7 +281,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
|
||||
@SuppressForbidden(reason = "Needs currentTimeMillis for mock requests")
|
||||
@SuppressWarnings({"unchecked"})
|
||||
private void mockOperation(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws InterruptedException {
|
||||
private void mockOperation(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws InterruptedException {
|
||||
//only for test purposes
|
||||
Thread.sleep(message.getInt("sleep", 1));
|
||||
if (log.isInfoEnabled()) {
|
||||
|
@ -300,7 +299,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
|
||||
@SuppressWarnings({"unchecked"})
|
||||
private void reloadCollection(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) {
|
||||
private void reloadCollection(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) {
|
||||
ModifiableSolrParams params = new ModifiableSolrParams();
|
||||
params.set(CoreAdminParams.ACTION, CoreAdminAction.RELOAD.toString());
|
||||
|
||||
|
@ -309,7 +308,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
|
||||
@SuppressWarnings("unchecked")
|
||||
private void processRebalanceLeaders(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
private void processRebalanceLeaders(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
throws Exception {
|
||||
checkRequired(message, COLLECTION_PROP, SHARD_ID_PROP, CORE_NAME_PROP, ELECTION_NODE_PROP,
|
||||
CORE_NODE_NAME_PROP, BASE_URL_PROP, REJOIN_AT_HEAD_PROP);
|
||||
|
@ -338,7 +337,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
|
||||
@SuppressWarnings("unchecked")
|
||||
private void processReplicaAddPropertyCommand(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
private void processReplicaAddPropertyCommand(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
throws Exception {
|
||||
checkRequired(message, COLLECTION_PROP, SHARD_ID_PROP, REPLICA_PROP, PROPERTY_PROP, PROPERTY_VALUE_PROP);
|
||||
SolrZkClient zkClient = zkStateReader.getZkClient();
|
||||
|
@ -349,7 +348,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
overseer.offerStateUpdate(Utils.toJSON(m));
|
||||
}
|
||||
|
||||
private void processReplicaDeletePropertyCommand(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
private void processReplicaDeletePropertyCommand(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
throws Exception {
|
||||
checkRequired(message, COLLECTION_PROP, SHARD_ID_PROP, REPLICA_PROP, PROPERTY_PROP);
|
||||
SolrZkClient zkClient = zkStateReader.getZkClient();
|
||||
|
@ -360,7 +359,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
overseer.offerStateUpdate(Utils.toJSON(m));
|
||||
}
|
||||
|
||||
private void balanceProperty(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
private void balanceProperty(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
if (StringUtils.isBlank(message.getStr(COLLECTION_PROP)) || StringUtils.isBlank(message.getStr(PROPERTY_PROP))) {
|
||||
throw new SolrException(ErrorCode.BAD_REQUEST,
|
||||
"The '" + COLLECTION_PROP + "' and '" + PROPERTY_PROP +
|
||||
|
@ -584,7 +583,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
|
||||
|
||||
private void modifyCollection(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
private void modifyCollection(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results)
|
||||
throws Exception {
|
||||
|
||||
final String collectionName = message.getStr(ZkStateReader.COLLECTION_PROP);
|
||||
|
@ -596,7 +595,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
validateConfigOrThrowSolrException(configName);
|
||||
|
||||
createConfNode(cloudManager.getDistribStateManager(), configName, collectionName);
|
||||
reloadCollection(null, cloudConfig, new ZkNodeProps(NAME, collectionName), results);
|
||||
reloadCollection(null, new ZkNodeProps(NAME, collectionName), results);
|
||||
}
|
||||
|
||||
overseer.offerStateUpdate(Utils.toJSON(message));
|
||||
|
@ -631,16 +630,16 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
|
||||
// if switching to/from read-only mode reload the collection
|
||||
if (message.keySet().contains(ZkStateReader.READ_ONLY)) {
|
||||
reloadCollection(null, cloudConfig, new ZkNodeProps(NAME, collectionName), results);
|
||||
reloadCollection(null, new ZkNodeProps(NAME, collectionName), results);
|
||||
}
|
||||
}
|
||||
|
||||
void cleanupCollection(CloudConfig cloudConfig, String collectionName, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
void cleanupCollection(String collectionName, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
log.error("Cleaning up collection [{}].", collectionName);
|
||||
Map<String, Object> props = makeMap(
|
||||
Overseer.QUEUE_OPERATION, DELETE.toLower(),
|
||||
NAME, collectionName);
|
||||
commandMap.get(DELETE).call(zkStateReader.getClusterState(), cloudConfig, new ZkNodeProps(props), results);
|
||||
commandMap.get(DELETE).call(zkStateReader.getClusterState(), new ZkNodeProps(props), results);
|
||||
}
|
||||
|
||||
Map<String, Replica> waitToSeeReplicasInState(String collectionName, Collection<String> coreNames) throws InterruptedException {
|
||||
|
@ -674,10 +673,10 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
}
|
||||
|
||||
List<ZkNodeProps> addReplica(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, Runnable onComplete)
|
||||
List<ZkNodeProps> addReplica(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results, Runnable onComplete)
|
||||
throws Exception {
|
||||
|
||||
return ((AddReplicaCmd) commandMap.get(ADDREPLICA)).addReplica(clusterState, cloudConfig, message, results, onComplete);
|
||||
return ((AddReplicaCmd) commandMap.get(ADDREPLICA)).addReplica(clusterState, message, results, onComplete);
|
||||
}
|
||||
|
||||
void validateConfigOrThrowSolrException(String configName) throws IOException, KeeperException, InterruptedException {
|
||||
|
@ -907,7 +906,7 @@ public class OverseerCollectionMessageHandler implements OverseerMessageHandler,
|
|||
}
|
||||
|
||||
protected interface Cmd {
|
||||
void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception;
|
||||
void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception;
|
||||
}
|
||||
|
||||
/*
|
||||
|
|
|
@ -32,7 +32,6 @@ import org.apache.solr.common.cloud.ZkStateReader;
|
|||
import org.apache.solr.common.params.CollectionParams.CollectionAction;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
import org.apache.zookeeper.data.Stat;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -58,7 +57,7 @@ public class OverseerRoleCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked", "rawtypes"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, NamedList results) throws Exception {
|
||||
ZkStateReader zkStateReader = ocmh.zkStateReader;
|
||||
SolrZkClient zkClient = zkStateReader.getZkClient();
|
||||
Map roles = null;
|
||||
|
|
|
@ -31,7 +31,6 @@ import org.apache.solr.common.cloud.ZkNodeProps;
|
|||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.util.stats.MetricUtils;
|
||||
import org.apache.zookeeper.data.Stat;
|
||||
import org.slf4j.Logger;
|
||||
|
@ -47,7 +46,7 @@ public class OverseerStatusCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings("unchecked")
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
ZkStateReader zkStateReader = ocmh.zkStateReader;
|
||||
String leaderNode = OverseerTaskProcessor.getLeaderNode(zkStateReader.getZkClient());
|
||||
results.add("leader", leaderNode);
|
||||
|
|
|
@ -56,7 +56,6 @@ import org.apache.solr.common.params.CoreAdminParams;
|
|||
import org.apache.solr.common.params.ModifiableSolrParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.util.TestInjection;
|
||||
import org.apache.solr.util.TimeOut;
|
||||
import org.apache.zookeeper.CreateMode;
|
||||
|
@ -167,7 +166,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState clusterState, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
|
||||
log.debug("*** called: {}", message);
|
||||
|
||||
|
@ -286,7 +285,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CommonParams.NAME, chkCollection,
|
||||
CoreAdminParams.DELETE_METRICS_HISTORY, "true"
|
||||
);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("deleting old checkpoint collection " + chkCollection, cmdResults, true);
|
||||
}
|
||||
|
||||
|
@ -332,7 +331,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
// create the target collection
|
||||
cmd = new ZkNodeProps(propMap);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, cmd, cmdResults);
|
||||
createdTarget = true;
|
||||
ocmh.checkResults("creating target collection " + targetCollection, cmdResults, true);
|
||||
|
||||
|
@ -346,7 +345,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CommonAdminParams.WAIT_FOR_FINAL_STATE, "true"
|
||||
);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("creating checkpoint collection " + chkCollection, cmdResults, true);
|
||||
// wait for a while until we see both collections
|
||||
TimeOut waitUntil = new TimeOut(30, TimeUnit.SECONDS, ocmh.timeSource);
|
||||
|
@ -434,7 +433,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CommonParams.NAME, extCollection,
|
||||
"collections", targetCollection);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATEALIAS).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.CREATEALIAS).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("setting up alias " + extCollection + " -> " + targetCollection, cmdResults, true);
|
||||
reindexingState.put("alias", extCollection + " -> " + targetCollection);
|
||||
}
|
||||
|
@ -457,7 +456,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CoreAdminParams.DELETE_METRICS_HISTORY, "true"
|
||||
);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("deleting checkpoint collection " + chkCollection, cmdResults, true);
|
||||
|
||||
// 7. optionally delete the source collection
|
||||
|
@ -470,7 +469,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CoreAdminParams.DELETE_METRICS_HISTORY, "true"
|
||||
);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("deleting source collection " + collection, cmdResults, true);
|
||||
} else {
|
||||
// 8. clear readOnly on source
|
||||
|
@ -496,7 +495,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
aborted = true;
|
||||
} finally {
|
||||
if (aborted) {
|
||||
cleanup(cloudConfig, collection, targetCollection, chkCollection, daemonUrl, targetCollection, createdTarget);
|
||||
cleanup(collection, targetCollection, chkCollection, daemonUrl, targetCollection, createdTarget);
|
||||
if (exc != null) {
|
||||
results.add("error", exc.toString());
|
||||
}
|
||||
|
@ -763,7 +762,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
}
|
||||
}
|
||||
|
||||
private void cleanup(CloudConfig cloudConfig, String collection, String targetCollection, String chkCollection,
|
||||
private void cleanup(String collection, String targetCollection, String chkCollection,
|
||||
String daemonUrl, String daemonName, boolean createdTarget) throws Exception {
|
||||
log.info("## Cleaning up after abort or error");
|
||||
// 1. kill the daemon
|
||||
|
@ -783,7 +782,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
FOLLOW_ALIASES, "false",
|
||||
CoreAdminParams.DELETE_METRICS_HISTORY, "true"
|
||||
);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("CLEANUP: deleting target collection " + targetCollection, cmdResults, false);
|
||||
|
||||
}
|
||||
|
@ -797,7 +796,7 @@ public class ReindexCollectionCmd implements OverseerCollectionMessageHandler.Cm
|
|||
CoreAdminParams.DELETE_METRICS_HISTORY, "true"
|
||||
);
|
||||
cmdResults = new NamedList<>();
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cloudConfig, cmd, cmdResults);
|
||||
ocmh.commandMap.get(CollectionParams.CollectionAction.DELETE).call(clusterState, cmd, cmdResults);
|
||||
ocmh.checkResults("CLEANUP: deleting checkpoint collection " + chkCollection, cmdResults, false);
|
||||
}
|
||||
log.debug(" -- turning readOnly mode off for {}", collection);
|
||||
|
|
|
@ -26,7 +26,6 @@ import org.apache.solr.common.cloud.ZkNodeProps;
|
|||
import org.apache.solr.common.params.CollectionAdminParams;
|
||||
import org.apache.solr.common.params.CoreAdminParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
@ -45,7 +44,7 @@ public class RenameCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String extCollectionName = message.getStr(CoreAdminParams.NAME);
|
||||
String target = message.getStr(CollectionAdminParams.TARGET);
|
||||
|
||||
|
|
|
@ -41,7 +41,6 @@ import org.apache.solr.common.params.CollectionParams;
|
|||
import org.apache.solr.common.params.CommonAdminParams;
|
||||
import org.apache.solr.common.params.CoreAdminParams;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.zookeeper.KeeperException;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
@ -61,7 +60,7 @@ public class ReplaceNodeCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
ZkStateReader zkStateReader = ocmh.zkStateReader;
|
||||
String source = message.getStr(CollectionParams.SOURCE_NODE, message.getStr("source"));
|
||||
String target = message.getStr(CollectionParams.TARGET_NODE, message.getStr("target"));
|
||||
|
@ -122,12 +121,12 @@ public class ReplaceNodeCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
.onNodes(new ArrayList<>(ocmh.cloudManager.getClusterStateProvider().getLiveNodes()))
|
||||
.build();
|
||||
Assign.AssignStrategyFactory assignStrategyFactory = new Assign.AssignStrategyFactory(ocmh.cloudManager);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, cloudConfig, clusterState.getCollection(sourceCollection));
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, clusterState.getCollection(sourceCollection));
|
||||
targetNode = assignStrategy.assign(ocmh.cloudManager, assignRequest).get(0).node;
|
||||
}
|
||||
ZkNodeProps msg = sourceReplica.plus("parallel", String.valueOf(parallel)).plus(CoreAdminParams.NODE, targetNode);
|
||||
if (async != null) msg.getProperties().put(ASYNC, async);
|
||||
final ZkNodeProps addedReplica = ocmh.addReplica(clusterState, cloudConfig,
|
||||
final ZkNodeProps addedReplica = ocmh.addReplica(clusterState,
|
||||
msg, nl, () -> {
|
||||
countDownLatch.countDown();
|
||||
if (nl.get("failure") != null) {
|
||||
|
|
|
@ -55,7 +55,6 @@ import org.apache.solr.common.util.NamedList;
|
|||
import org.apache.solr.common.util.SimpleOrderedMap;
|
||||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.core.backup.BackupManager;
|
||||
import org.apache.solr.core.backup.repository.BackupRepository;
|
||||
|
@ -86,7 +85,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@Override
|
||||
@SuppressWarnings({"unchecked", "rawtypes"})
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, NamedList results) throws Exception {
|
||||
// TODO maybe we can inherit createCollection's options/code
|
||||
|
||||
String restoreCollectionName = message.getStr(COLLECTION_PROP);
|
||||
|
@ -195,7 +194,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
propMap.put(OverseerCollectionMessageHandler.SHARDS_PROP, newSlices);
|
||||
}
|
||||
|
||||
ocmh.commandMap.get(CREATE).call(zkStateReader.getClusterState(), cloudConfig, new ZkNodeProps(propMap), new NamedList());
|
||||
ocmh.commandMap.get(CREATE).call(zkStateReader.getClusterState(), new ZkNodeProps(propMap), new NamedList());
|
||||
// note: when createCollection() returns, the collection exists (no race)
|
||||
}
|
||||
|
||||
|
@ -232,7 +231,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
.onNodes(nodeList)
|
||||
.build();
|
||||
Assign.AssignStrategyFactory assignStrategyFactory = new Assign.AssignStrategyFactory(ocmh.cloudManager);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, cloudConfig, restoreCollection);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, restoreCollection);
|
||||
List<ReplicaPosition> replicaPositions = assignStrategy.assign(ocmh.cloudManager, assignRequest);
|
||||
|
||||
CountDownLatch countDownLatch = new CountDownLatch(restoreCollection.getSlices().size());
|
||||
|
@ -273,7 +272,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
ocmh.addPropertyParams(message, propMap);
|
||||
final NamedList addReplicaResult = new NamedList();
|
||||
ocmh.addReplica(clusterState, cloudConfig, new ZkNodeProps(propMap), addReplicaResult, () -> {
|
||||
ocmh.addReplica(clusterState, new ZkNodeProps(propMap), addReplicaResult, () -> {
|
||||
Object addResultFailure = addReplicaResult.get("failure");
|
||||
if (addResultFailure != null) {
|
||||
SimpleOrderedMap failure = (SimpleOrderedMap) results.get("failure");
|
||||
|
@ -301,7 +300,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
Object failures = results.get("failure");
|
||||
if (failures != null && ((SimpleOrderedMap) failures).size() > 0) {
|
||||
log.error("Restore failed to create initial replicas.");
|
||||
ocmh.cleanupCollection(cloudConfig, restoreCollectionName, new NamedList<Object>());
|
||||
ocmh.cleanupCollection(restoreCollectionName, new NamedList<Object>());
|
||||
return;
|
||||
}
|
||||
|
||||
|
@ -414,7 +413,7 @@ public class RestoreCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
ocmh.addPropertyParams(message, propMap);
|
||||
|
||||
ocmh.addReplica(zkStateReader.getClusterState(), cloudConfig, new ZkNodeProps(propMap), results, null);
|
||||
ocmh.addReplica(zkStateReader.getClusterState(), new ZkNodeProps(propMap), results, null);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -27,7 +27,6 @@ import org.apache.solr.common.cloud.ClusterState;
|
|||
import org.apache.solr.common.cloud.ZkNodeProps;
|
||||
import org.apache.solr.common.cloud.ZkStateReader;
|
||||
import org.apache.solr.common.util.NamedList;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
@ -48,7 +47,7 @@ public class SetAliasPropCmd implements Cmd {
|
|||
}
|
||||
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
String aliasName = message.getStr(NAME);
|
||||
|
||||
final ZkStateReader.AliasesManager aliasesManager = messageHandler.zkStateReader.aliasesManager;
|
||||
|
|
|
@ -37,7 +37,6 @@ import org.apache.solr.common.params.ModifiableSolrParams;
|
|||
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.CloudConfig;
|
||||
import org.apache.solr.handler.component.ShardHandler;
|
||||
import org.apache.solr.update.SolrIndexSplitter;
|
||||
import org.apache.solr.util.RTimerTree;
|
||||
|
@ -76,12 +75,12 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
|
||||
@SuppressWarnings("unchecked")
|
||||
@Override
|
||||
public void call(ClusterState state, CloudConfig cloudConfig, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
split(state, cloudConfig, message,(NamedList<Object>) results);
|
||||
public void call(ClusterState state, ZkNodeProps message, @SuppressWarnings({"rawtypes"})NamedList results) throws Exception {
|
||||
split(state, message,(NamedList<Object>) results);
|
||||
}
|
||||
|
||||
@SuppressWarnings({"rawtypes"})
|
||||
public boolean split(ClusterState clusterState, CloudConfig cloudConfig, ZkNodeProps message, NamedList<Object> results) throws Exception {
|
||||
public boolean split(ClusterState clusterState, ZkNodeProps message, NamedList<Object> results) throws Exception {
|
||||
final String asyncId = message.getStr(ASYNC);
|
||||
|
||||
boolean waitForFinalState = message.getBool(CommonAdminParams.WAIT_FOR_FINAL_STATE, false);
|
||||
|
@ -247,7 +246,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
propMap.put(SHARD_ID_PROP, subSlice);
|
||||
ZkNodeProps m = new ZkNodeProps(propMap);
|
||||
try {
|
||||
ocmh.commandMap.get(DELETESHARD).call(clusterState, cloudConfig, m, new NamedList());
|
||||
ocmh.commandMap.get(DELETESHARD).call(clusterState, m, new NamedList());
|
||||
} catch (Exception e) {
|
||||
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR, "Unable to delete already existing sub shard: " + subSlice,
|
||||
e);
|
||||
|
@ -310,7 +309,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
if (asyncId != null) {
|
||||
propMap.put(ASYNC, asyncId);
|
||||
}
|
||||
ocmh.addReplica(clusterState, cloudConfig, new ZkNodeProps(propMap), results, null);
|
||||
ocmh.addReplica(clusterState, new ZkNodeProps(propMap), results, null);
|
||||
}
|
||||
|
||||
|
||||
|
@ -433,7 +432,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
.onNodes(new ArrayList<>(clusterState.getLiveNodes()))
|
||||
.build();
|
||||
Assign.AssignStrategyFactory assignStrategyFactory = new Assign.AssignStrategyFactory(ocmh.cloudManager);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, cloudConfig, collection);
|
||||
Assign.AssignStrategy assignStrategy = assignStrategyFactory.create(clusterState, collection);
|
||||
List<ReplicaPosition> replicaPositions = assignStrategy.assign(ocmh.cloudManager, assignRequest);
|
||||
t.stop();
|
||||
|
||||
|
@ -551,7 +550,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
t = timings.sub("createCoresForReplicas");
|
||||
// now actually create replica cores on sub shard nodes
|
||||
for (Map<String, Object> replica : replicas) {
|
||||
ocmh.addReplica(clusterState, cloudConfig, new ZkNodeProps(replica), results, null);
|
||||
ocmh.addReplica(clusterState, new ZkNodeProps(replica), results, null);
|
||||
}
|
||||
|
||||
assert TestInjection.injectSplitFailureAfterReplicaCreation();
|
||||
|
@ -588,7 +587,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR, null, e);
|
||||
} finally {
|
||||
if (!success) {
|
||||
cleanupAfterFailure(zkStateReader, cloudConfig, collectionName, parentSlice.getName(), subSlices, offlineSlices);
|
||||
cleanupAfterFailure(zkStateReader, collectionName, parentSlice.getName(), subSlices, offlineSlices);
|
||||
unlockForSplit(ocmh.cloudManager, collectionName, parentSlice.getName());
|
||||
}
|
||||
}
|
||||
|
@ -645,7 +644,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
}
|
||||
}
|
||||
|
||||
private void cleanupAfterFailure(ZkStateReader zkStateReader, CloudConfig cloudConfig, String collectionName, String parentShard,
|
||||
private void cleanupAfterFailure(ZkStateReader zkStateReader, String collectionName, String parentShard,
|
||||
List<String> subSlices, Set<String> offlineSlices) {
|
||||
log.info("Cleaning up after a failed split of {}/{}", collectionName, parentShard);
|
||||
// get the latest state
|
||||
|
@ -726,7 +725,7 @@ public class SplitShardCmd implements OverseerCollectionMessageHandler.Cmd {
|
|||
props.put(SHARD_ID_PROP, subSlice);
|
||||
ZkNodeProps m = new ZkNodeProps(props);
|
||||
try {
|
||||
ocmh.commandMap.get(DELETESHARD).call(clusterState, cloudConfig, m, new NamedList<Object>());
|
||||
ocmh.commandMap.get(DELETESHARD).call(clusterState, m, new NamedList<Object>());
|
||||
} catch (Exception e) {
|
||||
log.warn("Cleanup failed after failed split of {}/{} : (deleting existing sub shard{})", collectionName, parentShard, subSlice, e);
|
||||
}
|
||||
|
|
|
@ -59,7 +59,6 @@ import org.apache.solr.common.util.ObjectCache;
|
|||
import org.apache.solr.common.util.StrUtils;
|
||||
import org.apache.solr.common.util.TimeSource;
|
||||
import org.apache.solr.common.util.Utils;
|
||||
import org.apache.solr.core.CloudConfig;
|
||||
import org.apache.solr.core.CoreContainer;
|
||||
import org.apache.solr.handler.component.HttpShardHandler;
|
||||
import org.apache.solr.handler.component.HttpShardHandlerFactory;
|
||||
|
@ -114,7 +113,6 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
|
|||
private static HttpShardHandlerFactory shardHandlerFactoryMock;
|
||||
private static HttpShardHandler shardHandlerMock;
|
||||
private static ZkStateReader zkStateReaderMock;
|
||||
private static CloudConfig cloudConfigMock;
|
||||
private static ClusterState clusterStateMock;
|
||||
private static SolrZkClient solrZkClientMock;
|
||||
private static DistribStateManager stateManagerMock;
|
||||
|
@ -140,14 +138,14 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
|
|||
OverseerCollectionConfigSetProcessor {
|
||||
|
||||
|
||||
public OverseerCollectionConfigSetProcessorToBeTested(CloudConfig cloudConfig, ZkStateReader zkStateReader,
|
||||
public OverseerCollectionConfigSetProcessorToBeTested(ZkStateReader zkStateReader,
|
||||
String myId, HttpShardHandlerFactory shardHandlerFactory,
|
||||
String adminPath,
|
||||
OverseerTaskQueue workQueue, DistributedMap runningMap,
|
||||
Overseer overseer,
|
||||
DistributedMap completedMap,
|
||||
DistributedMap failureMap) {
|
||||
super(cloudConfig, zkStateReader, myId, shardHandlerFactory, adminPath, new Stats(), overseer, new OverseerNodePrioritizer(zkStateReader, overseer.getStateUpdateQueue(), adminPath, shardHandlerFactory), workQueue, runningMap, completedMap, failureMap);
|
||||
super(zkStateReader, myId, shardHandlerFactory, adminPath, new Stats(), overseer, new OverseerNodePrioritizer(zkStateReader, overseer.getStateUpdateQueue(), adminPath, shardHandlerFactory), workQueue, runningMap, completedMap, failureMap);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -169,7 +167,6 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
|
|||
shardHandlerFactoryMock = mock(HttpShardHandlerFactory.class);
|
||||
shardHandlerMock = mock(HttpShardHandler.class);
|
||||
zkStateReaderMock = mock(ZkStateReader.class);
|
||||
cloudConfigMock = mock(CloudConfig.class);
|
||||
clusterStateMock = mock(ClusterState.class);
|
||||
solrZkClientMock = mock(SolrZkClient.class);
|
||||
overseerMock = mock(Overseer.class);
|
||||
|
@ -729,7 +726,7 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
|
|||
|
||||
if (random().nextBoolean()) Collections.shuffle(createNodeList, random());
|
||||
|
||||
underTest = new OverseerCollectionConfigSetProcessorToBeTested(cloudConfigMock, zkStateReaderMock,
|
||||
underTest = new OverseerCollectionConfigSetProcessorToBeTested(zkStateReaderMock,
|
||||
"1234", shardHandlerFactoryMock, ADMIN_PATH, workQueueMock, runningMapMock,
|
||||
overseerMock, completedMapMock, failureMapMock);
|
||||
|
||||
|
|
Loading…
Reference in New Issue