Skip to content
Draft
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
10 changes: 10 additions & 0 deletions solr/core/build.gradle
Original file line number Diff line number Diff line change
Expand Up @@ -121,6 +121,16 @@ dependencies {
implementation 'org.eclipse.jetty:jetty-io'
implementation 'org.eclipse.jetty.toolchain:jetty-servlet-api'

implementation('org.apache.curator:curator-framework', {
exclude group: 'org.apache.zookeeper', module: 'zookeeper'
})
testImplementation('org.apache.curator:curator-client', {
exclude group: 'org.apache.zookeeper', module: 'zookeeper'
})
testImplementation('org.apache.curator:curator-test', {
exclude group: 'org.apache.zookeeper', module: 'zookeeper'
})

// ZooKeeper
implementation('org.apache.zookeeper:zookeeper', {
exclude group: "org.apache.yetus", module: "audience-annotations"
Expand Down
11 changes: 5 additions & 6 deletions solr/core/src/java/org/apache/solr/cli/AuthTool.java
Original file line number Diff line number Diff line change
Expand Up @@ -217,8 +217,7 @@ private int handleKerberos(CommandLine cli) throws Exception {
.withTimeout(
SolrZkClientTimeout.DEFAULT_ZK_CLIENT_TIMEOUT, TimeUnit.MILLISECONDS)
.build()) {
zkClient.setData(
"/security.json", securityJson.getBytes(StandardCharsets.UTF_8), true);
zkClient.setData("/security.json", securityJson.getBytes(StandardCharsets.UTF_8));
} catch (Exception ex) {
CLIO.out(
"Unable to access ZooKeeper. Please add the following security.json to ZooKeeper (in case of SolrCloud):\n"
Expand Down Expand Up @@ -384,7 +383,7 @@ private int handleBasicAuth(CommandLine cli) throws Exception {
.withUrl(zkHost)
.withTimeout(SolrZkClientTimeout.DEFAULT_ZK_CLIENT_TIMEOUT, TimeUnit.MILLISECONDS)
.build()) {
zkClient.setData("/security.json", securityJson.getBytes(StandardCharsets.UTF_8), true);
zkClient.setData("/security.json", securityJson.getBytes(StandardCharsets.UTF_8));
}
}

Expand Down Expand Up @@ -452,8 +451,8 @@ private int handleBasicAuth(CommandLine cli) throws Exception {

private void checkSecurityJsonExists(SolrZkClient zkClient)
throws KeeperException, InterruptedException {
if (zkClient.exists("/security.json", true)) {
byte[] oldSecurityBytes = zkClient.getData("/security.json", null, null, true);
if (zkClient.exists("/security.json")) {
byte[] oldSecurityBytes = zkClient.getData("/security.json", null, null);
if (!"{}".equals(new String(oldSecurityBytes, StandardCharsets.UTF_8).trim())) {
CLIO.out(
"Security is already enabled. You can disable it with 'bin/solr auth disable'. Existing security.json: \n"
Expand All @@ -479,7 +478,7 @@ private void clearSecurityJson(CommandLine cli, boolean updateIncludeFileOnly) t
.withUrl(zkHost)
.withTimeout(SolrZkClientTimeout.DEFAULT_ZK_CLIENT_TIMEOUT, TimeUnit.MILLISECONDS)
.build()) {
zkClient.setData("/security.json", "{}".getBytes(StandardCharsets.UTF_8), true);
zkClient.setData("/security.json", "{}".getBytes(StandardCharsets.UTF_8));
}
}
}
Expand Down
4 changes: 1 addition & 3 deletions solr/core/src/java/org/apache/solr/cli/CreateTool.java
Original file line number Diff line number Diff line change
Expand Up @@ -243,9 +243,7 @@ protected void createCollection(CloudSolrClient cloudSolrClient, CommandLine cli
boolean configExistsInZk =
confName != null
&& !confName.trim().isEmpty()
&& ZkStateReader.from(cloudSolrClient)
.getZkClient()
.exists("/configs/" + confName, true);
&& ZkStateReader.from(cloudSolrClient).getZkClient().exists("/configs/" + confName);

if (CollectionAdminParams.SYSTEM_COLL.equals(collectionName)) {
// do nothing
Expand Down
2 changes: 1 addition & 1 deletion solr/core/src/java/org/apache/solr/cli/ZkRmTool.java
Original file line number Diff line number Diff line change
Expand Up @@ -87,7 +87,7 @@ public void runImpl(CommandLine cli) throws Exception {
.withUrl(zkHost)
.withTimeout(SolrZkClientTimeout.DEFAULT_ZK_CLIENT_TIMEOUT, TimeUnit.MILLISECONDS)
.build()) {
if (!recurse && zkClient.getChildren(znode, null, true).size() != 0) {
if (!recurse && zkClient.getChildren(znode, null).size() != 0) {
throw new SolrServerException(
"ZooKeeper node " + znode + " has children and recurse has NOT been specified.");
}
Expand Down
4 changes: 2 additions & 2 deletions solr/core/src/java/org/apache/solr/cloud/CloudUtil.java
Original file line number Diff line number Diff line change
Expand Up @@ -140,10 +140,10 @@ public static String unifiedResourcePath(SolrResourceLoader loader) {
public static Map<String, byte[]> getTrustedKeys(SolrZkClient zk, String dir) {
Map<String, byte[]> result = new HashMap<>();
try {
List<String> children = zk.getChildren("/keys/" + dir, null, true);
List<String> children = zk.getChildren("/keys/" + dir, null);
for (String key : children) {
if (key.endsWith(".der"))
result.put(key, zk.getData("/keys/" + dir + "/" + key, null, null, true));
result.put(key, zk.getData("/keys/" + dir + "/" + key, null, null));
}
} catch (KeeperException.NoNodeException e) {
log.info("Error fetching key names");
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -311,8 +311,8 @@ enum State {
this.rootNodePath = rootNodePath;

try {
if (!zkClient.exists(rootNodePath, true)) {
zkClient.makePath(rootNodePath, new byte[0], CreateMode.PERSISTENT, true);
if (!zkClient.exists(rootNodePath)) {
zkClient.makePath(rootNodePath, new byte[0], CreateMode.PERSISTENT);
}
} catch (KeeperException.NodeExistsException nee) {
// Some other thread (on this or another JVM) beat us to create the node, that's ok, the
Expand All @@ -329,27 +329,25 @@ void createNewInFlightTask(String asyncId) throws KeeperException, InterruptedEx
zkClient.create(
getPath(asyncId),
State.SUBMITTED.shorthand.getBytes(StandardCharsets.UTF_8),
CreateMode.EPHEMERAL,
true);
CreateMode.EPHEMERAL);
}

void setTaskRunning(String asyncId) throws KeeperException, InterruptedException {
zkClient.setData(
getPath(asyncId), State.RUNNING.shorthand.getBytes(StandardCharsets.UTF_8), true);
zkClient.setData(getPath(asyncId), State.RUNNING.shorthand.getBytes(StandardCharsets.UTF_8));
}

void deleteInFlightTask(String asyncId) throws KeeperException, InterruptedException {
zkClient.delete(getPath(asyncId), -1, true);
zkClient.delete(getPath(asyncId), -1);
}

State getInFlightState(String asyncId) throws KeeperException, InterruptedException {
if (!zkClient.exists(getPath(asyncId), true)) {
if (!zkClient.exists(getPath(asyncId))) {
return State.NOT_FOUND;
}

final byte[] bytes;
try {
bytes = zkClient.getData(getPath(asyncId), null, null, true);
bytes = zkClient.getData(getPath(asyncId), null, null);
} catch (KeeperException.NoNodeException nne) {
// Unlikely race, but not impossible...
if (log.isInfoEnabled()) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -598,7 +598,7 @@ private void doStateDotJsonCasUpdate(ClusterState updatedState)
if (updater.isCollectionCreation()) {
// The state.json file does not exist yet (more precisely it is assumed not to exist)
log.debug("going to create collection {}", jsonPath);
zkStateReader.getZkClient().create(jsonPath, stateJson, CreateMode.PERSISTENT, true);
zkStateReader.getZkClient().create(jsonPath, stateJson, CreateMode.PERSISTENT);
} else {
// We're updating an existing state.json
if (log.isDebugEnabled()) {
Expand All @@ -607,9 +607,7 @@ private void doStateDotJsonCasUpdate(ClusterState updatedState)
jsonPath,
collection.getZNodeVersion());
}
zkStateReader
.getZkClient()
.setData(jsonPath, stateJson, collection.getZNodeVersion(), true);
zkStateReader.getZkClient().setData(jsonPath, stateJson, collection.getZNodeVersion());
}
}
}
Expand All @@ -623,7 +621,7 @@ private void doStateDotJsonCasUpdate(ClusterState updatedState)
private ClusterState fetchStateForCollection() throws KeeperException, InterruptedException {
String collectionStatePath = DocCollection.getCollectionPath(updater.getCollectionName());
Stat stat = new Stat();
byte[] data = zkStateReader.getZkClient().getData(collectionStatePath, null, stat, true);
byte[] data = zkStateReader.getZkClient().getData(collectionStatePath, null, stat);

// This factory method can detect a missing configName and supply it by reading it from the
// old ZK location.
Expand Down Expand Up @@ -933,7 +931,7 @@ public static void executeNodeDownStateUpdate(String nodeName, ZkStateReader zkS

try {
final List<String> collectionNames =
zkStateReader.getZkClient().getChildren(COLLECTIONS_ZKNODE, null, true);
zkStateReader.getZkClient().getChildren(COLLECTIONS_ZKNODE, null);

// Collections are totally independent of each other. Multiple threads could share the load
// here (need a ZK connection for each though).
Expand Down
20 changes: 9 additions & 11 deletions solr/core/src/java/org/apache/solr/cloud/DistributedMap.java
Original file line number Diff line number Diff line change
Expand Up @@ -68,8 +68,7 @@ private void assertKeyFormat(String trackingId) {

public void put(String trackingId, byte[] data) throws KeeperException, InterruptedException {
assertKeyFormat(trackingId);
zookeeper.makePath(
dir + "/" + PREFIX + trackingId, data, CreateMode.PERSISTENT, null, false, true);
zookeeper.makePath(dir + "/" + PREFIX + trackingId, data, CreateMode.PERSISTENT, null, false);
}

/**
Expand All @@ -81,25 +80,24 @@ public boolean putIfAbsent(String trackingId, byte[] data)
throws KeeperException, InterruptedException {
assertKeyFormat(trackingId);
try {
zookeeper.makePath(
dir + "/" + PREFIX + trackingId, data, CreateMode.PERSISTENT, null, true, true);
zookeeper.makePath(dir + "/" + PREFIX + trackingId, data, CreateMode.PERSISTENT, null, true);
return true;
} catch (NodeExistsException e) {
return false;
}
}

public byte[] get(String trackingId) throws KeeperException, InterruptedException {
return zookeeper.getData(dir + "/" + PREFIX + trackingId, null, null, true);
return zookeeper.getData(dir + "/" + PREFIX + trackingId, null, null);
}

public boolean contains(String trackingId) throws KeeperException, InterruptedException {
return zookeeper.exists(dir + "/" + PREFIX + trackingId, true);
return zookeeper.exists(dir + "/" + PREFIX + trackingId);
}

public int size() throws KeeperException, InterruptedException {
Stat stat = new Stat();
zookeeper.getData(dir, null, stat, true);
zookeeper.getData(dir, null, stat);
return stat.getNumChildren();
}

Expand All @@ -110,7 +108,7 @@ public int size() throws KeeperException, InterruptedException {
public boolean remove(String trackingId) throws KeeperException, InterruptedException {
final var path = dir + "/" + PREFIX + trackingId;
try {
zookeeper.delete(path, -1, true);
zookeeper.delete(path, -1);
} catch (KeeperException.NoNodeException e) {
return false;
} catch (KeeperException.NotEmptyException hack) {
Expand All @@ -123,15 +121,15 @@ public boolean remove(String trackingId) throws KeeperException, InterruptedExce

/** Helper method to clear all child nodes for a parent node. */
public void clear() throws KeeperException, InterruptedException {
List<String> childNames = zookeeper.getChildren(dir, null, true);
List<String> childNames = zookeeper.getChildren(dir, null);
for (String childName : childNames) {
zookeeper.delete(dir + "/" + childName, -1, true);
zookeeper.delete(dir + "/" + childName, -1);
}
}

/** Returns the keys of all the elements in the map */
public Collection<String> keys() throws KeeperException, InterruptedException {
List<String> childs = zookeeper.getChildren(dir, null, true);
List<String> childs = zookeeper.getChildren(dir, null);
final List<String> ids = new ArrayList<>(childs.size());
childs.stream().forEach((child) -> ids.add(child.substring(PREFIX.length())));
return ids;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -55,7 +55,7 @@ public void cancelElection() throws InterruptedException, KeeperException {
if (leaderSeqPath != null) {
try {
log.debug("Canceling election {}", leaderSeqPath);
zkClient.delete(leaderSeqPath, -1, true);
zkClient.delete(leaderSeqPath, -1);
} catch (NoNodeException e) {
// fine
log.debug("cancelElection did not find election node to remove {}", leaderSeqPath);
Expand Down
33 changes: 12 additions & 21 deletions solr/core/src/java/org/apache/solr/cloud/LeaderElector.java
Original file line number Diff line number Diff line change
Expand Up @@ -25,7 +25,6 @@
import java.util.regex.Matcher;
import java.util.regex.Pattern;
import org.apache.solr.cloud.ZkController.ContextKey;
import org.apache.solr.common.AlreadyClosedException;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.cloud.ZkMaintenanceUtils;
Expand Down Expand Up @@ -95,7 +94,7 @@ private void checkIfIamLeader(final ElectionContext context, boolean replacement
context.checkIfIamLeaderFired();
// get all other numbers...
final String holdElectionPath = context.electionPath + ELECTION_NODE;
List<String> seqs = zkClient.getChildren(holdElectionPath, null, true);
List<String> seqs = zkClient.getChildren(holdElectionPath, null);
sortSeqs(seqs);

String leaderSeqNodeName =
Expand All @@ -107,15 +106,15 @@ private void checkIfIamLeader(final ElectionContext context, boolean replacement

// If any double-registrations exist for me, remove all but this latest one!
// TODO: can we even get into this state?
String prefix = zkClient.getZooKeeper().getSessionId() + "-" + context.id + "-";
String prefix = zkClient.getZkSessionId() + "-" + context.id + "-";
Iterator<String> it = seqs.iterator();
while (it.hasNext()) {
String elec = it.next();
if (!elec.equals(leaderSeqNodeName) && elec.startsWith(prefix)) {
try {
String toDelete = holdElectionPath + "/" + elec;
log.warn("Deleting duplicate registration: {}", toDelete);
zkClient.delete(toDelete, -1, true);
zkClient.delete(toDelete, -1);
} catch (KeeperException.NoNodeException e) {
// ignore
}
Expand Down Expand Up @@ -149,8 +148,7 @@ private void checkIfIamLeader(final ElectionContext context, boolean replacement
watcher =
new ElectionWatcher(
context.leaderSeqPath, watchedNode, getSeq(context.leaderSeqPath), context),
null,
true);
null);
log.debug("Watching path {} to know if I could be the leader", watchedNode);
} catch (KeeperException.SessionExpiredException e) {
throw e;
Expand Down Expand Up @@ -227,7 +225,7 @@ public int joinElection(ElectionContext context, boolean replacement, boolean jo

final String shardsElectZkPath = context.electionPath + LeaderElector.ELECTION_NODE;

long sessionId = zkClient.getZooKeeper().getSessionId();
long sessionId = zkClient.getZkSessionId();
String id = sessionId + "-" + context.id;
String leaderSeqPath = null;
boolean cont = true;
Expand All @@ -241,10 +239,7 @@ public int joinElection(ElectionContext context, boolean replacement, boolean jo
if (nodes.size() < 2) {
leaderSeqPath =
zkClient.create(
shardsElectZkPath + "/" + id + "-n_",
null,
CreateMode.EPHEMERAL_SEQUENTIAL,
false);
shardsElectZkPath + "/" + id + "-n_", null, CreateMode.EPHEMERAL_SEQUENTIAL);
} else {
String firstInLine = nodes.get(1);
log.debug("The current head: {}", firstInLine);
Expand All @@ -253,23 +248,20 @@ public int joinElection(ElectionContext context, boolean replacement, boolean jo
throw new IllegalStateException("Could not find regex match in:" + firstInLine);
}
leaderSeqPath = shardsElectZkPath + "/" + id + "-n_" + m.group(1);
zkClient.create(leaderSeqPath, null, CreateMode.EPHEMERAL, false);
zkClient.create(leaderSeqPath, null, CreateMode.EPHEMERAL);
}
} else {
leaderSeqPath =
zkClient.create(
shardsElectZkPath + "/" + id + "-n_",
null,
CreateMode.EPHEMERAL_SEQUENTIAL,
false);
shardsElectZkPath + "/" + id + "-n_", null, CreateMode.EPHEMERAL_SEQUENTIAL);
}

log.debug("Joined leadership election with path: {}", leaderSeqPath);
context.leaderSeqPath = leaderSeqPath;
cont = false;
} catch (ConnectionLossException e) {
// we don't know if we made our node or not...
List<String> entries = zkClient.getChildren(shardsElectZkPath, null, true);
List<String> entries = zkClient.getChildren(shardsElectZkPath, null);

boolean foundId = false;
for (String entry : entries) {
Expand Down Expand Up @@ -337,7 +329,7 @@ public void process(WatchedEvent event) {
if (canceled) {
log.debug("This watcher is not active anymore {}", myNode);
try {
zkClient.delete(myNode, -1, true);
zkClient.delete(myNode, -1);
} catch (KeeperException.NoNodeException nne) {
// expected . don't do anything
} catch (Exception e) {
Expand All @@ -348,7 +340,7 @@ public void process(WatchedEvent event) {
try {
// am I the next leader?
checkIfIamLeader(context, true);
} catch (AlreadyClosedException e) {
} catch (IllegalStateException e) {

} catch (Exception e) {
if (!zkClient.isClosed()) {
Expand All @@ -365,8 +357,7 @@ public void setup(final ElectionContext context) throws InterruptedException, Ke
ZkMaintenanceUtils.ensureExists(electZKPath, zkClient);
} else {
// we use 2 param so that replica won't create /collection/{collection} if it doesn't exist
ZkMaintenanceUtils.ensureExists(
electZKPath, (byte[]) null, CreateMode.PERSISTENT, zkClient, 2);
ZkMaintenanceUtils.ensureExists(electZKPath, null, CreateMode.PERSISTENT, zkClient, 2);
}

this.context = context;
Expand Down
Loading