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

Moving get snapshot requests to listener based async calls #8377

Merged
Merged
Show file tree
Hide file tree
Changes from 20 commits
Commits
Show all changes
26 commits
Select commit Hold shift + click to select a range
2eb5e97
Moving get snapshot requests to listener based async calls
indrajohn7 Jun 30, 2023
04569cf
Adding test coverage for TransportGetSnapshotsAction
indrajohn7 Jul 2, 2023
41af99a
Merge remote-tracking branch 'upstream/main' into get_snapshot_async_…
indrajohn7 Jul 2, 2023
312fa4c
Adding Changelog
indrajohn7 Jul 2, 2023
0c47fa5
Merging from main
indrajohn7 Jul 7, 2023
77f4876
Adding TransportGetSnapshotsAction integ tests
cenation07 Jul 10, 2023
f99596a
Merge remote-tracking branch 'upstream/main' into get_snapshot_async_…
cenation07 Jul 10, 2023
dbc4884
Adding Minor test fix
cenation07 Jul 12, 2023
19dc525
Merging from main
cenation07 Jul 12, 2023
1d7bf94
Adding minor typo fix
cenation07 Jul 12, 2023
f0b3834
Removing stale tests
cenation07 Jul 17, 2023
8d14c06
Merging from main
cenation07 Jul 17, 2023
5bb2e39
Cleaning transport tests
cenation07 Jul 17, 2023
ef5aab8
Cleaning up Changelog
cenation07 Jul 17, 2023
b9954c0
Adding getRepositoryData call when isCurrentSnapshotsOnly is false
cenation07 Jul 17, 2023
b947da2
Adding listener callback restricted to nonCurrentSnapshotsOnly
cenation07 Jul 21, 2023
6d25f8e
Merging from main
cenation07 Jul 21, 2023
891648d
Calling getRepositoryData when CurrentSnapshotOnly is false
cenation07 Jul 23, 2023
3661e4f
Merge remote-tracking branch 'upstream/main' into get_snapshot_async_…
cenation07 Jul 23, 2023
c4a668e
refactoring getRepositoryData
cenation07 Jul 24, 2023
7cee224
Adding current in-progress snapshots tests
cenation07 Aug 4, 2023
0ce4fe2
Merging from main
cenation07 Aug 4, 2023
3440908
refactoring tests
cenation07 Aug 5, 2023
63ab279
Merge remote-tracking branch 'upstream/main' into get_snapshot_async_…
cenation07 Aug 5, 2023
bf039ec
Merge remote-tracking branch 'upstream/main' into get_snapshot_async_…
cenation07 Aug 7, 2023
0d7e7fb
Fixing typo error of adding back comment
cenation07 Aug 7, 2023
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
1 change: 1 addition & 0 deletions CHANGELOG.md
Original file line number Diff line number Diff line change
Expand Up @@ -46,6 +46,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/),
- Change http code on create index API with bad input raising NotXContentException from 500 to 400 ([#4773](https://github.com/opensearch-project/OpenSearch/pull/4773))
- Improve summary error message for invalid setting updates ([#4792](https://github.com/opensearch-project/OpenSearch/pull/4792))
- Remote Segment Store Repository setting moved from `index.remote_store.repository` to `index.remote_store.segment.repository` and `cluster.remote_store.repository` to `cluster.remote_store.segment.repository` respectively for Index and Cluster level settings ([#8719](https://github.com/opensearch-project/OpenSearch/pull/8719))
- Removed blocking wait in TransportGetSnapshotsAction which was exhausting generic threadpool ([#8377](https://github.com/opensearch-project/OpenSearch/pull/8377))

### Deprecated

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,8 +37,8 @@
import org.apache.logging.log4j.message.ParameterizedMessage;
import org.apache.lucene.util.CollectionUtil;
import org.opensearch.action.ActionListener;
import org.opensearch.action.StepListener;
import org.opensearch.action.support.ActionFilters;
import org.opensearch.action.support.PlainActionFuture;
import org.opensearch.action.support.clustermanager.TransportClusterManagerNodeAction;
import org.opensearch.cluster.ClusterState;
import org.opensearch.cluster.SnapshotsInProgress;
Expand Down Expand Up @@ -138,57 +138,64 @@
currentSnapshots.add(snapshotInfo);
}

final RepositoryData repositoryData;
final StepListener<RepositoryData> repositoryDataListener = new StepListener<>();
if (isCurrentSnapshotsOnly(request.snapshots()) == false) {
repositoryData = PlainActionFuture.get(fut -> repositoriesService.getRepositoryData(repository, fut));
for (SnapshotId snapshotId : repositoryData.getSnapshotIds()) {
allSnapshotIds.put(snapshotId.getName(), snapshotId);
}
repositoriesService.getRepositoryData(repository, repositoryDataListener);
} else {
repositoryData = null;
// Setting repositoryDataListener response to be null if the request has only current snapshot
repositoryDataListener.onResponse(null);

Check warning on line 146 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L146

Added line #L146 was not covered by tests
}
repositoryDataListener.whenComplete(repositoryData -> {
if (repositoryData != null) {
for (SnapshotId snapshotId : repositoryData.getSnapshotIds()) {
allSnapshotIds.put(snapshotId.getName(), snapshotId);
}
}

final Set<SnapshotId> toResolve = new HashSet<>();
if (isAllSnapshots(request.snapshots())) {
toResolve.addAll(allSnapshotIds.values());
} else {
for (String snapshotOrPattern : request.snapshots()) {
if (GetSnapshotsRequest.CURRENT_SNAPSHOT.equalsIgnoreCase(snapshotOrPattern)) {
toResolve.addAll(currentSnapshots.stream().map(SnapshotInfo::snapshotId).collect(Collectors.toList()));
} else if (Regex.isSimpleMatchPattern(snapshotOrPattern) == false) {
if (allSnapshotIds.containsKey(snapshotOrPattern)) {
toResolve.add(allSnapshotIds.get(snapshotOrPattern));
} else if (request.ignoreUnavailable() == false) {
throw new SnapshotMissingException(repository, snapshotOrPattern);
}
} else {
for (Map.Entry<String, SnapshotId> entry : allSnapshotIds.entrySet()) {
if (Regex.simpleMatch(snapshotOrPattern, entry.getKey())) {
toResolve.add(entry.getValue());
final Set<SnapshotId> toResolve = new HashSet<>();
if (isAllSnapshots(request.snapshots())) {
toResolve.addAll(allSnapshotIds.values());

Check warning on line 157 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L157

Added line #L157 was not covered by tests
} else {
for (String snapshotOrPattern : request.snapshots()) {
if (GetSnapshotsRequest.CURRENT_SNAPSHOT.equalsIgnoreCase(snapshotOrPattern)) {
toResolve.addAll(currentSnapshots.stream().map(SnapshotInfo::snapshotId).collect(Collectors.toList()));

Check warning on line 161 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L161

Added line #L161 was not covered by tests
} else if (Regex.isSimpleMatchPattern(snapshotOrPattern) == false) {
if (allSnapshotIds.containsKey(snapshotOrPattern)) {
toResolve.add(allSnapshotIds.get(snapshotOrPattern));
} else if (request.ignoreUnavailable() == false) {
throw new SnapshotMissingException(repository, snapshotOrPattern);
}
} else {
for (Map.Entry<String, SnapshotId> entry : allSnapshotIds.entrySet()) {
if (Regex.simpleMatch(snapshotOrPattern, entry.getKey())) {
toResolve.add(entry.getValue());

Check warning on line 171 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L171

Added line #L171 was not covered by tests
}
}
}
}
}

if (toResolve.isEmpty() && request.ignoreUnavailable() == false && isCurrentSnapshotsOnly(request.snapshots()) == false) {
throw new SnapshotMissingException(repository, request.snapshots()[0]);
if (toResolve.isEmpty()
&& request.ignoreUnavailable() == false
&& isCurrentSnapshotsOnly(request.snapshots()) == false) {
throw new SnapshotMissingException(repository, request.snapshots()[0]);

Check warning on line 180 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L180

Added line #L180 was not covered by tests
}
}
}

final List<SnapshotInfo> snapshotInfos;
if (request.verbose()) {
snapshotInfos = snapshots(snapshotsInProgress, repository, new ArrayList<>(toResolve), request.ignoreUnavailable());
} else {
if (repositoryData != null) {
// want non-current snapshots as well, which are found in the repository data
snapshotInfos = buildSimpleSnapshotInfos(toResolve, repositoryData, currentSnapshots);
final List<SnapshotInfo> snapshotInfos;
if (request.verbose()) {
snapshotInfos = snapshots(snapshotsInProgress, repository, new ArrayList<>(toResolve), request.ignoreUnavailable());
} else {
// only want current snapshots
snapshotInfos = currentSnapshots.stream().map(SnapshotInfo::basic).collect(Collectors.toList());
CollectionUtil.timSort(snapshotInfos);
if (repositoryData != null) {
// want non-current snapshots as well, which are found in the repository data
snapshotInfos = buildSimpleSnapshotInfos(toResolve, repositoryData, currentSnapshots);
} else {
// only want current snapshots
snapshotInfos = currentSnapshots.stream().map(SnapshotInfo::basic).collect(Collectors.toList());
CollectionUtil.timSort(snapshotInfos);

Check warning on line 194 in server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java

View check run for this annotation

Codecov / codecov/patch

server/src/main/java/org/opensearch/action/admin/cluster/snapshots/get/TransportGetSnapshotsAction.java#L193-L194

Added lines #L193 - L194 were not covered by tests
}
}
}
listener.onResponse(new GetSnapshotsResponse(snapshotInfos));
listener.onResponse(new GetSnapshotsResponse(snapshotInfos));
}, listener::onFailure);
} catch (Exception e) {
listener.onFailure(e);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -56,6 +56,9 @@
import org.opensearch.action.admin.cluster.snapshots.delete.DeleteSnapshotAction;
import org.opensearch.action.admin.cluster.snapshots.delete.DeleteSnapshotRequest;
import org.opensearch.action.admin.cluster.snapshots.delete.TransportDeleteSnapshotAction;
import org.opensearch.action.admin.cluster.snapshots.get.GetSnapshotsAction;
import org.opensearch.action.admin.cluster.snapshots.get.GetSnapshotsRequest;
import org.opensearch.action.admin.cluster.snapshots.get.TransportGetSnapshotsAction;
import org.opensearch.action.admin.cluster.snapshots.restore.RestoreSnapshotAction;
import org.opensearch.action.admin.cluster.snapshots.restore.RestoreSnapshotRequest;
import org.opensearch.action.admin.cluster.snapshots.restore.RestoreSnapshotResponse;
Expand Down Expand Up @@ -635,15 +638,13 @@ public void testConcurrentSnapshotCreateAndDeleteOther() {
);

final StepListener<CreateSnapshotResponse> createOtherSnapshotResponseStepListener = new StepListener<>();

continueOrDie(
createSnapshotResponseStepListener,
createSnapshotResponse -> client().admin()
.cluster()
.prepareCreateSnapshot(repoName, "snapshot-2")
.execute(createOtherSnapshotResponseStepListener)
);

final StepListener<AcknowledgedResponse> deleteSnapshotStepListener = new StepListener<>();

continueOrDie(
Expand Down Expand Up @@ -676,7 +677,6 @@ public void testConcurrentSnapshotCreateAndDeleteOther() {
Collection<SnapshotId> snapshotIds = getRepositoryData(repository).getSnapshotIds();
// We end up with two snapshots no matter if the delete worked out or not
indrajohn7 marked this conversation as resolved.
Show resolved Hide resolved
assertThat(snapshotIds, hasSize(2));

for (SnapshotId snapshotId : snapshotIds) {
final SnapshotInfo snapshotInfo = repository.getSnapshotInfo(snapshotId);
assertEquals(SnapshotState.SUCCESS, snapshotInfo.state());
Expand All @@ -686,6 +686,55 @@ public void testConcurrentSnapshotCreateAndDeleteOther() {
}
}

public void testTransportGetSnapshotsAction() {
indrajohn7 marked this conversation as resolved.
Show resolved Hide resolved
setupTestCluster(randomFrom(1, 3, 5), randomIntBetween(2, 10));

String repoName = "repo";
final String[] snapshotsList = { "snapshot-1", "snapshot-2" };
final String[] indexList = { "index-1", "index-2" };
final boolean[] snapshotRequestMode = { true, false };
final int shards = randomIntBetween(1, 10);

TestClusterNodes.TestClusterNode clusterManagerNode = testClusterNodes.currentClusterManager(
testClusterNodes.nodes.values().iterator().next().clusterService.state()
);

for (int i = 0; i < snapshotsList.length; i++) {
final StepListener<CreateSnapshotResponse> createSnapshotResponseStepListener = new StepListener<>();
final String snapshot = snapshotsList[i];
final String index = indexList[i];
continueOrDie(
createRepoAndIndex(repoName, index, shards),
createSnapshotResponse -> client().admin()
.cluster()
.prepareCreateSnapshot(repoName, snapshot)
.setWaitForCompletion(true)
.execute(createSnapshotResponseStepListener)
);
}
deterministicTaskQueue.runAllRunnableTasks();

TransportAction getSnapshotsAction = clusterManagerNode.actions.get(GetSnapshotsAction.INSTANCE);
TransportGetSnapshotsAction transportGetSnapshotsAction = (TransportGetSnapshotsAction) getSnapshotsAction;
for (boolean mode : snapshotRequestMode) {
GetSnapshotsRequest repoSnapshotRequest = new GetSnapshotsRequest().repository(repoName)
.snapshots(snapshotsList)
.ignoreUnavailable(mode)
.verbose(mode);

transportGetSnapshotsAction.execute(null, repoSnapshotRequest, ActionListener.wrap(repoSnapshotResponse -> {
assertNotNull("Snapshot list should be registered", repoSnapshotResponse.getSnapshots());
indrajohn7 marked this conversation as resolved.
Show resolved Hide resolved
assertThat(repoSnapshotResponse.getSnapshots(), hasSize(2));
List<SnapshotInfo> snapshotInfos = repoSnapshotResponse.getSnapshots();
for (SnapshotInfo snapshotInfo : snapshotInfos) {
assertEquals(SnapshotState.SUCCESS, snapshotInfo.state());
assertEquals(0, snapshotInfo.failedShards());
assertTrue(Arrays.stream(snapshotsList).anyMatch(snapshotInfo.snapshotId().getName()::equals));
}
}, exception -> { throw new AssertionError(exception); }));
}
}

public void testBulkSnapshotDeleteWithAbort() {
setupTestCluster(randomFrom(1, 3, 5), randomIntBetween(2, 10));

Expand Down Expand Up @@ -1625,7 +1674,6 @@ public TestClusterNode currentClusterManager(ClusterState state) {
}

private final class TestClusterNode {

private final Logger logger = LogManager.getLogger(TestClusterNode.class);

private final NamedWriteableRegistry namedWriteableRegistry = new NamedWriteableRegistry(
Expand Down Expand Up @@ -1669,6 +1717,8 @@ private final class TestClusterNode {

private Coordinator coordinator;

private Map<ActionType, TransportAction> actions = new HashMap<>();

TestClusterNode(DiscoveryNode node) throws IOException {
this.node = node;
final Environment environment = createEnvironment(node.getName());
Expand Down Expand Up @@ -1904,7 +1954,7 @@ public void onFailure(final Exception e) {
SegmentReplicationCheckpointPublisher.EMPTY,
mock(RemoteRefreshSegmentPressureService.class)
);
Map<ActionType, TransportAction> actions = new HashMap<>();

final SystemIndices systemIndices = new SystemIndices(emptyMap());
final ShardLimitValidator shardLimitValidator = new ShardLimitValidator(settings, clusterService, systemIndices);
final MetadataCreateIndexService metadataCreateIndexService = new MetadataCreateIndexService(
Expand Down Expand Up @@ -2135,6 +2185,17 @@ public void onFailure(final Exception e) {
indexNameExpressionResolver
)
);
actions.put(
indrajohn7 marked this conversation as resolved.
Show resolved Hide resolved
GetSnapshotsAction.INSTANCE,
new TransportGetSnapshotsAction(
transportService,
clusterService,
threadPool,
repositoriesService,
actionFilters,
indexNameExpressionResolver
)
);
actions.put(
ClusterStateAction.INSTANCE,
new TransportClusterStateAction(
Expand Down
Loading