Skip to content

Commit

Permalink
[Remote Store] Add support to disable flush based on translog reader …
Browse files Browse the repository at this point in the history
…count (#14027)

Signed-off-by: Shourya Dutta Biswas <[email protected]>
  • Loading branch information
shourya035 authored Jun 6, 2024
1 parent 298aeda commit b9ca5a8
Show file tree
Hide file tree
Showing 5 changed files with 49 additions and 3 deletions.
1 change: 1 addition & 0 deletions CHANGELOG.md
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/),
- [Remote State] Add async remote state deletion task running on an interval, configurable by a setting ([#13131](https://github.com/opensearch-project/OpenSearch/pull/13131))
- Allow setting query parameters on requests ([#13776](https://github.com/opensearch-project/OpenSearch/issues/13776))
- Add remote routing table for remote state publication with experimental feature flag ([#13304](https://github.com/opensearch-project/OpenSearch/pull/13304))
- [Remote Store] Add support to disable flush based on translog reader count ([#14027](https://github.com/opensearch-project/OpenSearch/pull/14027))

### Dependencies
- Bump `com.github.spullara.mustache.java:compiler` from 0.9.10 to 0.9.13 ([#13329](https://github.com/opensearch-project/OpenSearch/pull/13329), [#13559](https://github.com/opensearch-project/OpenSearch/pull/13559))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -852,7 +852,9 @@ public void testFlushOnTooManyRemoteTranslogFiles() throws Exception {

ClusterUpdateSettingsRequest updateSettingsRequest = new ClusterUpdateSettingsRequest();
updateSettingsRequest.persistentSettings(
Settings.builder().put(RemoteStoreSettings.CLUSTER_REMOTE_MAX_TRANSLOG_READERS.getKey(), "100")
Settings.builder()
.put(RemoteStoreSettings.CLUSTER_REMOTE_MAX_TRANSLOG_READERS.getKey(), "100")
.put(CLUSTER_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.getKey(), "0ms")
);
assertAcked(client().admin().cluster().updateSettings(updateSettingsRequest).actionGet());

Expand Down Expand Up @@ -883,5 +885,27 @@ public void testFlushOnTooManyRemoteTranslogFiles() throws Exception {
assertEquals(totalFiles, 1L);
}
}, 30, TimeUnit.SECONDS);

// Disabling max translog readers
assertAcked(
internalCluster().client()
.admin()
.cluster()
.prepareUpdateSettings()
.setPersistentSettings(Settings.builder().put(RemoteStoreSettings.CLUSTER_REMOTE_MAX_TRANSLOG_READERS.getKey(), "-1"))
.get()
);

// Indexing 500 more docs
for (int i = 0; i < 500; i++) {
indexBulk(INDEX_NAME, 1);
}

// No flush is triggered since max_translog_readers is set to -1
// Total tlog files would be incremented by 500
try (Stream<Path> files = Files.list(translogLocation)) {
long totalFiles = files.filter(f -> f.getFileName().toString().endsWith(Translog.TRANSLOG_FILE_SUFFIX)).count();
assertEquals(totalFiles, 501L);
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -706,6 +706,10 @@ int availablePermits() {
*/
@Override
protected boolean shouldFlush() {
return readers.size() >= translogTransferManager.getMaxRemoteTranslogReadersSettings();
int maxRemoteTlogReaders = translogTransferManager.getMaxRemoteTranslogReadersSettings();
if (maxRemoteTlogReaders == -1) {
return false;
}
return readers.size() >= maxRemoteTlogReaders;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@
*/
@PublicApi(since = "2.14.0")
public class RemoteStoreSettings {
private static final int MIN_CLUSTER_REMOTE_MAX_TRANSLOG_READERS = 100;

/**
* Used to specify the default translog buffer interval for remote store backed indexes.
Expand Down Expand Up @@ -112,7 +113,12 @@ public class RemoteStoreSettings {
public static final Setting<Integer> CLUSTER_REMOTE_MAX_TRANSLOG_READERS = Setting.intSetting(
"cluster.remote_store.translog.max_readers",
1000,
100,
-1,
v -> {
if (v != -1 && v < MIN_CLUSTER_REMOTE_MAX_TRANSLOG_READERS) {
throw new IllegalArgumentException("Cannot set value lower than " + MIN_CLUSTER_REMOTE_MAX_TRANSLOG_READERS);
}
},
Property.Dynamic,
Property.NodeScope
);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -116,4 +116,15 @@ public void testMaxRemoteReferencedTranslogFiles() {
);
assertEquals(500, remoteStoreSettings.getMaxRemoteTranslogReaders());
}

public void testDisableMaxRemoteReferencedTranslogFiles() {
// Test default value
assertEquals(1000, remoteStoreSettings.getMaxRemoteTranslogReaders());

// Test override with valid value
clusterSettings.applySettings(
Settings.builder().put(RemoteStoreSettings.CLUSTER_REMOTE_MAX_TRANSLOG_READERS.getKey(), "-1").build()
);
assertEquals(-1, remoteStoreSettings.getMaxRemoteTranslogReaders());
}
}

0 comments on commit b9ca5a8

Please sign in to comment.