Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

SOLR-16415: ZK DistributedMap should reject slash in IDs. #1824

Merged
merged 3 commits into from
Sep 11, 2023
Merged
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
3 changes: 3 additions & 0 deletions solr/CHANGES.txt
Original file line number Diff line number Diff line change
Expand Up @@ -166,6 +166,9 @@ Bug Fixes

* SOLR-16044: SlowRequest logging is no longer disabled if SolrCore logger set to ERROR (janhoy, hossman)

* SOLR-16415: asyncId must not have '/'; enforce this. Enhance ZK cleanup to process directories
instead of fail. (David Smiley, Paul McArthur)

Dependency Upgrades
---------------------

Expand Down
23 changes: 21 additions & 2 deletions solr/core/src/java/org/apache/solr/cloud/DistributedMap.java
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@
*/
package org.apache.solr.cloud;

import java.lang.invoke.MethodHandles;
import java.util.ArrayList;
import java.util.Collection;
import java.util.List;
Expand All @@ -27,12 +28,17 @@
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.KeeperException.NodeExistsException;
import org.apache.zookeeper.data.Stat;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

/**
* A distributed map. This supports basic map functions e.g. get, put, contains for interaction with
* zk which don't have to be ordered i.e. DistributedQueue.
*/
public class DistributedMap {

private static final Logger log = LoggerFactory.getLogger(MethodHandles.lookup().lookupClass());

protected final String dir;

protected SolrZkClient zookeeper;
Expand All @@ -54,18 +60,26 @@ public DistributedMap(SolrZkClient zookeeper, String dir) {
this.zookeeper = zookeeper;
}

private void assertKeyFormat(String trackingId) {
if (trackingId == null || trackingId.length() == 0 || trackingId.contains("/")) {
throw new SolrException(ErrorCode.BAD_REQUEST, "Unsupported key format: " + trackingId);
}
}

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

/**
* Puts an element in the map only if there isn't one with the same trackingId already
*
* @return True if the the element was added. False if it wasn't (because the key already exists)
* @return True if the element was added. False if it wasn't (because the key already exists)
*/
public boolean putIfAbsent(String trackingId, byte[] data)
throws KeeperException, InterruptedException {
assertKeyFormat(trackingId);
try {
zookeeper.makePath(
dir + "/" + PREFIX + trackingId, data, CreateMode.PERSISTENT, null, true, true);
Expand Down Expand Up @@ -94,10 +108,15 @@ public int size() throws KeeperException, InterruptedException {
* not deleted exception an exception occurred while deleting
*/
public boolean remove(String trackingId) throws KeeperException, InterruptedException {
final var path = dir + "/" + PREFIX + trackingId;
try {
zookeeper.delete(dir + "/" + PREFIX + trackingId, -1, true);
zookeeper.delete(path, -1, true);
} catch (KeeperException.NoNodeException e) {
return false;
} catch (KeeperException.NotEmptyException hack) {
// because dirty data before we enforced the rules on put() (trackingId shouldn't have slash)
log.warn("Cleaning malformed key ID starting with {}", path);
zookeeper.clean(path);
}
return true;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -91,13 +91,11 @@ protected boolean lessThan(Long a, Long b) {
for (String child : children) {
Long id = childToModificationZxid.get(child);
if (id != null && id <= topElementMzxId) {
try {
zookeeper.delete(dir + "/" + child, -1, true);
if (onOverflowObserver != null)
onOverflowObserver.onChildDelete(child.substring(PREFIX.length()));
} catch (KeeperException.NoNodeException ignored) {
// this could happen if multiple threads try to clean the same map
}
String trackingId = child.substring(PREFIX.length());
boolean removed = remove(trackingId);
if (removed && onOverflowObserver != null) {
onOverflowObserver.onChildDelete(trackingId);
} // else, probably multiple threads cleaning the queue simultaneously
}
}
}
Expand Down
31 changes: 31 additions & 0 deletions solr/core/src/test/org/apache/solr/cloud/TestDistributedMap.java
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@
import java.util.concurrent.TimeUnit;
import org.apache.commons.io.file.PathUtils;
import org.apache.solr.SolrTestCaseJ4;
import org.apache.solr.common.SolrException;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.zookeeper.CreateMode;
import org.apache.zookeeper.KeeperException;
Expand Down Expand Up @@ -218,6 +219,36 @@ public void testClear() throws KeeperException, InterruptedException {
}
}

public void testMalformed() throws InterruptedException, KeeperException {
try (SolrZkClient zkClient =
new SolrZkClient.Builder()
.withUrl(zkServer.getZkHost())
.withTimeout(10000, TimeUnit.MILLISECONDS)
.build()) {
String path = getAndMakeInitialPath(zkClient);
DistributedMap map = createMap(zkClient, path);
expectThrows(SolrException.class, () -> map.put("has/slash", new byte[0]));
}
}

public void testRemoveMalformed() throws InterruptedException, KeeperException {
try (SolrZkClient zkClient =
new SolrZkClient.Builder()
.withUrl(zkServer.getZkHost())
.withTimeout(10000, TimeUnit.MILLISECONDS)
.build()) {
String path = getAndMakeInitialPath(zkClient);
// Add a "legacy" / malformed key
final var key = "slash/test/0";
zkClient.makePath(path + "/" + DistributedMap.PREFIX + key, new byte[0], true);

DistributedMap map = createMap(zkClient, path);
assertEquals(1, map.size());
map.remove("slash");
assertEquals(0, map.size());
}
}

protected DistributedMap createMap(SolrZkClient zkClient, String path) {
return new DistributedMap(zkClient, path);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -25,9 +25,11 @@
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Future;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicInteger;
import org.apache.solr.common.cloud.SolrZkClient;
import org.apache.solr.common.util.ExecutorUtil;
import org.apache.solr.common.util.SolrNamedThreadFactory;
import org.apache.zookeeper.KeeperException;

public class TestSizeLimitedDistributedMap extends TestDistributedMap {

Expand Down Expand Up @@ -136,4 +138,32 @@ public void testConcurrentCleanup() throws Exception {
protected DistributedMap createMap(SolrZkClient zkClient, String path) {
return new SizeLimitedDistributedMap(zkClient, path, Overseer.NUM_RESPONSES_TO_STORE, null);
}

public void testCleanupForKeysWithSlashes() throws InterruptedException, KeeperException {
try (SolrZkClient zkClient =
new SolrZkClient.Builder()
.withUrl(zkServer.getZkHost())
.withTimeout(10000, TimeUnit.MILLISECONDS)
.build()) {
int maxEntries = 10;
String path = getAndMakeInitialPath(zkClient);

// Add a "legacy" / malformed key
zkClient.makePath(path + "/" + DistributedMap.PREFIX + "slash/test/0", new byte[0], true);

AtomicInteger overFlowCounter = new AtomicInteger();
DistributedMap map =
new SizeLimitedDistributedMap(
zkClient, path, maxEntries, (element) -> overFlowCounter.incrementAndGet());

// Now add regular keys until we reach the size limit of the map.
// Once we hit the limit, the oldest item (the one we added above with slashes) is deleted,
// but that fails.
for (int i = 1; i <= maxEntries; ++i) {
map.put(String.valueOf(i), new byte[0]);
}
assertTrue(map.size() <= maxEntries);
assertEquals(1, overFlowCounter.get());
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -36,6 +36,7 @@ Because this API has a large number of commands and options, we've grouped the c

Since some collection API calls can be long running tasks (such as SPLITSHARD), you can optionally have the calls run asynchronously.
Specifying `async=<request-id>` enables you to make an asynchronous call, the status of which can be requested using the <<requeststatus,REQUESTSTATUS>> call at any time.
The ID provided can be any string so long as it doesn't have a `/` in it.

As of now, REQUESTSTATUS does not automatically clean up the tracking data structures, meaning the status of completed or failed tasks stays stored in ZooKeeper unless cleared manually.
DELETESTATUS can be used to clear the stored statuses.
Expand Down
Loading