From ed54727de11f6ec1d29959b4f2d615f9e283bbbf Mon Sep 17 00:00:00 2001 From: Ashish Singh Date: Sun, 1 Sep 2024 19:34:03 +0530 Subject: [PATCH] Add prefix support to hashed prefix & infix path types on remote store Signed-off-by: Ashish Singh --- CHANGELOG.md | 1 + .../TransportCleanupRepositoryAction.java | 6 +- .../index/remote/RemoteIndexPath.java | 11 +++- .../index/remote/RemoteIndexPathUploader.java | 9 ++- .../index/remote/RemoteStoreEnums.java | 8 ++- .../index/remote/RemoteStorePathStrategy.java | 12 ++++ .../opensearch/index/shard/IndexShard.java | 10 +-- .../opensearch/index/shard/StoreRecovery.java | 8 ++- .../RemoteSegmentStoreDirectoryFactory.java | 14 ++++- .../RemoteStoreLockManagerFactory.java | 11 +++- .../index/translog/RemoteFsTranslog.java | 2 + .../indices/RemoteStoreSettings.java | 33 ++++++++++ .../main/java/org/opensearch/node/Node.java | 12 ++-- .../blobstore/BlobStoreRepository.java | 7 ++- .../snapshots/SnapshotsService.java | 6 +- .../GatewayMetaStatePersistedStateTests.java | 11 +++- .../RemoteClusterStateServiceTests.java | 61 +++++++++++++++++-- .../opensearch/index/IndexModuleTests.java | 2 +- .../index/remote/RemoteIndexPathTests.java | 16 +++-- .../remote/RemoteIndexPathUploaderTests.java | 22 ++++--- .../index/shard/StoreRecoveryTests.java | 5 +- ...moteSegmentStoreDirectoryFactoryTests.java | 7 ++- .../RemoteStoreLockManagerFactoryTests.java | 3 +- .../snapshots/SnapshotResiliencyTests.java | 8 ++- .../blobstore/BlobStoreTestUtil.java | 5 ++ 25 files changed, 237 insertions(+), 53 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 67091eaa6b850..870ab6f51c485 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -41,6 +41,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - [Workload Management] Add query group level failure tracking ([#15227](https://github.com/opensearch-project/OpenSearch/pull/15527)) - Add support to upload snapshot shard blobs with hashed prefix ([#15426](https://github.com/opensearch-project/OpenSearch/pull/15426)) - [Remote Publication] Add remote download stats ([#15291](https://github.com/opensearch-project/OpenSearch/pull/15291))) +- Add prefix support to hashed prefix & infix path types on remote store ([#15557](https://github.com/opensearch-project/OpenSearch/pull/15557)) ### Dependencies - Bump `netty` from 4.1.111.Final to 4.1.112.Final ([#15081](https://github.com/opensearch-project/OpenSearch/pull/15081)) diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/repositories/cleanup/TransportCleanupRepositoryAction.java b/server/src/main/java/org/opensearch/action/admin/cluster/repositories/cleanup/TransportCleanupRepositoryAction.java index 774bffa10da4f..4e610142fe3b8 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/repositories/cleanup/TransportCleanupRepositoryAction.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/repositories/cleanup/TransportCleanupRepositoryAction.java @@ -53,6 +53,7 @@ import org.opensearch.core.action.ActionListener; import org.opensearch.core.common.io.stream.StreamInput; import org.opensearch.index.store.lockmanager.RemoteStoreLockManagerFactory; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.Repository; import org.opensearch.repositories.RepositoryCleanupResult; @@ -109,7 +110,8 @@ public TransportCleanupRepositoryAction( SnapshotsService snapshotsService, ThreadPool threadPool, ActionFilters actionFilters, - IndexNameExpressionResolver indexNameExpressionResolver + IndexNameExpressionResolver indexNameExpressionResolver, + RemoteStoreSettings remoteStoreSettings ) { super( CleanupRepositoryAction.NAME, @@ -122,7 +124,7 @@ public TransportCleanupRepositoryAction( ); this.repositoriesService = repositoriesService; this.snapshotsService = snapshotsService; - this.remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(() -> repositoriesService); + this.remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(() -> repositoriesService, remoteStoreSettings); // We add a state applier that will remove any dangling repository cleanup actions on cluster-manager failover. // This is safe to do since cleanups will increment the repository state id before executing any operations to prevent concurrent // operations from corrupting the repository. This is the same safety mechanism used by snapshot deletes. diff --git a/server/src/main/java/org/opensearch/index/remote/RemoteIndexPath.java b/server/src/main/java/org/opensearch/index/remote/RemoteIndexPath.java index 899ff16c9d607..e686bb60140c0 100644 --- a/server/src/main/java/org/opensearch/index/remote/RemoteIndexPath.java +++ b/server/src/main/java/org/opensearch/index/remote/RemoteIndexPath.java @@ -20,6 +20,7 @@ import org.opensearch.index.remote.RemoteStoreEnums.PathType; import org.opensearch.index.remote.RemoteStorePathStrategy.PathInput; import org.opensearch.index.remote.RemoteStorePathStrategy.ShardDataPathInput; +import org.opensearch.indices.RemoteStoreSettings; import java.io.IOException; import java.util.Collections; @@ -68,6 +69,7 @@ public class RemoteIndexPath implements ToXContentFragment { private final Iterable basePath; private final PathType pathType; private final PathHashAlgorithm pathHashAlgorithm; + private final RemoteStoreSettings remoteStoreSettings; /** * This keeps the map of paths that would be present in the content of the index path file. For eg - It is possible @@ -82,7 +84,8 @@ public RemoteIndexPath( Iterable basePath, PathType pathType, PathHashAlgorithm pathHashAlgorithm, - Map> pathCreationMap + Map> pathCreationMap, + RemoteStoreSettings remoteStoreSettings ) { if (Objects.isNull(pathCreationMap) || Objects.isNull(pathType) @@ -119,6 +122,7 @@ public RemoteIndexPath( this.pathType = pathType; this.pathHashAlgorithm = pathHashAlgorithm; this.pathCreationMap = pathCreationMap; + this.remoteStoreSettings = remoteStoreSettings; } @Override @@ -148,6 +152,11 @@ public XContentBuilder toXContent(XContentBuilder builder, Params params) throws .shardId(Integer.toString(shardNo)) .dataCategory(dataCategory) .dataType(type) + .fixedPrefix( + dataCategory == TRANSLOG + ? remoteStoreSettings.getTranslogPathFixedPrefix() + : remoteStoreSettings.getSegmentsPathFixedPrefix() + ) .build(); builder.value(pathType.path(pathInput, pathHashAlgorithm).buildAsString()); } diff --git a/server/src/main/java/org/opensearch/index/remote/RemoteIndexPathUploader.java b/server/src/main/java/org/opensearch/index/remote/RemoteIndexPathUploader.java index d5617bdfd94a7..5878dff03acc2 100644 --- a/server/src/main/java/org/opensearch/index/remote/RemoteIndexPathUploader.java +++ b/server/src/main/java/org/opensearch/index/remote/RemoteIndexPathUploader.java @@ -25,6 +25,7 @@ import org.opensearch.gateway.remote.IndexMetadataUploadListener; import org.opensearch.gateway.remote.RemoteStateTransferException; import org.opensearch.index.remote.RemoteStoreEnums.PathType; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.node.Node; import org.opensearch.node.remotestore.RemoteStoreNodeAttribute; import org.opensearch.repositories.RepositoriesService; @@ -79,6 +80,7 @@ public class RemoteIndexPathUploader extends IndexMetadataUploadListener { private final Settings settings; private final boolean isRemoteDataAttributePresent; private final boolean isTranslogSegmentRepoSame; + private final RemoteStoreSettings remoteStoreSettings; private final Supplier repositoriesService; private volatile TimeValue metadataUploadTimeout; @@ -89,7 +91,8 @@ public RemoteIndexPathUploader( ThreadPool threadPool, Settings settings, Supplier repositoriesService, - ClusterSettings clusterSettings + ClusterSettings clusterSettings, + RemoteStoreSettings remoteStoreSettings ) { super(threadPool, ThreadPool.Names.GENERIC); this.settings = Objects.requireNonNull(settings); @@ -100,6 +103,7 @@ public RemoteIndexPathUploader( Objects.requireNonNull(clusterSettings); metadataUploadTimeout = clusterSettings.get(GLOBAL_METADATA_UPLOAD_TIMEOUT_SETTING); clusterSettings.addSettingsUpdateConsumer(GLOBAL_METADATA_UPLOAD_TIMEOUT_SETTING, this::setMetadataUploadTimeout); + this.remoteStoreSettings = remoteStoreSettings; } @Override @@ -208,7 +212,8 @@ private void writePathToRemoteStore( basePath, pathType, hashAlgorithm, - pathCreationMap + pathCreationMap, + remoteStoreSettings ); String fileName = generateFileName(indexUUID, idxMD.getVersion(), remoteIndexPath.getVersion()); REMOTE_INDEX_PATH_FORMAT.writeAsyncWithUrgentPriority(remoteIndexPath, blobContainer, fileName, actionListener); diff --git a/server/src/main/java/org/opensearch/index/remote/RemoteStoreEnums.java b/server/src/main/java/org/opensearch/index/remote/RemoteStoreEnums.java index 0af0d07e11597..59f210664d6f8 100644 --- a/server/src/main/java/org/opensearch/index/remote/RemoteStoreEnums.java +++ b/server/src/main/java/org/opensearch/index/remote/RemoteStoreEnums.java @@ -107,7 +107,11 @@ boolean requiresHashAlgorithm() { @Override public BlobPath generatePath(PathInput pathInput, PathHashAlgorithm hashAlgorithm) { assert Objects.nonNull(hashAlgorithm) : "hashAlgorithm is expected to be non-null"; - return BlobPath.cleanPath().add(hashAlgorithm.hash(pathInput)).add(pathInput.basePath()).add(pathInput.fixedSubPath()); + return BlobPath.cleanPath() + .add(pathInput.fixedPrefix()) + .add(hashAlgorithm.hash(pathInput)) + .add(pathInput.basePath()) + .add(pathInput.fixedSubPath()); } @Override @@ -119,7 +123,7 @@ boolean requiresHashAlgorithm() { @Override public BlobPath generatePath(PathInput pathInput, PathHashAlgorithm hashAlgorithm) { assert Objects.nonNull(hashAlgorithm) : "hashAlgorithm is expected to be non-null"; - return pathInput.basePath().add(hashAlgorithm.hash(pathInput)).add(pathInput.fixedSubPath()); + return pathInput.basePath().add(pathInput.fixedPrefix()).add(hashAlgorithm.hash(pathInput)).add(pathInput.fixedSubPath()); } @Override diff --git a/server/src/main/java/org/opensearch/index/remote/RemoteStorePathStrategy.java b/server/src/main/java/org/opensearch/index/remote/RemoteStorePathStrategy.java index 7bead3e75f12e..ee28f47664614 100644 --- a/server/src/main/java/org/opensearch/index/remote/RemoteStorePathStrategy.java +++ b/server/src/main/java/org/opensearch/index/remote/RemoteStorePathStrategy.java @@ -83,10 +83,12 @@ public BlobPath generatePath(PathInput pathInput) { public static class PathInput { private final BlobPath basePath; private final String indexUUID; + private final String fixedPrefix; public PathInput(Builder builder) { this.basePath = Objects.requireNonNull(builder.basePath); this.indexUUID = Objects.requireNonNull(builder.indexUUID); + this.fixedPrefix = Objects.isNull(builder.fixedPrefix) ? "" : builder.fixedPrefix; } BlobPath basePath() { @@ -97,6 +99,10 @@ String indexUUID() { return indexUUID; } + String fixedPrefix() { + return fixedPrefix; + } + BlobPath fixedSubPath() { return BlobPath.cleanPath().add(indexUUID); } @@ -126,6 +132,7 @@ public void assertIsValid() { public static class Builder> { private BlobPath basePath; private String indexUUID; + private String fixedPrefix; public T basePath(BlobPath basePath) { this.basePath = basePath; @@ -137,6 +144,11 @@ public T indexUUID(String indexUUID) { return self(); } + public T fixedPrefix(String fixedPrefix) { + this.fixedPrefix = fixedPrefix; + return self(); + } + protected T self() { return (T) this; } diff --git a/server/src/main/java/org/opensearch/index/shard/IndexShard.java b/server/src/main/java/org/opensearch/index/shard/IndexShard.java index 4b18df7a7829d..bc57286e5c068 100644 --- a/server/src/main/java/org/opensearch/index/shard/IndexShard.java +++ b/server/src/main/java/org/opensearch/index/shard/IndexShard.java @@ -2857,7 +2857,7 @@ public void recoverFromLocalShards( // we are the first primary, recover from the gateway // if its post api allocation, the index should exists assert shardRouting.primary() : "recover from local shards only makes sense if the shard is a primary shard"; - StoreRecovery storeRecovery = new StoreRecovery(shardId, logger); + StoreRecovery storeRecovery = new StoreRecovery(shardId, logger, remoteStoreSettings); storeRecovery.recoverFromLocalShards(mappingUpdateConsumer, this, snapshots, recoveryListener); success = true; } finally { @@ -2872,13 +2872,13 @@ public void recoverFromStore(ActionListener listener) { // if its post api allocation, the index should exists assert shardRouting.primary() : "recover from store only makes sense if the shard is a primary shard"; assert shardRouting.initializing() : "can only start recovery on initializing shard"; - StoreRecovery storeRecovery = new StoreRecovery(shardId, logger); + StoreRecovery storeRecovery = new StoreRecovery(shardId, logger, remoteStoreSettings); storeRecovery.recoverFromStore(this, listener); } public void restoreFromRemoteStore(ActionListener listener) { assert shardRouting.primary() : "recover from store only makes sense if the shard is a primary shard"; - StoreRecovery storeRecovery = new StoreRecovery(shardId, logger); + StoreRecovery storeRecovery = new StoreRecovery(shardId, logger, remoteStoreSettings); storeRecovery.recoverFromRemoteStore(this, listener); } @@ -2891,7 +2891,7 @@ public void restoreFromSnapshotAndRemoteStore( assert shardRouting.primary() : "recover from store only makes sense if the shard is a primary shard"; assert recoveryState.getRecoverySource().getType() == RecoverySource.Type.SNAPSHOT : "invalid recovery type: " + recoveryState.getRecoverySource(); - StoreRecovery storeRecovery = new StoreRecovery(shardId, logger); + StoreRecovery storeRecovery = new StoreRecovery(shardId, logger, remoteStoreSettings); storeRecovery.recoverFromSnapshotAndRemoteStore(this, repository, repositoriesService, listener, threadPool); } catch (Exception e) { listener.onFailure(e); @@ -2903,7 +2903,7 @@ public void restoreFromRepository(Repository repository, ActionListener assert shardRouting.primary() : "recover from store only makes sense if the shard is a primary shard"; assert recoveryState.getRecoverySource().getType() == RecoverySource.Type.SNAPSHOT : "invalid recovery type: " + recoveryState.getRecoverySource(); - StoreRecovery storeRecovery = new StoreRecovery(shardId, logger); + StoreRecovery storeRecovery = new StoreRecovery(shardId, logger, remoteStoreSettings); storeRecovery.recoverFromRepository(this, repository, listener); } catch (Exception e) { listener.onFailure(e); diff --git a/server/src/main/java/org/opensearch/index/shard/StoreRecovery.java b/server/src/main/java/org/opensearch/index/shard/StoreRecovery.java index 8d689e8769728..13c1ac635b49f 100644 --- a/server/src/main/java/org/opensearch/index/shard/StoreRecovery.java +++ b/server/src/main/java/org/opensearch/index/shard/StoreRecovery.java @@ -67,6 +67,7 @@ import org.opensearch.index.translog.Checkpoint; import org.opensearch.index.translog.Translog; import org.opensearch.index.translog.TranslogHeader; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.indices.recovery.RecoveryState; import org.opensearch.indices.replication.common.ReplicationLuceneIndex; import org.opensearch.repositories.IndexId; @@ -99,10 +100,12 @@ final class StoreRecovery { private final Logger logger; private final ShardId shardId; + private final RemoteStoreSettings remoteStoreSettings; - StoreRecovery(ShardId shardId, Logger logger) { + StoreRecovery(ShardId shardId, Logger logger, RemoteStoreSettings remoteStoreSettings) { this.logger = logger; this.shardId = shardId; + this.remoteStoreSettings = remoteStoreSettings; } /** @@ -397,7 +400,8 @@ void recoverFromSnapshotAndRemoteStore( RemoteSegmentStoreDirectoryFactory directoryFactory = new RemoteSegmentStoreDirectoryFactory( () -> repositoriesService, - threadPool + threadPool, + remoteStoreSettings ); RemoteSegmentStoreDirectory sourceRemoteDirectory = (RemoteSegmentStoreDirectory) directoryFactory.newDirectory( remoteStoreRepository, diff --git a/server/src/main/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactory.java b/server/src/main/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactory.java index 3f6f4eeeef87b..fed095f2e4804 100644 --- a/server/src/main/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactory.java +++ b/server/src/main/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactory.java @@ -17,6 +17,7 @@ import org.opensearch.index.shard.ShardPath; import org.opensearch.index.store.lockmanager.RemoteStoreLockManager; import org.opensearch.index.store.lockmanager.RemoteStoreLockManagerFactory; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.plugins.IndexStorePlugin; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.Repository; @@ -40,11 +41,17 @@ @PublicApi(since = "2.3.0") public class RemoteSegmentStoreDirectoryFactory implements IndexStorePlugin.DirectoryFactory { private final Supplier repositoriesService; + private final RemoteStoreSettings remoteStoreSettings; private final ThreadPool threadPool; - public RemoteSegmentStoreDirectoryFactory(Supplier repositoriesService, ThreadPool threadPool) { + public RemoteSegmentStoreDirectoryFactory( + Supplier repositoriesService, + ThreadPool threadPool, + RemoteStoreSettings remoteStoreSettings + ) { this.repositoriesService = repositoriesService; + this.remoteStoreSettings = remoteStoreSettings; this.threadPool = threadPool; } @@ -71,6 +78,7 @@ public Directory newDirectory(String repositoryName, String indexUUID, ShardId s .shardId(shardIdStr) .dataCategory(SEGMENTS) .dataType(DATA) + .fixedPrefix(remoteStoreSettings.getSegmentsPathFixedPrefix()) .build(); // Derive the path for data directory of SEGMENTS BlobPath dataPath = pathStrategy.generatePath(dataPathInput); @@ -87,6 +95,7 @@ public Directory newDirectory(String repositoryName, String indexUUID, ShardId s .shardId(shardIdStr) .dataCategory(SEGMENTS) .dataType(METADATA) + .fixedPrefix(remoteStoreSettings.getSegmentsPathFixedPrefix()) .build(); // Derive the path for metadata directory of SEGMENTS BlobPath mdPath = pathStrategy.generatePath(mdPathInput); @@ -98,7 +107,8 @@ public Directory newDirectory(String repositoryName, String indexUUID, ShardId s repositoryName, indexUUID, shardIdStr, - pathStrategy + pathStrategy, + remoteStoreSettings ); return new RemoteSegmentStoreDirectory(dataDirectory, metadataDirectory, mdLockManager, threadPool, shardId); diff --git a/server/src/main/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactory.java b/server/src/main/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactory.java index 993c1bbdf033f..a9eed80100bd9 100644 --- a/server/src/main/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactory.java +++ b/server/src/main/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactory.java @@ -13,6 +13,7 @@ import org.opensearch.common.blobstore.BlobPath; import org.opensearch.index.remote.RemoteStorePathStrategy; import org.opensearch.index.store.RemoteBufferedOutputDirectory; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.Repository; import org.opensearch.repositories.RepositoryMissingException; @@ -31,9 +32,11 @@ @PublicApi(since = "2.8.0") public class RemoteStoreLockManagerFactory { private final Supplier repositoriesService; + private final RemoteStoreSettings remoteStoreSettings; - public RemoteStoreLockManagerFactory(Supplier repositoriesService) { + public RemoteStoreLockManagerFactory(Supplier repositoriesService, RemoteStoreSettings remoteStoreSettings) { this.repositoriesService = repositoriesService; + this.remoteStoreSettings = remoteStoreSettings; } public RemoteStoreLockManager newLockManager( @@ -42,7 +45,7 @@ public RemoteStoreLockManager newLockManager( String shardId, RemoteStorePathStrategy pathStrategy ) { - return newLockManager(repositoriesService.get(), repositoryName, indexUUID, shardId, pathStrategy); + return newLockManager(repositoriesService.get(), repositoryName, indexUUID, shardId, pathStrategy, remoteStoreSettings); } public static RemoteStoreMetadataLockManager newLockManager( @@ -50,7 +53,8 @@ public static RemoteStoreMetadataLockManager newLockManager( String repositoryName, String indexUUID, String shardId, - RemoteStorePathStrategy pathStrategy + RemoteStorePathStrategy pathStrategy, + RemoteStoreSettings remoteStoreSettings ) { try (Repository repository = repositoriesService.repository(repositoryName)) { assert repository instanceof BlobStoreRepository : "repository should be instance of BlobStoreRepository"; @@ -62,6 +66,7 @@ public static RemoteStoreMetadataLockManager newLockManager( .shardId(shardId) .dataCategory(SEGMENTS) .dataType(LOCK_FILES) + .fixedPrefix(remoteStoreSettings.getSegmentsPathFixedPrefix()) .build(); BlobPath lockDirectoryPath = pathStrategy.generatePath(lockFilesPathInput); BlobContainer lockDirectoryBlobContainer = ((BlobStoreRepository) repository).blobStore().blobContainer(lockDirectoryPath); diff --git a/server/src/main/java/org/opensearch/index/translog/RemoteFsTranslog.java b/server/src/main/java/org/opensearch/index/translog/RemoteFsTranslog.java index f6afc4c508720..66eea2a51b36a 100644 --- a/server/src/main/java/org/opensearch/index/translog/RemoteFsTranslog.java +++ b/server/src/main/java/org/opensearch/index/translog/RemoteFsTranslog.java @@ -392,6 +392,7 @@ public static TranslogTransferManager buildTranslogTransferManager( .shardId(shardIdStr) .dataCategory(TRANSLOG) .dataType(DATA) + .fixedPrefix(remoteStoreSettings.getTranslogPathFixedPrefix()) .build(); BlobPath dataPath = pathStrategy.generatePath(dataPathInput); RemoteStorePathStrategy.ShardDataPathInput mdPathInput = RemoteStorePathStrategy.ShardDataPathInput.builder() @@ -400,6 +401,7 @@ public static TranslogTransferManager buildTranslogTransferManager( .shardId(shardIdStr) .dataCategory(TRANSLOG) .dataType(METADATA) + .fixedPrefix(remoteStoreSettings.getTranslogPathFixedPrefix()) .build(); BlobPath mdPath = pathStrategy.generatePath(mdPathInput); BlobStoreTransferService transferService = new BlobStoreTransferService(blobStoreRepository.blobStore(), threadPool); diff --git a/server/src/main/java/org/opensearch/indices/RemoteStoreSettings.java b/server/src/main/java/org/opensearch/indices/RemoteStoreSettings.java index 00d9ffdac6086..78be13bcfc919 100644 --- a/server/src/main/java/org/opensearch/indices/RemoteStoreSettings.java +++ b/server/src/main/java/org/opensearch/indices/RemoteStoreSettings.java @@ -164,6 +164,26 @@ public class RemoteStoreSettings { Setting.Property.NodeScope ); + /** + * Controls the fixed prefix for the translog path on remote store. + */ + public static final Setting CLUSTER_REMOTE_STORE_TRANSLOG_PATH_PREFIX_CHAR = Setting.simpleString( + "cluster.remote_store.translog.path.prefix", + "", + Property.NodeScope, + Property.Final + ); + + /** + * Controls the fixed prefix for the segments path on remote store. + */ + public static final Setting CLUSTER_REMOTE_STORE_SEGMENTS_PATH_PREFIX_CHAR = Setting.simpleString( + "cluster.remote_store.segments.path.prefix", + "", + Property.NodeScope, + Property.Final + ); + private volatile TimeValue clusterRemoteTranslogBufferInterval; private volatile int minRemoteSegmentMetadataFiles; private volatile TimeValue clusterRemoteTranslogTransferTimeout; @@ -175,6 +195,8 @@ public class RemoteStoreSettings { private static volatile boolean isPinnedTimestampsEnabled; private static volatile TimeValue pinnedTimestampsSchedulerInterval; private static volatile TimeValue pinnedTimestampsLookbackInterval; + private final String translogPathFixedPrefix; + private final String segmentsPathFixedPrefix; public RemoteStoreSettings(Settings settings, ClusterSettings clusterSettings) { clusterRemoteTranslogBufferInterval = CLUSTER_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.get(settings); @@ -216,6 +238,9 @@ public RemoteStoreSettings(Settings settings, ClusterSettings clusterSettings) { pinnedTimestampsSchedulerInterval = CLUSTER_REMOTE_STORE_PINNED_TIMESTAMP_SCHEDULER_INTERVAL.get(settings); pinnedTimestampsLookbackInterval = CLUSTER_REMOTE_STORE_PINNED_TIMESTAMP_LOOKBACK_INTERVAL.get(settings); isPinnedTimestampsEnabled = CLUSTER_REMOTE_STORE_PINNED_TIMESTAMP_ENABLED.get(settings); + + translogPathFixedPrefix = CLUSTER_REMOTE_STORE_TRANSLOG_PATH_PREFIX_CHAR.get(settings); + segmentsPathFixedPrefix = CLUSTER_REMOTE_STORE_SEGMENTS_PATH_PREFIX_CHAR.get(settings); } public TimeValue getClusterRemoteTranslogBufferInterval() { @@ -300,4 +325,12 @@ public static void setPinnedTimestampsLookbackInterval(TimeValue pinnedTimestamp public static boolean isPinnedTimestampsEnabled() { return isPinnedTimestampsEnabled; } + + public String getTranslogPathFixedPrefix() { + return translogPathFixedPrefix; + } + + public String getSegmentsPathFixedPrefix() { + return segmentsPathFixedPrefix; + } } diff --git a/server/src/main/java/org/opensearch/node/Node.java b/server/src/main/java/org/opensearch/node/Node.java index 6ce7e2678d6d0..955df4c32cdba 100644 --- a/server/src/main/java/org/opensearch/node/Node.java +++ b/server/src/main/java/org/opensearch/node/Node.java @@ -791,6 +791,7 @@ protected Node( clusterService.getClusterSettings(), threadPool::relativeTimeInMillis ); + final RemoteStoreSettings remoteStoreSettings = new RemoteStoreSettings(settings, settingsModule.getClusterSettings()); final RemoteClusterStateService remoteClusterStateService; final RemoteClusterStateCleanupManager remoteClusterStateCleanupManager; final RemoteIndexPathUploader remoteIndexPathUploader; @@ -799,7 +800,8 @@ protected Node( threadPool, settings, repositoriesServiceReference::get, - clusterService.getClusterSettings() + clusterService.getClusterSettings(), + remoteStoreSettings ); remoteClusterStateService = new RemoteClusterStateService( nodeEnvironment.nodeId(), @@ -871,12 +873,12 @@ protected Node( final RecoverySettings recoverySettings = new RecoverySettings(settings, settingsModule.getClusterSettings()); - final RemoteStoreSettings remoteStoreSettings = new RemoteStoreSettings(settings, settingsModule.getClusterSettings()); final CompositeIndexSettings compositeIndexSettings = new CompositeIndexSettings(settings, settingsModule.getClusterSettings()); final IndexStorePlugin.DirectoryFactory remoteDirectoryFactory = new RemoteSegmentStoreDirectoryFactory( repositoriesServiceReference::get, - threadPool + threadPool, + remoteStoreSettings ); final TaskResourceTrackingService taskResourceTrackingService = new TaskResourceTrackingService( @@ -1209,7 +1211,8 @@ protected Node( repositoryService, transportService, actionModule.getActionFilters(), - remoteStorePinnedTimestampService + remoteStorePinnedTimestampService, + remoteStoreSettings ); SnapshotShardsService snapshotShardsService = new SnapshotShardsService( settings, @@ -1419,6 +1422,7 @@ protected Node( b.bind(SnapshotsInfoService.class).toInstance(snapshotsInfoService); b.bind(GatewayMetaState.class).toInstance(gatewayMetaState); b.bind(Discovery.class).toInstance(discoveryModule.getDiscovery()); + b.bind(RemoteStoreSettings.class).toInstance(remoteStoreSettings); { b.bind(PeerRecoverySourceService.class) .toInstance(new PeerRecoverySourceService(transportService, indicesService, recoverySettings)); diff --git a/server/src/main/java/org/opensearch/repositories/blobstore/BlobStoreRepository.java b/server/src/main/java/org/opensearch/repositories/blobstore/BlobStoreRepository.java index d2df42fed6db5..dd139478e29c9 100644 --- a/server/src/main/java/org/opensearch/repositories/blobstore/BlobStoreRepository.java +++ b/server/src/main/java/org/opensearch/repositories/blobstore/BlobStoreRepository.java @@ -130,6 +130,7 @@ import org.opensearch.index.store.lockmanager.FileLockInfo; import org.opensearch.index.store.lockmanager.RemoteStoreLockManager; import org.opensearch.index.store.lockmanager.RemoteStoreLockManagerFactory; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.indices.recovery.RecoverySettings; import org.opensearch.indices.recovery.RecoveryState; import org.opensearch.repositories.IndexId; @@ -420,6 +421,8 @@ public abstract class BlobStoreRepository extends AbstractLifecycleComponent imp private final RecoverySettings recoverySettings; + private final RemoteStoreSettings remoteStoreSettings; + private final NamedXContentRegistry namedXContentRegistry; /** @@ -472,6 +475,7 @@ protected BlobStoreRepository( this.threadPool = clusterService.getClusterApplierService().threadPool(); this.clusterService = clusterService; this.recoverySettings = recoverySettings; + this.remoteStoreSettings = new RemoteStoreSettings(clusterService.getSettings(), clusterService.getClusterSettings()); } @Override @@ -1296,7 +1300,8 @@ protected void releaseRemoteStoreLockAndCleanup( // related issue: https://github.com/opensearch-project/OpenSearch/issues/8469 RemoteSegmentStoreDirectoryFactory remoteDirectoryFactory = new RemoteSegmentStoreDirectoryFactory( remoteStoreLockManagerFactory.getRepositoriesService(), - threadPool + threadPool, + remoteStoreSettings ); remoteDirectoryCleanupAsync( remoteDirectoryFactory, diff --git a/server/src/main/java/org/opensearch/snapshots/SnapshotsService.java b/server/src/main/java/org/opensearch/snapshots/SnapshotsService.java index 5ecb991127fd5..2a725c3087ea9 100644 --- a/server/src/main/java/org/opensearch/snapshots/SnapshotsService.java +++ b/server/src/main/java/org/opensearch/snapshots/SnapshotsService.java @@ -92,6 +92,7 @@ import org.opensearch.core.index.Index; import org.opensearch.core.index.shard.ShardId; import org.opensearch.index.store.lockmanager.RemoteStoreLockManagerFactory; +import org.opensearch.indices.RemoteStoreSettings; import org.opensearch.node.remotestore.RemoteStorePinnedTimestampService; import org.opensearch.repositories.IndexId; import org.opensearch.repositories.RepositoriesService; @@ -214,12 +215,13 @@ public SnapshotsService( RepositoriesService repositoriesService, TransportService transportService, ActionFilters actionFilters, - @Nullable RemoteStorePinnedTimestampService remoteStorePinnedTimestampService + @Nullable RemoteStorePinnedTimestampService remoteStorePinnedTimestampService, + RemoteStoreSettings remoteStoreSettings ) { this.clusterService = clusterService; this.indexNameExpressionResolver = indexNameExpressionResolver; this.repositoriesService = repositoriesService; - this.remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(() -> repositoriesService); + this.remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(() -> repositoriesService, remoteStoreSettings); this.threadPool = transportService.getThreadPool(); this.transportService = transportService; this.remoteStorePinnedTimestampService = remoteStorePinnedTimestampService; diff --git a/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java b/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java index 5ac94281822b8..9972bbfff5d66 100644 --- a/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java +++ b/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java @@ -73,6 +73,7 @@ import org.opensearch.index.recovery.RemoteStoreRestoreService; import org.opensearch.index.recovery.RemoteStoreRestoreService.RemoteRestoreResult; import org.opensearch.index.remote.RemoteIndexPathUploader; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.node.Node; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.fs.FsRepository; @@ -504,7 +505,15 @@ public void testDataOnlyNodePersistence() throws Exception { clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, settings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + settings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ); } else { diff --git a/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java b/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java index d57cc17acedcb..9219fb548bb46 100644 --- a/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java +++ b/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java @@ -62,6 +62,7 @@ import org.opensearch.gateway.remote.model.RemoteReadResult; import org.opensearch.gateway.remote.model.RemoteTransientSettingsMetadata; import org.opensearch.index.remote.RemoteIndexPathUploader; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.indices.IndicesModule; import org.opensearch.repositories.FilterRepository; import org.opensearch.repositories.RepositoriesService; @@ -252,7 +253,15 @@ public void setup() { clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, settings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + settings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), namedWriteableRegistry ); } @@ -289,7 +298,15 @@ public void testFailInitializationWhenRemoteStateDisabled() { clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, settings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + settings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ) ); @@ -361,7 +378,15 @@ public void testWriteFullMetadataSuccessPublicationEnabled() throws IOException clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, settings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + settings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ); final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()) @@ -731,7 +756,15 @@ public void testWriteIncrementalMetadataSuccessWhenPublicationEnabled() throws I clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, settings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + settings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ); final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); @@ -2636,7 +2669,15 @@ public void testRemoteRoutingTableInitializedWhenEnabled() { clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, newSettings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + newSettings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ); assertTrue(remoteClusterStateService.getRemoteRoutingTableService() instanceof InternalRemoteRoutingTableService); @@ -2912,7 +2953,15 @@ private void initializeRoutingTable() { clusterService, () -> 0L, threadPool, - List.of(new RemoteIndexPathUploader(threadPool, newSettings, repositoriesServiceSupplier, clusterSettings)), + List.of( + new RemoteIndexPathUploader( + threadPool, + newSettings, + repositoriesServiceSupplier, + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE + ) + ), writableRegistry() ); } diff --git a/server/src/test/java/org/opensearch/index/IndexModuleTests.java b/server/src/test/java/org/opensearch/index/IndexModuleTests.java index 4ce4936c047d9..cf6fa39dd0990 100644 --- a/server/src/test/java/org/opensearch/index/IndexModuleTests.java +++ b/server/src/test/java/org/opensearch/index/IndexModuleTests.java @@ -260,7 +260,7 @@ private IndexService newIndexService(IndexModule module) throws IOException { writableRegistry(), () -> false, null, - new RemoteSegmentStoreDirectoryFactory(() -> repositoriesService, threadPool), + new RemoteSegmentStoreDirectoryFactory(() -> repositoriesService, threadPool, DefaultRemoteStoreSettings.INSTANCE), translogFactorySupplier, () -> IndexSettings.DEFAULT_REFRESH_INTERVAL, DefaultRecoverySettings.INSTANCE, diff --git a/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathTests.java b/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathTests.java index 8ddbd383756e7..3e1b7a3424c10 100644 --- a/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathTests.java +++ b/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathTests.java @@ -15,6 +15,7 @@ import org.opensearch.core.xcontent.XContentParser; import org.opensearch.index.remote.RemoteStoreEnums.PathHashAlgorithm; import org.opensearch.index.remote.RemoteStoreEnums.PathType; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.test.OpenSearchTestCase; import java.io.IOException; @@ -40,7 +41,8 @@ public void testToXContentWithSegmentRepo() throws IOException { new BlobPath().add("djsd878ndjh").add("hcs87cj8"), PathType.HASHED_PREFIX, PathHashAlgorithm.FNV_1A_BASE64, - RemoteIndexPath.SEGMENT_PATH + RemoteIndexPath.SEGMENT_PATH, + DefaultRemoteStoreSettings.INSTANCE ); XContentBuilder xContentBuilder = MediaTypeRegistry.contentBuilder(MediaTypeRegistry.JSON); xContentBuilder.startObject(); @@ -61,7 +63,8 @@ public void testToXContentForTranslogRepoOnly() throws IOException { new BlobPath().add("djsd878ndjh").add("hcs87cj8"), PathType.HASHED_PREFIX, PathHashAlgorithm.FNV_1A_BASE64, - RemoteIndexPath.TRANSLOG_PATH + RemoteIndexPath.TRANSLOG_PATH, + DefaultRemoteStoreSettings.INSTANCE ); XContentBuilder xContentBuilder = MediaTypeRegistry.contentBuilder(MediaTypeRegistry.JSON); xContentBuilder.startObject(); @@ -85,7 +88,8 @@ public void testToXContentForBothRepos() throws IOException { new BlobPath().add("nxf9yv0").add("c3ejoi"), PathType.HASHED_PREFIX, PathHashAlgorithm.FNV_1A_BASE64, - pathCreationMap + pathCreationMap, + DefaultRemoteStoreSettings.INSTANCE ); XContentBuilder xContentBuilder = MediaTypeRegistry.contentBuilder(MediaTypeRegistry.JSON); xContentBuilder.startObject(); @@ -105,7 +109,8 @@ public void testRemoteIndexPathWithInvalidPathCreationMap() throws IOException { new BlobPath().add("djsd878ndjh").add("hcs87cj8"), PathType.HASHED_PREFIX, PathHashAlgorithm.FNV_1A_BASE64, - new HashMap<>() + new HashMap<>(), + DefaultRemoteStoreSettings.INSTANCE ) ); assertEquals( @@ -132,7 +137,8 @@ public void testInvalidPathCreationMap() { new BlobPath().add("djsd878ndjh").add("hcs87cj8"), PathType.HASHED_PREFIX, PathHashAlgorithm.FNV_1A_BASE64, - Map.of(TRANSLOG, List.of(LOCK_FILES)) + Map.of(TRANSLOG, List.of(LOCK_FILES)), + DefaultRemoteStoreSettings.INSTANCE ) ); assertEquals("pathCreationMap={TRANSLOG=[LOCK_FILES]} is having illegal combination of category and type", ex.getMessage()); diff --git a/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathUploaderTests.java b/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathUploaderTests.java index e0a75f7296705..d6519d9db8ee6 100644 --- a/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathUploaderTests.java +++ b/server/src/test/java/org/opensearch/index/remote/RemoteIndexPathUploaderTests.java @@ -25,6 +25,7 @@ import org.opensearch.gateway.remote.RemoteStateTransferException; import org.opensearch.index.remote.RemoteStoreEnums.PathHashAlgorithm; import org.opensearch.index.remote.RemoteStoreEnums.PathType; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.node.Node; import org.opensearch.node.remotestore.RemoteStoreNodeAttribute; import org.opensearch.repositories.RepositoriesService; @@ -131,7 +132,8 @@ public void testInterceptWithNoRemoteDataAttributes() { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); List indexMetadataList = Mockito.mock(List.class); ActionListener actionListener = ActionListener.wrap( @@ -149,7 +151,8 @@ public void testInterceptWithEmptyIndexMetadataList() { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); ActionListener actionListener = ActionListener.wrap( @@ -166,7 +169,8 @@ public void testInterceptWithEmptyEligibleIndexMetadataList() { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); ActionListener actionListener = ActionListener.wrap( @@ -228,7 +232,8 @@ public void testInterceptWithSameRepo() throws IOException { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); ActionListener actionListener = ActionListener.wrap( @@ -251,7 +256,8 @@ public void testInterceptWithDifferentRepo() throws IOException { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); ActionListener actionListener = ActionListener.wrap( @@ -271,7 +277,8 @@ public void testInterceptWithLatchAwaitTimeout() throws IOException { threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); @@ -302,7 +309,8 @@ public void testInterceptWithInterruptedExceptionDuringLatchAwait() throws Excep threadPool, settings, () -> repositoriesService, - clusterSettings + clusterSettings, + DefaultRemoteStoreSettings.INSTANCE ); remoteIndexPathUploader.start(); Settings settings = Settings.builder() diff --git a/server/src/test/java/org/opensearch/index/shard/StoreRecoveryTests.java b/server/src/test/java/org/opensearch/index/shard/StoreRecoveryTests.java index 846b975a9520e..90d0a4fc91d6d 100644 --- a/server/src/test/java/org/opensearch/index/shard/StoreRecoveryTests.java +++ b/server/src/test/java/org/opensearch/index/shard/StoreRecoveryTests.java @@ -62,6 +62,7 @@ import org.opensearch.index.mapper.IdFieldMapper; import org.opensearch.index.mapper.Uid; import org.opensearch.index.seqno.SequenceNumbers; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.indices.replication.common.ReplicationLuceneIndex; import org.opensearch.test.OpenSearchTestCase; @@ -108,7 +109,7 @@ public void testAddIndices() throws IOException { writer.commit(); writer.close(); } - StoreRecovery storeRecovery = new StoreRecovery(new ShardId("foo", "bar", 1), logger); + StoreRecovery storeRecovery = new StoreRecovery(new ShardId("foo", "bar", 1), logger, DefaultRemoteStoreSettings.INSTANCE); ReplicationLuceneIndex indexStats = new ReplicationLuceneIndex(); Directory target = newFSDirectory(createTempDir()); final long maxSeqNo = randomNonNegativeLong(); @@ -174,7 +175,7 @@ public void testSplitShard() throws IOException { writer.commit(); writer.close(); - StoreRecovery storeRecovery = new StoreRecovery(new ShardId("foo", "bar", 1), logger); + StoreRecovery storeRecovery = new StoreRecovery(new ShardId("foo", "bar", 1), logger, DefaultRemoteStoreSettings.INSTANCE); ReplicationLuceneIndex indexStats = new ReplicationLuceneIndex(); Directory target = newFSDirectory(createTempDir()); final long maxSeqNo = randomNonNegativeLong(); diff --git a/server/src/test/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactoryTests.java b/server/src/test/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactoryTests.java index cad5e47531cc6..ea0fc3a69d5ef 100644 --- a/server/src/test/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactoryTests.java +++ b/server/src/test/java/org/opensearch/index/store/RemoteSegmentStoreDirectoryFactoryTests.java @@ -20,6 +20,7 @@ import org.opensearch.core.index.shard.ShardId; import org.opensearch.index.IndexSettings; import org.opensearch.index.shard.ShardPath; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.RepositoryMissingException; import org.opensearch.repositories.blobstore.BlobStoreRepository; @@ -57,7 +58,11 @@ public void setup() { repositoriesService = mock(RepositoriesService.class); threadPool = mock(ThreadPool.class); when(repositoriesServiceSupplier.get()).thenReturn(repositoriesService); - remoteSegmentStoreDirectoryFactory = new RemoteSegmentStoreDirectoryFactory(repositoriesServiceSupplier, threadPool); + remoteSegmentStoreDirectoryFactory = new RemoteSegmentStoreDirectoryFactory( + repositoriesServiceSupplier, + threadPool, + DefaultRemoteStoreSettings.INSTANCE + ); } public void testNewDirectory() throws IOException { diff --git a/server/src/test/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactoryTests.java b/server/src/test/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactoryTests.java index de3dfbdaa4778..93064baff30c5 100644 --- a/server/src/test/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactoryTests.java +++ b/server/src/test/java/org/opensearch/index/store/lockmanager/RemoteStoreLockManagerFactoryTests.java @@ -13,6 +13,7 @@ import org.opensearch.common.blobstore.BlobStore; import org.opensearch.index.remote.RemoteStoreEnums.PathType; import org.opensearch.index.remote.RemoteStorePathStrategy; +import org.opensearch.indices.DefaultRemoteStoreSettings; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.blobstore.BlobStoreRepository; import org.opensearch.test.OpenSearchTestCase; @@ -42,7 +43,7 @@ public void setup() throws IOException { repositoriesServiceSupplier = mock(Supplier.class); repositoriesService = mock(RepositoriesService.class); when(repositoriesServiceSupplier.get()).thenReturn(repositoriesService); - remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(repositoriesServiceSupplier); + remoteStoreLockManagerFactory = new RemoteStoreLockManagerFactory(repositoriesServiceSupplier, DefaultRemoteStoreSettings.INSTANCE); } public void testNewLockManager() throws IOException { diff --git a/server/src/test/java/org/opensearch/snapshots/SnapshotResiliencyTests.java b/server/src/test/java/org/opensearch/snapshots/SnapshotResiliencyTests.java index fd84c6b48df8a..24e402119fe2d 100644 --- a/server/src/test/java/org/opensearch/snapshots/SnapshotResiliencyTests.java +++ b/server/src/test/java/org/opensearch/snapshots/SnapshotResiliencyTests.java @@ -2013,7 +2013,8 @@ public void onFailure(final Exception e) { repositoriesService, transportService, actionFilters, - null + null, + DefaultRemoteStoreSettings.INSTANCE ); nodeEnv = new NodeEnvironment(settings, environment); final NamedXContentRegistry namedXContentRegistry = new NamedXContentRegistry(Collections.emptyList()); @@ -2070,7 +2071,7 @@ public void onFailure(final Exception e) { emptyMap(), null, emptyMap(), - new RemoteSegmentStoreDirectoryFactory(() -> repositoriesService, threadPool), + new RemoteSegmentStoreDirectoryFactory(() -> repositoriesService, threadPool, DefaultRemoteStoreSettings.INSTANCE), repositoriesServiceReference::get, null, new RemoteStoreStatsTrackerFactory(clusterService, settings), @@ -2366,7 +2367,8 @@ public void onFailure(final Exception e) { snapshotsService, threadPool, actionFilters, - indexNameExpressionResolver + indexNameExpressionResolver, + DefaultRemoteStoreSettings.INSTANCE ) ); actions.put( diff --git a/test/framework/src/main/java/org/opensearch/repositories/blobstore/BlobStoreTestUtil.java b/test/framework/src/main/java/org/opensearch/repositories/blobstore/BlobStoreTestUtil.java index e290cdee836b1..027b1bef84e7f 100644 --- a/test/framework/src/main/java/org/opensearch/repositories/blobstore/BlobStoreTestUtil.java +++ b/test/framework/src/main/java/org/opensearch/repositories/blobstore/BlobStoreTestUtil.java @@ -51,6 +51,8 @@ import org.opensearch.common.blobstore.BlobMetadata; import org.opensearch.common.blobstore.BlobPath; import org.opensearch.common.blobstore.BlobStore; +import org.opensearch.common.settings.ClusterSettings; +import org.opensearch.common.settings.Settings; import org.opensearch.common.unit.TimeValue; import org.opensearch.common.xcontent.LoggingDeprecationHandler; import org.opensearch.core.common.Strings; @@ -463,6 +465,9 @@ private static ClusterService mockClusterService(ClusterState initialState) { return null; }).when(clusterService).addStateApplier(any(ClusterStateApplier.class)); when(clusterApplierService.threadPool()).thenReturn(threadPool); + when(clusterService.getSettings()).thenReturn(Settings.EMPTY); + ClusterSettings clusterSettings = new ClusterSettings(Settings.EMPTY, ClusterSettings.BUILT_IN_CLUSTER_SETTINGS); + when(clusterService.getClusterSettings()).thenReturn(clusterSettings); return clusterService; }