SOLR-9834: A variety of spots in the code can create a collection zk node after the collection has been removed.

This commit is contained in:
markrmiller 2016-12-08 12:03:55 -05:00
parent b97d9d7478
commit 1055209940
15 changed files with 308 additions and 170 deletions

View File

@ -230,6 +230,9 @@ Bug Fixes
* SOLR-9832: Schema modifications are not immediately visible on the coordinating node. (Steve Rowe)
* SOLR-9834: A variety of spots in the code can create a collection zk node after the collection has been
removed. (Mark Miller)
Other Changes
----------------------

View File

@ -16,12 +16,20 @@
*/
package org.apache.solr.cloud;
import java.lang.invoke.MethodHandles;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.ZkStateReader;
import org.apache.solr.core.ConfigSetService;
import org.apache.solr.core.CoreDescriptor;
import org.apache.solr.core.SolrResourceLoader;
import org.apache.zookeeper.KeeperException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
public class CloudConfigSetService extends ConfigSetService {
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
private final ZkController zkController;
public CloudConfigSetService(SolrResourceLoader loader, ZkController zkController) {
@ -31,8 +39,18 @@ public class CloudConfigSetService extends ConfigSetService {
@Override
public SolrResourceLoader createCoreResourceLoader(CoreDescriptor cd) {
// TODO: Shouldn't the collection node be created by the Collections API?
zkController.createCollectionZkNode(cd.getCloudDescriptor());
try {
// for back compat with cores that can create collections without the collections API
if (!zkController.getZkClient().exists(ZkStateReader.COLLECTIONS_ZKNODE + "/" + cd.getCollectionName(), true)) {
CreateCollectionCmd.createCollectionZkNode(zkController.getZkClient(), cd.getCollectionName(), cd.getCloudDescriptor().getParams());
}
} catch (KeeperException e) {
SolrException.log(log, null, e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
SolrException.log(log, null, e);
}
String configName = zkController.getZkStateReader().readConfigName(cd.getCollectionName());
return new ZkSolrResourceLoader(cd.getInstanceDir(), configName, parentLoader.getClassLoader(),
cd.getSubstitutableProperties(), zkController);

View File

@ -33,7 +33,7 @@ public class CloudDescriptor {
private String roles = null;
private Integer numShards;
private String nodeName = null;
private Map<String, String> collectionParams = new HashMap<>();
private Map<String,String> collectionParams = new HashMap<>();
private volatile boolean isLeader = false;

View File

@ -25,19 +25,23 @@ import java.util.HashMap;
import java.util.LinkedHashMap;
import java.util.List;
import java.util.Map;
import java.util.Properties;
import java.util.concurrent.TimeUnit;
import org.apache.solr.cloud.OverseerCollectionMessageHandler.Cmd;
import org.apache.solr.cloud.overseer.ClusterStateMutator;
import org.apache.solr.cloud.rule.ReplicaAssigner;
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.DocRouter;
import org.apache.solr.common.cloud.ImplicitDocRouter;
import org.apache.solr.common.cloud.Replica;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.cloud.ZkConfigManager;
import org.apache.solr.common.cloud.ZkNodeProps;
import org.apache.solr.common.cloud.ZkStateReader;
import org.apache.solr.common.cloud.ZooKeeperException;
import org.apache.solr.common.params.CoreAdminParams;
import org.apache.solr.common.params.ModifiableSolrParams;
import org.apache.solr.common.util.NamedList;
@ -46,7 +50,9 @@ import org.apache.solr.common.util.Utils;
import org.apache.solr.handler.component.ShardHandler;
import org.apache.solr.handler.component.ShardRequest;
import org.apache.solr.util.TimeOut;
import org.apache.zookeeper.CreateMode;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.KeeperException.NoNodeException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -64,9 +70,11 @@ import static org.apache.solr.common.util.StrUtils.formatString;
public class CreateCollectionCmd implements Cmd {
private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());
private final OverseerCollectionMessageHandler ocmh;
private SolrZkClient zkClient;
public CreateCollectionCmd(OverseerCollectionMessageHandler ocmh) {
this.ocmh = ocmh;
this.zkClient = ocmh.zkStateReader.getZkClient();
}
@Override
@ -84,7 +92,6 @@ public class CreateCollectionCmd implements Cmd {
ocmh.validateConfigOrThrowSolrException(configName);
try {
// look at the replication factor and see if it matches reality
// if it does not, find best nodes to create more cores
@ -157,10 +164,20 @@ public class CreateCollectionCmd implements Cmd {
}
ZkStateReader zkStateReader = ocmh.zkStateReader;
boolean isLegacyCloud = Overseer.isLegacy(zkStateReader);
boolean isLegacyCloud = Overseer.isLegacy(zkStateReader);
ocmh.createConfNode(configName, collectionName, isLegacyCloud);
Map<String,String> collectionParams = new HashMap<>();
Map<String,Object> collectionProps = message.getProperties();
for (String propName : collectionProps.keySet()) {
if (propName.startsWith(ZkController.COLLECTION_PARAM_PREFIX)) {
collectionParams.put(propName.substring(ZkController.COLLECTION_PARAM_PREFIX.length()), (String) collectionProps.get(propName));
}
}
createCollectionZkNode(zkClient, collectionName, collectionParams);
Overseer.getStateUpdateQueue(zkStateReader.getZkClient()).offer(Utils.toJSON(message));
// wait for a while until we don't see the collection
@ -288,4 +305,129 @@ public class CreateCollectionCmd implements Cmd {
}
return configName;
}
public static void createCollectionZkNode(SolrZkClient zkClient, String collection, Map<String,String> params) {
log.debug("Check for collection zkNode:" + collection);
String collectionPath = ZkStateReader.COLLECTIONS_ZKNODE + "/" + collection;
try {
if (!zkClient.exists(collectionPath, true)) {
log.debug("Creating collection in ZooKeeper:" + collection);
try {
Map<String,Object> collectionProps = new HashMap<>();
// TODO: if collection.configName isn't set, and there isn't already a conf in zk, just use that?
String defaultConfigName = System.getProperty(ZkController.COLLECTION_PARAM_PREFIX + ZkController.CONFIGNAME_PROP, collection);
if (params.size() > 0) {
collectionProps.putAll(params);
// if the config name wasn't passed in, use the default
if (!collectionProps.containsKey(ZkController.CONFIGNAME_PROP)) {
// users can create the collection node and conf link ahead of time, or this may return another option
getConfName(zkClient, collection, collectionPath, collectionProps);
}
} else if (System.getProperty("bootstrap_confdir") != null) {
// if we are bootstrapping a collection, default the config for
// a new collection to the collection we are bootstrapping
log.info("Setting config for collection:" + collection + " to " + defaultConfigName);
Properties sysProps = System.getProperties();
for (String sprop : System.getProperties().stringPropertyNames()) {
if (sprop.startsWith(ZkController.COLLECTION_PARAM_PREFIX)) {
collectionProps.put(sprop.substring(ZkController.COLLECTION_PARAM_PREFIX.length()), sysProps.getProperty(sprop));
}
}
// if the config name wasn't passed in, use the default
if (!collectionProps.containsKey(ZkController.CONFIGNAME_PROP))
collectionProps.put(ZkController.CONFIGNAME_PROP, defaultConfigName);
} else if (Boolean.getBoolean("bootstrap_conf")) {
// the conf name should should be the collection name of this core
collectionProps.put(ZkController.CONFIGNAME_PROP, collection);
} else {
getConfName(zkClient, collection, collectionPath, collectionProps);
}
collectionProps.remove(ZkStateReader.NUM_SHARDS_PROP); // we don't put numShards in the collections properties
ZkNodeProps zkProps = new ZkNodeProps(collectionProps);
zkClient.makePath(collectionPath, Utils.toJSON(zkProps), CreateMode.PERSISTENT, null, true);
} catch (KeeperException e) {
// it's okay if the node already exists
if (e.code() != KeeperException.Code.NODEEXISTS) {
throw e;
}
}
} else {
log.debug("Collection zkNode exists");
}
} catch (KeeperException e) {
// it's okay if another beats us creating the node
if (e.code() == KeeperException.Code.NODEEXISTS) {
return;
}
throw new SolrException(ErrorCode.SERVER_ERROR, "Error creating collection node in Zookeeper", e);
} catch (InterruptedException e) {
Thread.interrupted();
throw new SolrException(ErrorCode.SERVER_ERROR, "Error creating collection node in Zookeeper", e);
}
}
private static void getConfName(SolrZkClient zkClient, String collection, String collectionPath, Map<String,Object> collectionProps) throws KeeperException,
InterruptedException {
// check for configName
log.debug("Looking for collection configName");
if (collectionProps.containsKey("configName")) {
log.info("configName was passed as a param {}", collectionProps.get("configName"));
return;
}
List<String> configNames = null;
int retry = 1;
int retryLimt = 6;
for (; retry < retryLimt; retry++) {
if (zkClient.exists(collectionPath, true)) {
ZkNodeProps cProps = ZkNodeProps.load(zkClient.getData(collectionPath, null, null, true));
if (cProps.containsKey(ZkController.CONFIGNAME_PROP)) {
break;
}
}
// if there is only one conf, use that
try {
configNames = zkClient.getChildren(ZkConfigManager.CONFIGS_ZKNODE, null,
true);
} catch (NoNodeException e) {
// just keep trying
}
if (configNames != null && configNames.size() == 1) {
// no config set named, but there is only 1 - use it
log.info("Only one config set found in zk - using it:" + configNames.get(0));
collectionProps.put(ZkController.CONFIGNAME_PROP, configNames.get(0));
break;
}
if (configNames != null && configNames.contains(collection)) {
log.info(
"Could not find explicit collection configName, but found config name matching collection name - using that set.");
collectionProps.put(ZkController.CONFIGNAME_PROP, collection);
break;
}
log.info("Could not find collection configName - pausing for 3 seconds and trying again - try: " + retry);
Thread.sleep(3000);
}
if (retry == retryLimt) {
log.error("Could not find configName for collection " + collection);
throw new ZooKeeperException(
SolrException.ErrorCode.SERVER_ERROR,
"Could not find configName for collection " + collection + " found:" + configNames);
}
}
}

View File

@ -125,17 +125,6 @@ class ShardLeaderElectionContextBase extends ElectionContext {
this.zkClient = zkStateReader.getZkClient();
this.shardId = shardId;
this.collection = collection;
try {
new ZkCmdExecutor(zkStateReader.getZkClient().getZkClientTimeout())
.ensureExists(ZkStateReader.COLLECTIONS_ZKNODE + "/" + collection,
zkClient);
} catch (KeeperException e) {
throw new SolrException(ErrorCode.SERVER_ERROR, e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throw new SolrException(ErrorCode.SERVER_ERROR, e);
}
}
@Override
@ -175,9 +164,16 @@ class ShardLeaderElectionContextBase extends ElectionContext {
void runLeaderProcess(boolean weAreReplacement, int pauseBeforeStartMs)
throws KeeperException, InterruptedException, IOException {
// register as leader - if an ephemeral is already there, wait to see if it goes away
if (!zkClient.exists(ZkStateReader.COLLECTIONS_ZKNODE + "/" + collection, true)) {
log.info("Will not register as leader because collection appears to be gone.");
return;
}
String parent = new Path(leaderPath).getParent().toString();
ZkCmdExecutor zcmd = new ZkCmdExecutor(30000);
zcmd.ensureExists(parent, zkClient);
// only if /collections/{collection} exists already do we succeed in creating this path
zcmd.ensureExists(parent, (byte[])null, CreateMode.PERSISTENT, zkClient, 2);
try {
RetryUtil.retryOnThrowable(NodeExistsException.class, 60000, 5000, () -> {

View File

@ -360,8 +360,13 @@ public class LeaderElector {
public void setup(final ElectionContext context) throws InterruptedException,
KeeperException {
String electZKPath = context.electionPath + LeaderElector.ELECTION_NODE;
zkCmdExecutor.ensureExists(electZKPath, zkClient);
if (context instanceof OverseerElectionContext) {
zkCmdExecutor.ensureExists(electZKPath, zkClient);
} else {
// we use 2 param so that replica won't create /collection/{collection} if it doesn't exist
zkCmdExecutor.ensureExists(electZKPath, (byte[])null, CreateMode.PERSISTENT, zkClient, 2);
}
this.context = context;
}

View File

@ -34,7 +34,6 @@ import java.util.HashSet;
import java.util.List;
import java.util.Locale;
import java.util.Map;
import java.util.Properties;
import java.util.Set;
import java.util.concurrent.Callable;
import java.util.concurrent.ConcurrentHashMap;
@ -1273,130 +1272,6 @@ public class ZkController {
zkClient.printLayoutToStdOut();
}
public void createCollectionZkNode(CloudDescriptor cd) {
String collection = cd.getCollectionName();
log.debug("Check for collection zkNode:" + collection);
String collectionPath = ZkStateReader.COLLECTIONS_ZKNODE + "/" + collection;
try {
if (!zkClient.exists(collectionPath, true)) {
log.debug("Creating collection in ZooKeeper:" + collection);
try {
Map<String, Object> collectionProps = new HashMap<>();
// TODO: if collection.configName isn't set, and there isn't already a conf in zk, just use that?
String defaultConfigName = System.getProperty(COLLECTION_PARAM_PREFIX + CONFIGNAME_PROP, collection);
// params passed in - currently only done via core admin (create core commmand).
if (cd.getParams().size() > 0) {
collectionProps.putAll(cd.getParams());
// if the config name wasn't passed in, use the default
if (!collectionProps.containsKey(CONFIGNAME_PROP)) {
// TODO: getting the configName from the collectionPath should fail since we already know it doesn't exist?
getConfName(collection, collectionPath, collectionProps);
}
} else if (System.getProperty("bootstrap_confdir") != null) {
// if we are bootstrapping a collection, default the config for
// a new collection to the collection we are bootstrapping
log.info("Setting config for collection:" + collection + " to " + defaultConfigName);
Properties sysProps = System.getProperties();
for (String sprop : System.getProperties().stringPropertyNames()) {
if (sprop.startsWith(COLLECTION_PARAM_PREFIX)) {
collectionProps.put(sprop.substring(COLLECTION_PARAM_PREFIX.length()), sysProps.getProperty(sprop));
}
}
// if the config name wasn't passed in, use the default
if (!collectionProps.containsKey(CONFIGNAME_PROP))
collectionProps.put(CONFIGNAME_PROP, defaultConfigName);
} else if (Boolean.getBoolean("bootstrap_conf")) {
// the conf name should should be the collection name of this core
collectionProps.put(CONFIGNAME_PROP, cd.getCollectionName());
} else {
getConfName(collection, collectionPath, collectionProps);
}
collectionProps.remove(ZkStateReader.NUM_SHARDS_PROP); // we don't put numShards in the collections properties
ZkNodeProps zkProps = new ZkNodeProps(collectionProps);
zkClient.makePath(collectionPath, Utils.toJSON(zkProps), CreateMode.PERSISTENT, null, true);
} catch (KeeperException e) {
// it's okay if the node already exists
if (e.code() != KeeperException.Code.NODEEXISTS) {
throw e;
}
}
} else {
log.debug("Collection zkNode exists");
}
} catch (KeeperException e) {
// it's okay if another beats us creating the node
if (e.code() == KeeperException.Code.NODEEXISTS) {
return;
}
throw new SolrException(ErrorCode.SERVER_ERROR, "Error creating collection node in Zookeeper", e);
} catch (InterruptedException e) {
Thread.interrupted();
throw new SolrException(ErrorCode.SERVER_ERROR, "Error creating collection node in Zookeeper", e);
}
}
private void getConfName(String collection, String collectionPath,
Map<String, Object> collectionProps) throws KeeperException,
InterruptedException {
// check for configName
log.debug("Looking for collection configName");
List<String> configNames = null;
int retry = 1;
int retryLimt = 6;
for (; retry < retryLimt; retry++) {
if (zkClient.exists(collectionPath, true)) {
ZkNodeProps cProps = ZkNodeProps.load(zkClient.getData(collectionPath, null, null, true));
if (cProps.containsKey(CONFIGNAME_PROP)) {
break;
}
}
// if there is only one conf, use that
try {
configNames = zkClient.getChildren(ZkConfigManager.CONFIGS_ZKNODE, null,
true);
} catch (NoNodeException e) {
// just keep trying
}
if (configNames != null && configNames.size() == 1) {
// no config set named, but there is only 1 - use it
log.info("Only one config set found in zk - using it:" + configNames.get(0));
collectionProps.put(CONFIGNAME_PROP, configNames.get(0));
break;
}
if (configNames != null && configNames.contains(collection)) {
log.info("Could not find explicit collection configName, but found config name matching collection name - using that set.");
collectionProps.put(CONFIGNAME_PROP, collection);
break;
}
log.info("Could not find collection configName - pausing for 3 seconds and trying again - try: " + retry);
Thread.sleep(3000);
}
if (retry == retryLimt) {
log.error("Could not find configName for collection " + collection);
throw new ZooKeeperException(
SolrException.ErrorCode.SERVER_ERROR,
"Could not find configName for collection " + collection + " found:" + configNames);
}
}
public ZkStateReader getZkStateReader() {
return zkStateReader;
}
@ -2175,7 +2050,8 @@ public class ZkController {
} else {
String parentZNodePath = getLeaderInitiatedRecoveryZnodePath(collection, shardId);
try {
zkClient.makePath(parentZNodePath, retryOnConnLoss);
// make sure we don't create /collections/{collection} if they do not exist with 2 param
zkClient.makePath(parentZNodePath, (byte[]) null, CreateMode.PERSISTENT, (Watcher) null, true, retryOnConnLoss, 2);
} catch (KeeperException.NodeExistsException nee) {
// if it exists, that's great!
}

View File

@ -109,7 +109,7 @@ public class ZkSolrResourceLoader extends SolrResourceLoader {
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throw new IOException("Error opening " + file, e);
} catch (KeeperException e) {
} catch (Exception e) {
throw new IOException("Error opening " + file, e);
}
}

View File

@ -78,11 +78,10 @@ public abstract class ConfigSetService {
IndexSchema schema = createIndexSchema(dcore, solrConfig);
NamedList properties = createConfigSetProperties(dcore, coreLoader);
return new ConfigSet(configName(dcore), solrConfig, schema, properties);
}
catch (Exception e) {
} catch (Exception e) {
throw new SolrException(SolrException.ErrorCode.SERVER_ERROR,
"Could not load conf for core " + dcore.getName() +
": " + e.getMessage(), e);
"Could not load conf for core " + dcore.getName() +
": " + e.getMessage(), e);
}
}

View File

@ -346,9 +346,11 @@ public class CollectionsHandler extends RequestHandlerBase implements Permission
try {
String path = ZkStateReader.CONFIGS_ZKNODE + "/" + SYSTEM_COLL + "/schema.xml";
byte[] data = IOUtils.toByteArray(Thread.currentThread().getContextClassLoader().getResourceAsStream("SystemCollectionSchema.xml"));
assert data != null && data.length > 0;
cmdExecutor.ensureExists(path, data, CreateMode.PERSISTENT, zk);
path = ZkStateReader.CONFIGS_ZKNODE + "/" + SYSTEM_COLL + "/solrconfig.xml";
data = IOUtils.toByteArray(Thread.currentThread().getContextClassLoader().getResourceAsStream("SystemCollectionSolrConfig.xml"));
assert data != null && data.length > 0;
cmdExecutor.ensureExists(path, data, CreateMode.PERSISTENT, zk);
} catch (IOException e) {
throw new SolrException(ErrorCode.SERVER_ERROR, e);

View File

@ -80,6 +80,8 @@ public class LeaderElectionTest extends SolrTestCaseJ4 {
zkClient = new SolrZkClient(server.getZkAddress(), TIMEOUT);
zkStateReader = new ZkStateReader(zkClient);
seqToThread = Collections.synchronizedMap(new HashMap<Integer,Thread>());
zkClient.makePath("/collections/collection1", true);
zkClient.makePath("/collections/collection2", true);
}
class TestLeaderElectionContext extends ShardLeaderElectionContextBase {

View File

@ -44,6 +44,7 @@ import org.apache.solr.handler.component.ShardRequest;
import org.apache.solr.handler.component.ShardResponse;
import org.apache.solr.util.TimeOut;
import org.apache.zookeeper.CreateMode;
import org.apache.zookeeper.Watcher;
import org.easymock.Capture;
import org.easymock.EasyMock;
import org.junit.After;
@ -114,7 +115,6 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
zkStateReaderMock = createMock(ZkStateReader.class);
clusterStateMock = createMock(ClusterState.class);
solrZkClientMock = createMock(SolrZkClient.class);
}
@AfterClass
@ -143,9 +143,7 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
reset(zkStateReaderMock);
reset(clusterStateMock);
reset(solrZkClientMock);
underTest = new OverseerCollectionConfigSetProcessorToBeTested(zkStateReaderMock,
"1234", shardHandlerFactoryMock, ADMIN_PATH, workQueueMock, runningMapMock,
completedMapMock, failureMapMock);
zkMap.clear();
collectionsSet.clear();
}
@ -157,12 +155,12 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
}
protected Set<String> commonMocks(int liveNodesCount) throws Exception {
shardHandlerFactoryMock.getShardHandler();
expectLastCall().andAnswer(() -> {
log.info("SHARDHANDLER");
return shardHandlerMock;
}).anyTimes();
workQueueMock.peekTopN(EasyMock.anyInt(), anyObject(Predicate.class), EasyMock.anyLong());
expectLastCall().andAnswer(() -> {
Object result;
@ -203,12 +201,12 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
workQueueMock.poll();
expectLastCall().andAnswer(() -> queue.poll()).anyTimes();
zkStateReaderMock.getClusterState();
expectLastCall().andAnswer(() -> clusterStateMock).anyTimes();
zkStateReaderMock.getZkClient();
expectLastCall().andAnswer(() -> solrZkClientMock).anyTimes();
zkStateReaderMock.getClusterState();
expectLastCall().andAnswer(() -> clusterStateMock).anyTimes();
zkStateReaderMock.updateClusterState();
@ -262,6 +260,18 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
String key = (String) getCurrentArguments()[0];
return key;
}).anyTimes();
solrZkClientMock.makePath(anyObject(String.class), anyObject(byte[].class), anyObject(CreateMode.class), anyObject(Watcher.class), anyBoolean());
expectLastCall().andAnswer(() -> {
String key = (String) getCurrentArguments()[0];
return key;
}).anyTimes();
solrZkClientMock.makePath(anyObject(String.class), anyObject(byte[].class), anyObject(CreateMode.class), anyObject(Watcher.class), anyBoolean(), anyBoolean(), anyInt());
expectLastCall().andAnswer(() -> {
String key = (String) getCurrentArguments()[0];
return key;
}).anyTimes();
solrZkClientMock.exists(anyObject(String.class),anyBoolean());
expectLastCall().andAnswer(() -> {
@ -518,12 +528,17 @@ public class OverseerCollectionConfigSetProcessorTest extends SolrTestCaseJ4 {
replicationFactor);
}
replay(workQueueMock);
replay(solrZkClientMock);
replay(zkStateReaderMock);
replay(workQueueMock);
replay(clusterStateMock);
replay(shardHandlerFactoryMock);
replay(shardHandlerMock);
underTest = new OverseerCollectionConfigSetProcessorToBeTested(zkStateReaderMock,
"1234", shardHandlerFactoryMock, ADMIN_PATH, workQueueMock, runningMapMock,
completedMapMock, failureMapMock);
log.info("clusterstate " + clusterStateMock.hashCode());

View File

@ -26,6 +26,7 @@ import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.cloud.ZkCmdExecutor;
import org.apache.solr.common.cloud.ZkOperation;
import org.apache.solr.util.AbstractSolrTestCase;
import org.apache.zookeeper.CreateMode;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.WatchedEvent;
import org.apache.zookeeper.Watcher;
@ -304,6 +305,59 @@ public class ZkSolrClientTest extends AbstractSolrTestCase {
}
}
public void testSkipPathPartsOnMakePath() throws Exception {
try (ZkConnection conn = new ZkConnection()) {
final SolrZkClient zkClient = conn.getClient();
zkClient.makePath("/test", true);
// should work
zkClient.makePath("/test/path/here", (byte[]) null, CreateMode.PERSISTENT, (Watcher) null, true, true, 1);
zkClient.clean("/");
// should not work
try {
zkClient.makePath("/test/path/here", (byte[]) null, CreateMode.PERSISTENT, (Watcher) null, true, true, 1);
fail("We should not be able to create this path");
} catch (Exception e) {
}
zkClient.clean("/");
ZkCmdExecutor zkCmdExecutor = new ZkCmdExecutor(30000);
try {
zkCmdExecutor.ensureExists("/collection/collection/leader", (byte[]) null, CreateMode.PERSISTENT, zkClient, 2);
fail("We should not be able to create this path");
} catch (Exception e) {
}
zkClient.makePath("/collection", true);
try {
zkCmdExecutor.ensureExists("/collections/collection/leader", (byte[]) null, CreateMode.PERSISTENT, zkClient, 2);
fail("We should not be able to create this path");
} catch (Exception e) {
}
zkClient.makePath("/collection/collection", true);
byte[] bytes = new byte[10];
zkCmdExecutor.ensureExists("/collection/collection", bytes, CreateMode.PERSISTENT, zkClient, 2);
byte[] returnedBytes = zkClient.getData("/collection/collection", null, null, true);
assertNull("We skipped 2 path parts, so data won't be written", returnedBytes);
zkClient.makePath("/collection/collection/leader", true);
zkCmdExecutor.ensureExists("/collection/collection/leader", (byte[]) null, CreateMode.PERSISTENT, zkClient, 2);
}
}
@Override
public void tearDown() throws Exception {

View File

@ -411,13 +411,13 @@ public class SolrZkClient implements Closeable {
public void makePath(String path, boolean failOnExists, boolean retryOnConnLoss) throws KeeperException,
InterruptedException {
makePath(path, null, CreateMode.PERSISTENT, null, failOnExists, retryOnConnLoss);
makePath(path, null, CreateMode.PERSISTENT, null, failOnExists, retryOnConnLoss, 0);
}
public void makePath(String path, File file, boolean failOnExists, boolean retryOnConnLoss)
throws IOException, KeeperException, InterruptedException {
makePath(path, FileUtils.readFileToByteArray(file),
CreateMode.PERSISTENT, null, failOnExists, retryOnConnLoss);
CreateMode.PERSISTENT, null, failOnExists, retryOnConnLoss, 0);
}
public void makePath(String path, File file, boolean retryOnConnLoss) throws IOException,
@ -463,21 +463,35 @@ public class SolrZkClient implements Closeable {
*/
public void makePath(String path, byte[] data, CreateMode createMode,
Watcher watcher, boolean retryOnConnLoss) throws KeeperException, InterruptedException {
makePath(path, data, createMode, watcher, true, retryOnConnLoss);
makePath(path, data, createMode, watcher, true, retryOnConnLoss, 0);
}
/**
* Creates the path in ZooKeeper, creating each node as necessary.
*
* e.g. If <code>path=/solr/group/node</code> and none of the nodes, solr,
* group, node exist, each will be created.
*
* @param data to set on the last zkNode
*/
public void makePath(String path, byte[] data, CreateMode createMode,
Watcher watcher, boolean failOnExists, boolean retryOnConnLoss) throws KeeperException, InterruptedException {
makePath(path, data, createMode, watcher, failOnExists, retryOnConnLoss, 0);
}
/**
* Creates the path in ZooKeeper, creating each node as necessary.
*
* e.g. If <code>path=/solr/group/node</code> and none of the nodes, solr,
* group, node exist, each will be created.
*
* skipPathParts will force the call to fail if the first skipPathParts do not exist already.
*
* Note: retryOnConnLoss is only respected for the final node - nodes
* before that are always retried on connection loss.
*/
public void makePath(String path, byte[] data, CreateMode createMode,
Watcher watcher, boolean failOnExists, boolean retryOnConnLoss) throws KeeperException, InterruptedException {
Watcher watcher, boolean failOnExists, boolean retryOnConnLoss, int skipPathParts) throws KeeperException, InterruptedException {
log.debug("makePath: {}", path);
boolean retry = true;
@ -487,9 +501,12 @@ public class SolrZkClient implements Closeable {
String[] paths = path.split("/");
StringBuilder sbPath = new StringBuilder();
for (int i = 0; i < paths.length; i++) {
byte[] bytes = null;
String pathPiece = paths[i];
sbPath.append("/" + pathPiece);
if (i < skipPathParts) {
continue;
}
byte[] bytes = null;
final String currentPath = sbPath.toString();
Object exists = exists(currentPath, watcher, retryOnConnLoss);
if (exists == null || ((i == paths.length -1) && failOnExists)) {

View File

@ -75,17 +75,26 @@ public class ZkCmdExecutor {
}
public void ensureExists(String path, final SolrZkClient zkClient) throws KeeperException, InterruptedException {
ensureExists(path, null, CreateMode.PERSISTENT, zkClient);
ensureExists(path, null, CreateMode.PERSISTENT, zkClient, 0);
}
public void ensureExists(String path, final byte[] data, final SolrZkClient zkClient) throws KeeperException, InterruptedException {
ensureExists(path, data, CreateMode.PERSISTENT, zkClient, 0);
}
public void ensureExists(String path, final byte[] data, CreateMode createMode, final SolrZkClient zkClient) throws KeeperException, InterruptedException {
ensureExists(path, data, createMode, zkClient, 0);
}
public void ensureExists(final String path, final byte[] data,
CreateMode createMode, final SolrZkClient zkClient) throws KeeperException, InterruptedException {
CreateMode createMode, final SolrZkClient zkClient, int skipPathParts) throws KeeperException, InterruptedException {
if (zkClient.exists(path, true)) {
return;
}
try {
zkClient.makePath(path, data, createMode, true);
zkClient.makePath(path, data, createMode, null, true, true, skipPathParts);
} catch (NodeExistsException e) {
// it's okay if another beats us creating the node
}