diff --git a/CHANGELOG.md b/CHANGELOG.md index 52bcc878904a7..37661a2b1f22d 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -37,6 +37,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Added crypto-kms plugin to provide AWS KMS based key providers for encryption/decryption. ([#8465](https://github.com/opensearch-project/OpenSearch/pull/8465)) - [Remote state] Integrate remote cluster state in publish/commit flow ([#9665](https://github.com/opensearch-project/OpenSearch/pull/9665)) - [Remote Store] Changes to introduce repository registration during bootstrap via node attributes. ([#9105](https://github.com/opensearch-project/OpenSearch/pull/9105)) +- [Remote state] Auto restore index metadata from last known cluster state ([#9831](https://github.com/opensearch-project/OpenSearch/pull/9831)) ### Dependencies - Bump `org.apache.logging.log4j:log4j-core` from 2.17.1 to 2.20.0 ([#8307](https://github.com/opensearch-project/OpenSearch/pull/8307)) diff --git a/server/src/main/java/org/opensearch/gateway/GatewayMetaState.java b/server/src/main/java/org/opensearch/gateway/GatewayMetaState.java index 71691059ffa70..304b5c6af7f69 100644 --- a/server/src/main/java/org/opensearch/gateway/GatewayMetaState.java +++ b/server/src/main/java/org/opensearch/gateway/GatewayMetaState.java @@ -64,6 +64,8 @@ import org.opensearch.env.NodeMetadata; import org.opensearch.gateway.remote.ClusterMetadataManifest; import org.opensearch.gateway.remote.RemoteClusterStateService; +import org.opensearch.index.recovery.RemoteStoreRestoreService; +import org.opensearch.index.recovery.RemoteStoreRestoreService.RemoteRestoreResult; import org.opensearch.node.Node; import org.opensearch.plugins.MetadataUpgrader; import org.opensearch.repositories.RepositoryMissingException; @@ -127,7 +129,8 @@ public void start( MetadataUpgrader metadataUpgrader, PersistedClusterStateService persistedClusterStateService, RemoteClusterStateService remoteClusterStateService, - PersistedStateRegistry persistedStateRegistry + PersistedStateRegistry persistedStateRegistry, + RemoteStoreRestoreService remoteStoreRestoreService ) { assert this.persistedStateRegistry == null : "Persisted state registry should only be set once"; this.persistedStateRegistry = persistedStateRegistry; @@ -155,7 +158,7 @@ public void start( PersistedState remotePersistedState = null; boolean success = false; try { - final ClusterState clusterState = prepareInitialClusterState( + ClusterState clusterState = prepareInitialClusterState( transportService, clusterService, ClusterState.builder(ClusterName.CLUSTER_NAME_SETTING.get(settings)) @@ -165,10 +168,28 @@ public void start( ); if (DiscoveryNode.isClusterManagerNode(settings)) { - persistedState = new LucenePersistedState(persistedClusterStateService, currentTerm, clusterState); if (isRemoteStoreClusterStateEnabled(settings)) { + // If the cluster UUID loaded from local is unknown (_na_) then fetch the best state from remote + // If there is no valid state on remote, continue with initial empty state + // If there is a valid state, then restore index metadata using this state + if (ClusterState.UNKNOWN_UUID.equals(clusterState.metadata().clusterUUID())) { + String lastKnownClusterUUID = remoteClusterStateService.getLastKnownUUIDFromRemote( + clusterState.getClusterName().value() + ); + if (!ClusterState.UNKNOWN_UUID.equals(lastKnownClusterUUID)) { + // Load state from remote + final RemoteRestoreResult remoteRestoreResult = remoteStoreRestoreService.restore( + clusterState, + lastKnownClusterUUID, + false, + new String[] {} + ); + clusterState = remoteRestoreResult.getClusterState(); + } + } remotePersistedState = new RemotePersistedState(remoteClusterStateService); } + persistedState = new LucenePersistedState(persistedClusterStateService, currentTerm, clusterState); } else { persistedState = new AsyncLucenePersistedState( settings, @@ -652,12 +673,6 @@ public void setCurrentTerm(long currentTerm) { @Override public void setLastAcceptedState(ClusterState clusterState) { try { - if (lastAcceptedState == null || lastAcceptedState.blocks().hasGlobalBlock(GatewayService.STATE_NOT_RECOVERED_BLOCK)) { - // On the initial bootstrap, repository will not be available. So we do not persist the cluster state and bail out. - logger.info("Cluster is not yet ready to publish state to remote store"); - lastAcceptedState = clusterState; - return; - } final ClusterMetadataManifest manifest; if (shouldWriteFullClusterState(clusterState)) { manifest = remoteClusterStateService.writeFullMetadata(clusterState); @@ -707,13 +722,8 @@ private boolean shouldWriteFullClusterState(ClusterState clusterState) { @Override public void markLastAcceptedStateAsCommitted() { try { - if (lastAcceptedState == null - || lastAcceptedManifest == null - || lastAcceptedState.blocks().hasGlobalBlock(GatewayService.STATE_NOT_RECOVERED_BLOCK)) { - // On the initial bootstrap, repository will not be available. So we do not persist the cluster state and bail out. - logger.trace("Cluster is not yet ready to publish state to remote store"); - return; - } + assert lastAcceptedState != null : "Last accepted state is not present"; + assert lastAcceptedManifest != null : "Last accepted manifest is not present"; final ClusterMetadataManifest committedManifest = remoteClusterStateService.markLastStateAsCommitted( lastAcceptedState, lastAcceptedManifest diff --git a/server/src/main/java/org/opensearch/gateway/remote/RemoteClusterStateService.java b/server/src/main/java/org/opensearch/gateway/remote/RemoteClusterStateService.java index 990fc20a7e95d..2aa3384b0f33a 100644 --- a/server/src/main/java/org/opensearch/gateway/remote/RemoteClusterStateService.java +++ b/server/src/main/java/org/opensearch/gateway/remote/RemoteClusterStateService.java @@ -46,6 +46,7 @@ import java.util.Locale; import java.util.Map; import java.util.Objects; +import java.util.Optional; import java.util.Set; import java.util.concurrent.CountDownLatch; import java.util.concurrent.TimeUnit; @@ -136,14 +137,20 @@ public ClusterMetadataManifest writeFullMetadata(ClusterState clusterState) thro logger.error("Local node is not elected cluster manager. Exiting"); return null; } - ensureRepositorySet(); + + // should fetch the previous cluster UUID before writing full cluster state. + // Whenever a new election happens, a new leader will be elected and it might have stale previous UUID + final String previousClusterUUID = fetchPreviousClusterUUID( + clusterState.getClusterName().value(), + clusterState.metadata().clusterUUID() + ); // any validations before/after upload ? final List allUploadedIndexMetadata = writeIndexMetadataParallel( clusterState, new ArrayList<>(clusterState.metadata().indices().values()) ); - final ClusterMetadataManifest manifest = uploadManifest(clusterState, allUploadedIndexMetadata, false); + final ClusterMetadataManifest manifest = uploadManifest(clusterState, allUploadedIndexMetadata, previousClusterUUID, false); final long durationMillis = TimeValue.nsecToMSec(relativeTimeNanosSupplier.getAsLong() - startTimeNanos); if (durationMillis >= slowWriteLoggingThreshold.getMillis()) { logger.warn( @@ -220,7 +227,12 @@ public ClusterMetadataManifest writeIncrementalMetadata( for (String removedIndexName : previousStateIndexMetadataVersionByName.keySet()) { allUploadedIndexMetadata.remove(removedIndexName); } - final ClusterMetadataManifest manifest = uploadManifest(clusterState, new ArrayList<>(allUploadedIndexMetadata.values()), false); + final ClusterMetadataManifest manifest = uploadManifest( + clusterState, + new ArrayList<>(allUploadedIndexMetadata.values()), + previousManifest.getPreviousClusterUUID(), + false + ); final long durationMillis = TimeValue.nsecToMSec(relativeTimeNanosSupplier.getAsLong() - startTimeNanos); if (durationMillis >= slowWriteLoggingThreshold.getMillis()) { logger.warn( @@ -333,13 +345,13 @@ private void writeIndexMetadataAsync( clusterState.metadata().clusterUUID(), indexMetadata.getIndexUUID() ); - + final String indexMetadataFilename = indexMetadataFileName(indexMetadata); ActionListener completionListener = ActionListener.wrap( resp -> latchedActionListener.onResponse( new UploadedIndexMetadata( indexMetadata.getIndex().getName(), indexMetadata.getIndexUUID(), - indexMetadataContainer.path().buildAsString() + indexMetadataFileName(indexMetadata) + indexMetadataContainer.path().buildAsString() + indexMetadataFilename ) ), ex -> latchedActionListener.onFailure(new IndexMetadataTransferException(indexMetadata.getIndex().toString(), ex)) @@ -348,7 +360,7 @@ private void writeIndexMetadataAsync( INDEX_METADATA_FORMAT.writeAsync( indexMetadata, indexMetadataContainer, - indexMetadataFileName(indexMetadata), + indexMetadataFilename, blobStoreRepository.getCompressor(), completionListener ); @@ -363,7 +375,7 @@ public ClusterMetadataManifest markLastStateAsCommitted(ClusterState clusterStat } assert clusterState != null : "Last accepted cluster state is not set"; assert previousManifest != null : "Last cluster metadata manifest is not set"; - return uploadManifest(clusterState, previousManifest.getIndices(), true); + return uploadManifest(clusterState, previousManifest.getIndices(), previousManifest.getPreviousClusterUUID(), true); } @Override @@ -373,12 +385,8 @@ public void close() throws IOException { } } - // Visible for testing - void ensureRepositorySet() { - if (blobStoreRepository != null) { - return; - } - assert isRemoteStoreClusterStateEnabled(settings) : "Remote cluster state is not enabled"; + public void start() { + assert isRemoteStoreClusterStateEnabled(settings) == true : "Remote cluster state is not enabled"; final String remoteStoreRepo = settings.get( Node.NODE_ATTRIBUTES.getKey() + RemoteStoreNodeAttribute.REMOTE_STORE_CLUSTER_STATE_REPOSITORY_NAME_ATTRIBUTE_KEY ); @@ -391,6 +399,7 @@ void ensureRepositorySet() { private ClusterMetadataManifest uploadManifest( ClusterState clusterState, List uploadedIndexMetadata, + String previousClusterUUID, boolean committed ) throws IOException { synchronized (this) { @@ -404,8 +413,7 @@ private ClusterMetadataManifest uploadManifest( nodeId, committed, uploadedIndexMetadata, - // todo Change this to proper cluster UUID - ClusterState.UNKNOWN_UUID + previousClusterUUID ); writeMetadataManifest(clusterState.getClusterName().value(), clusterState.metadata().clusterUUID(), manifest, manifestFileName); return manifest; @@ -418,6 +426,16 @@ private void writeMetadataManifest(String clusterName, String clusterUUID, Clust CLUSTER_METADATA_MANIFEST_FORMAT.write(uploadManifest, metadataManifestContainer, fileName, blobStoreRepository.getCompressor()); } + private String fetchPreviousClusterUUID(String clusterName, String clusterUUID) { + final Optional latestManifest = getLatestClusterMetadataManifest(clusterName, clusterUUID); + if (!latestManifest.isPresent()) { + final String previousClusterUUID = getLastKnownUUIDFromRemote(clusterName); + assert !clusterUUID.equals(previousClusterUUID) : "Last cluster UUID is same current cluster UUID"; + return previousClusterUUID; + } + return latestManifest.get().getPreviousClusterUUID(); + } + private BlobContainer indexMetadataContainer(String clusterName, String clusterUUID, String indexUUID) { // 123456789012_test-cluster/cluster-state/dsgYj10Nkso7/index/ftqsCnn9TgOX return blobStoreRepository.blobStore() @@ -484,12 +502,15 @@ private static String indexMetadataFileName(IndexMetadata indexMetadata) { * @return {@code Map} latest IndexUUID to IndexMetadata map */ public Map getLatestIndexMetadata(String clusterName, String clusterUUID) throws IOException { - ensureRepositorySet(); + start(); Map remoteIndexMetadata = new HashMap<>(); - ClusterMetadataManifest clusterMetadataManifest = getLatestClusterMetadataManifest(clusterName, clusterUUID); - assert Objects.equals(clusterUUID, clusterMetadataManifest.getClusterUUID()) + Optional clusterMetadataManifest = getLatestClusterMetadataManifest(clusterName, clusterUUID); + if (!clusterMetadataManifest.isPresent()) { + throw new IllegalStateException("Latest index metadata is not present for the provided clusterUUID"); + } + assert Objects.equals(clusterUUID, clusterMetadataManifest.get().getClusterUUID()) : "Corrupt ClusterMetadataManifest found. Cluster UUID mismatch."; - for (UploadedIndexMetadata uploadedIndexMetadata : clusterMetadataManifest.getIndices()) { + for (UploadedIndexMetadata uploadedIndexMetadata : clusterMetadataManifest.get().getIndices()) { IndexMetadata indexMetadata = getIndexMetadata(clusterName, clusterUUID, uploadedIndexMetadata); remoteIndexMetadata.put(uploadedIndexMetadata.getIndexUUID(), indexMetadata); } @@ -527,9 +548,12 @@ private IndexMetadata getIndexMetadata(String clusterName, String clusterUUID, U * @param clusterName name of the cluster * @return ClusterMetadataManifest */ - public ClusterMetadataManifest getLatestClusterMetadataManifest(String clusterName, String clusterUUID) { - String latestManifestFileName = getLatestManifestFileName(clusterName, clusterUUID); - return fetchRemoteClusterMetadataManifest(clusterName, clusterUUID, latestManifestFileName); + public Optional getLatestClusterMetadataManifest(String clusterName, String clusterUUID) { + Optional latestManifestFileName = getLatestManifestFileName(clusterName, clusterUUID); + if (latestManifestFileName.isPresent()) { + return Optional.of(fetchRemoteClusterMetadataManifest(clusterName, clusterUUID, latestManifestFileName.get())); + } + return Optional.empty(); } /** @@ -538,7 +562,7 @@ public ClusterMetadataManifest getLatestClusterMetadataManifest(String clusterNa * @param clusterName The cluster name for which previous cluster UUID is to be fetched * @return Last valid cluster UUID */ - public String getLatestClusterUUID(String clusterName) { + public String getLastKnownUUIDFromRemote(String clusterName) { try { Set clusterUUIDs = getAllClusterUUIDs(clusterName); Map latestManifests = getLatestManifestForAllClusterUUIDs(clusterName, clusterUUIDs); @@ -566,8 +590,8 @@ private Map getLatestManifestForAllClusterUUIDs Map manifestsByClusterUUID = new HashMap<>(); for (String clusterUUID : clusterUUIDs) { try { - ClusterMetadataManifest manifest = getLatestClusterMetadataManifest(clusterName, clusterUUID); - manifestsByClusterUUID.put(clusterUUID, manifest); + Optional manifest = getLatestClusterMetadataManifest(clusterName, clusterUUID); + manifestsByClusterUUID.put(clusterUUID, manifest.get()); } catch (Exception e) { throw new IllegalStateException( String.format(Locale.ROOT, "Exception in fetching manifest for clusterUUID: %s", clusterUUID) @@ -627,7 +651,7 @@ private boolean isInvalidClusterUUID(ClusterMetadataManifest manifest) { * @param clusterName name of the cluster * @return latest ClusterMetadataManifest filename */ - private String getLatestManifestFileName(String clusterName, String clusterUUID) throws IllegalStateException { + private Optional getLatestManifestFileName(String clusterName, String clusterUUID) throws IllegalStateException { try { /** * {@link BlobContainer#listBlobsByPrefixInSortedOrder} will get the latest manifest file @@ -640,13 +664,13 @@ private String getLatestManifestFileName(String clusterName, String clusterUUID) BlobContainer.BlobNameSortOrder.LEXICOGRAPHIC ); if (manifestFilesMetadata != null && !manifestFilesMetadata.isEmpty()) { - return manifestFilesMetadata.get(0).name(); + return Optional.of(manifestFilesMetadata.get(0).name()); } } catch (IOException e) { throw new IllegalStateException("Error while fetching latest manifest file for remote cluster state", e); } - - throw new IllegalStateException(String.format(Locale.ROOT, "Remote Cluster State not found - %s", clusterUUID)); + logger.info("No manifest file present in remote store for cluster name: {}, cluster UUID: {}", clusterName, clusterUUID); + return Optional.empty(); } /** diff --git a/server/src/main/java/org/opensearch/node/Node.java b/server/src/main/java/org/opensearch/node/Node.java index 63598d8e9bd13..67546a4d7db50 100644 --- a/server/src/main/java/org/opensearch/node/Node.java +++ b/server/src/main/java/org/opensearch/node/Node.java @@ -1336,6 +1336,10 @@ public Node start() throws NodeValidationException { injector.getInstance(PeerRecoverySourceService.class).start(); injector.getInstance(SegmentReplicationSourceService.class).start(); + final RemoteClusterStateService remoteClusterStateService = injector.getInstance(RemoteClusterStateService.class); + if (remoteClusterStateService != null) { + remoteClusterStateService.start(); + } // Load (and maybe upgrade) the metadata stored on disk final GatewayMetaState gatewayMetaState = injector.getInstance(GatewayMetaState.class); gatewayMetaState.start( @@ -1347,7 +1351,8 @@ public Node start() throws NodeValidationException { injector.getInstance(MetadataUpgrader.class), injector.getInstance(PersistedClusterStateService.class), injector.getInstance(RemoteClusterStateService.class), - injector.getInstance(PersistedStateRegistry.class) + injector.getInstance(PersistedStateRegistry.class), + injector.getInstance(RemoteStoreRestoreService.class) ); if (Assertions.ENABLED) { try { diff --git a/server/src/test/java/org/opensearch/cluster/coordination/CoordinationStateTests.java b/server/src/test/java/org/opensearch/cluster/coordination/CoordinationStateTests.java index aa4472c4fcec5..d1c2dda615992 100644 --- a/server/src/test/java/org/opensearch/cluster/coordination/CoordinationStateTests.java +++ b/server/src/test/java/org/opensearch/cluster/coordination/CoordinationStateTests.java @@ -925,20 +925,18 @@ public void testHandlePrePublishAndCommitWhenRemoteStateEnabled() throws IOExcep final RemoteClusterStateService remoteClusterStateService = Mockito.mock(RemoteClusterStateService.class); final VotingConfiguration initialConfig = VotingConfiguration.of(node1); final ClusterState clusterState = clusterState(0L, 0L, node1, initialConfig, initialConfig, 42L); - Mockito.when(remoteClusterStateService.writeFullMetadata(clusterState)) - .thenReturn( - new ClusterMetadataManifest( - 0L, - 0L, - randomAlphaOfLength(10), - randomAlphaOfLength(10), - Version.CURRENT, - randomAlphaOfLength(10), - false, - Collections.emptyList(), - randomAlphaOfLength(10) - ) - ); + final ClusterMetadataManifest manifest = new ClusterMetadataManifest( + 0L, + 0L, + randomAlphaOfLength(10), + randomAlphaOfLength(10), + Version.CURRENT, + randomAlphaOfLength(10), + false, + Collections.emptyList(), + randomAlphaOfLength(10) + ); + Mockito.when(remoteClusterStateService.writeFullMetadata(clusterState)).thenReturn(manifest); final PersistedStateRegistry persistedStateRegistry = persistedStateRegistry(); persistedStateRegistry.addPersistedState(PersistedStateType.LOCAL, ps1); @@ -965,27 +963,11 @@ public void testHandlePrePublishAndCommitWhenRemoteStateEnabled() throws IOExcep final CoordinationState coordinationState = createCoordinationState(persistedStateRegistry, node1, settings); coordinationState.handlePrePublish(clusterState); - Mockito.verifyNoInteractions(remoteClusterStateService); + Mockito.verify(remoteClusterStateService, Mockito.times(1)).writeFullMetadata(clusterState); assertThat(persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE).getLastAcceptedState(), equalTo(clusterState)); - final ClusterState clusterState2 = clusterState(0L, 1L, node1, initialConfig, initialConfig, 42L); - final ClusterMetadataManifest manifest2 = new ClusterMetadataManifest( - 0L, - 1L, - randomAlphaOfLength(10), - randomAlphaOfLength(10), - Version.CURRENT, - randomAlphaOfLength(10), - false, - Collections.emptyList(), - randomAlphaOfLength(10) - ); - Mockito.when(remoteClusterStateService.writeFullMetadata(clusterState2)).thenReturn(manifest2); - coordinationState.handlePrePublish(clusterState2); - Mockito.verify(remoteClusterStateService, Mockito.times(1)).writeFullMetadata(clusterState2); - coordinationState.handlePreCommit(); - Mockito.verify(remoteClusterStateService, Mockito.times(1)).markLastStateAsCommitted(clusterState2, manifest2); + Mockito.verify(remoteClusterStateService, Mockito.times(1)).markLastStateAsCommitted(clusterState, manifest); } public static CoordinationState createCoordinationState( diff --git a/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java b/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java index 98a9c0ded2d9d..6a2f4cd0ab300 100644 --- a/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java +++ b/server/src/test/java/org/opensearch/gateway/GatewayMetaStatePersistedStateTests.java @@ -42,6 +42,7 @@ import org.opensearch.cluster.coordination.CoordinationMetadata; import org.opensearch.cluster.coordination.CoordinationMetadata.VotingConfigExclusion; import org.opensearch.cluster.coordination.CoordinationState; +import org.opensearch.cluster.coordination.CoordinationState.PersistedState; import org.opensearch.cluster.coordination.PersistedStateRegistry; import org.opensearch.cluster.coordination.PersistedStateRegistry.PersistedStateType; import org.opensearch.cluster.metadata.IndexMetadata; @@ -51,6 +52,7 @@ import org.opensearch.cluster.node.DiscoveryNodeRole; import org.opensearch.cluster.node.DiscoveryNodes; import org.opensearch.cluster.service.ClusterService; +import org.opensearch.common.collect.Tuple; import org.opensearch.common.settings.ClusterSettings; import org.opensearch.common.settings.Settings; import org.opensearch.common.util.BigArrays; @@ -63,8 +65,11 @@ import org.opensearch.env.NodeEnvironment; import org.opensearch.env.TestEnvironment; import org.opensearch.gateway.GatewayMetaState.RemotePersistedState; +import org.opensearch.gateway.PersistedClusterStateService.Writer; import org.opensearch.gateway.remote.ClusterMetadataManifest; import org.opensearch.gateway.remote.RemoteClusterStateService; +import org.opensearch.index.recovery.RemoteStoreRestoreService; +import org.opensearch.index.recovery.RemoteStoreRestoreService.RemoteRestoreResult; import org.opensearch.node.Node; import org.opensearch.repositories.RepositoriesService; import org.opensearch.repositories.fs.FsRepository; @@ -87,6 +92,7 @@ import org.mockito.Mockito; +import static org.opensearch.cluster.metadata.IndexMetadata.SETTING_INDEX_UUID; import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.REMOTE_STORE_CLUSTER_STATE_REPOSITORY_NAME_ATTRIBUTE_KEY; import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.REMOTE_STORE_REPOSITORY_SETTINGS_ATTRIBUTE_KEY_PREFIX; import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.REMOTE_STORE_REPOSITORY_TYPE_ATTRIBUTE_KEY_FORMAT; @@ -96,8 +102,12 @@ import static org.hamcrest.Matchers.instanceOf; import static org.hamcrest.Matchers.not; import static org.hamcrest.Matchers.nullValue; +import static org.mockito.ArgumentMatchers.any; +import static org.mockito.ArgumentMatchers.anyBoolean; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.times; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.verifyNoInteractions; import static org.mockito.Mockito.when; public class GatewayMetaStatePersistedStateTests extends OpenSearchTestCase { @@ -479,7 +489,8 @@ public void testDataOnlyNodePersistence() throws Exception { null, persistedClusterStateService, remoteClusterStateServiceSupplier.get(), - new PersistedStateRegistry() + new PersistedStateRegistry(), + null ); final CoordinationState.PersistedState persistedState = gateway.getPersistedState(); assertThat(persistedState, instanceOf(GatewayMetaState.AsyncLucenePersistedState.class)); @@ -712,7 +723,7 @@ public void testRemotePersistedState() throws IOException { ); remotePersistedState.setLastAcceptedState(clusterState); - Mockito.verify(remoteClusterStateService, times(0)).writeFullMetadata(Mockito.any()); + Mockito.verify(remoteClusterStateService).writeFullMetadata(clusterState); assertThat(remotePersistedState.getLastAcceptedState(), equalTo(clusterState)); assertThat(remotePersistedState.getCurrentTerm(), equalTo(clusterTerm)); @@ -723,7 +734,7 @@ public void testRemotePersistedState() throws IOException { ); remotePersistedState.setLastAcceptedState(secondClusterState); - Mockito.verify(remoteClusterStateService, times(1)).writeFullMetadata(Mockito.any()); + Mockito.verify(remoteClusterStateService, times(1)).writeFullMetadata(secondClusterState); assertThat(remotePersistedState.getLastAcceptedState(), equalTo(secondClusterState)); assertThat(remotePersistedState.getCurrentTerm(), equalTo(clusterTerm)); @@ -748,20 +759,19 @@ public void testRemotePersistedStateExceptionOnFullStateUpload() throws IOExcept Metadata.builder().coordinationMetadata(CoordinationMetadata.builder().term(clusterTerm).build()).build() ); - remotePersistedState.setLastAcceptedState(clusterState); - - final ClusterState secondClusterState = createClusterState( - randomNonNegativeLong(), - Metadata.builder().coordinationMetadata(CoordinationMetadata.builder().term(clusterTerm).build()).build() - ); - - assertThrows(OpenSearchException.class, () -> remotePersistedState.setLastAcceptedState(secondClusterState)); + assertThrows(OpenSearchException.class, () -> remotePersistedState.setLastAcceptedState(clusterState)); } public void testGatewayForRemoteState() throws IOException { MockGatewayMetaState gateway = null; try { - gateway = new MockGatewayMetaState(localNode, bigArrays); + RemoteClusterStateService remoteClusterStateService = mock(RemoteClusterStateService.class); + when(remoteClusterStateService.getLastKnownUUIDFromRemote("test-cluster")).thenReturn("test-cluster-uuid"); + RemoteStoreRestoreService remoteStoreRestoreService = mock(RemoteStoreRestoreService.class); + when(remoteStoreRestoreService.restore(any(), any(), anyBoolean(), any())).thenReturn( + RemoteRestoreResult.build("test-cluster-uuid", null, ClusterState.EMPTY_STATE) + ); + gateway = new MockGatewayMetaState(localNode, bigArrays, remoteClusterStateService, remoteStoreRestoreService); final PersistedStateRegistry persistedStateRegistry = persistedStateRegistry(); String stateRepoTypeAttributeKey = String.format( @@ -798,6 +808,173 @@ public void testGatewayForRemoteState() throws IOException { } } + public void testGatewayForRemoteStateForInitialBootstrap() throws IOException { + MockGatewayMetaState gateway = null; + try { + final RemoteClusterStateService remoteClusterStateService = mock(RemoteClusterStateService.class); + when(remoteClusterStateService.getLastKnownUUIDFromRemote(clusterName.value())).thenReturn(ClusterState.UNKNOWN_UUID); + + final RemoteStoreRestoreService remoteStoreRestoreService = mock(RemoteStoreRestoreService.class); + when(remoteStoreRestoreService.restore(any(), any(), anyBoolean(), any())).thenReturn( + RemoteRestoreResult.build("test-cluster-uuid", null, ClusterState.EMPTY_STATE) + ); + final PersistedStateRegistry persistedStateRegistry = persistedStateRegistry(); + gateway = newGatewayForRemoteState( + remoteClusterStateService, + remoteStoreRestoreService, + persistedStateRegistry, + ClusterState.EMPTY_STATE + ); + final CoordinationState.PersistedState lucenePersistedState = gateway.getPersistedState(); + PersistedState remotePersistedState = persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE); + verify(remoteClusterStateService).getLastKnownUUIDFromRemote(Mockito.any()); // change this + verifyNoInteractions(remoteStoreRestoreService); + assertThat(remotePersistedState.getLastAcceptedState(), nullValue()); + assertThat(lucenePersistedState.getLastAcceptedState().metadata(), equalTo(ClusterState.EMPTY_STATE.metadata())); + } finally { + IOUtils.close(gateway); + } + } + + public void testGatewayForRemoteStateForNodeReplacement() throws IOException { + MockGatewayMetaState gateway = null; + try { + final RemoteClusterStateService remoteClusterStateService = mock(RemoteClusterStateService.class); + when(remoteClusterStateService.getLastKnownUUIDFromRemote("test-cluster")).thenReturn("test-cluster-uuid"); + final ClusterState previousState = createClusterState( + randomNonNegativeLong(), + Metadata.builder() + .coordinationMetadata(CoordinationMetadata.builder().term(randomLong()).build()) + .put( + IndexMetadata.builder("test-index1") + .settings(settings(Version.CURRENT).put(SETTING_INDEX_UUID, randomAlphaOfLength(10))) + .numberOfShards(5) + .numberOfReplicas(1) + .build(), + false + ) + .clusterUUID(randomAlphaOfLength(10)) + .build() + ); + when(remoteClusterStateService.getLastKnownUUIDFromRemote(clusterName.value())).thenReturn( + previousState.metadata().clusterUUID() + ); + + final RemoteStoreRestoreService remoteStoreRestoreService = mock(RemoteStoreRestoreService.class); + when(remoteStoreRestoreService.restore(any(), any(), anyBoolean(), any())).thenReturn( + RemoteRestoreResult.build("test-cluster-uuid", null, previousState) + ); + final PersistedStateRegistry persistedStateRegistry = persistedStateRegistry(); + gateway = newGatewayForRemoteState( + remoteClusterStateService, + remoteStoreRestoreService, + persistedStateRegistry, + ClusterState.EMPTY_STATE + ); + final CoordinationState.PersistedState lucenePersistedState = gateway.getPersistedState(); + PersistedState remotePersistedState = persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE); + verify(remoteClusterStateService).getLastKnownUUIDFromRemote(Mockito.any()); + verify(remoteStoreRestoreService).restore(any(), any(), anyBoolean(), any()); + assertThat(remotePersistedState.getLastAcceptedState(), nullValue()); + assertThat(lucenePersistedState.getLastAcceptedState().metadata(), equalTo(previousState.metadata())); + } finally { + IOUtils.close(gateway); + } + } + + public void testGatewayForRemoteStateForNodeReboot() throws IOException { + MockGatewayMetaState gateway = null; + try { + final RemoteClusterStateService remoteClusterStateService = mock(RemoteClusterStateService.class); + final RemoteStoreRestoreService remoteStoreRestoreService = mock(RemoteStoreRestoreService.class); + final PersistedStateRegistry persistedStateRegistry = persistedStateRegistry(); + final IndexMetadata indexMetadata = IndexMetadata.builder("test-index1") + .settings(settings(Version.CURRENT).put(SETTING_INDEX_UUID, randomAlphaOfLength(10))) + .numberOfShards(5) + .numberOfReplicas(1) + .build(); + final ClusterState clusterState = createClusterState( + randomNonNegativeLong(), + Metadata.builder() + .coordinationMetadata(CoordinationMetadata.builder().term(randomLong()).build()) + .put(indexMetadata, false) + .clusterUUID(randomAlphaOfLength(10)) + .build() + ); + gateway = newGatewayForRemoteState(remoteClusterStateService, remoteStoreRestoreService, persistedStateRegistry, clusterState); + final CoordinationState.PersistedState lucenePersistedState = gateway.getPersistedState(); + PersistedState remotePersistedState = persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE); + verifyNoInteractions(remoteClusterStateService); + verifyNoInteractions(remoteStoreRestoreService); + assertThat(remotePersistedState.getLastAcceptedState(), nullValue()); + logger.info("lucene state metadata: {}", lucenePersistedState.getLastAcceptedState().toString()); + logger.info("initial metadata: {}", clusterState.toString()); + assertThat(lucenePersistedState.getLastAcceptedState().metadata().indices().size(), equalTo(1)); + assertThat(lucenePersistedState.getLastAcceptedState().metadata().indices().get("test-index1"), equalTo(indexMetadata)); + } finally { + IOUtils.close(gateway); + } + } + + private MockGatewayMetaState newGatewayForRemoteState( + RemoteClusterStateService remoteClusterStateService, + RemoteStoreRestoreService remoteStoreRestoreService, + PersistedStateRegistry persistedStateRegistry, + ClusterState currentState + ) throws IOException { + MockGatewayMetaState gateway = new MockGatewayMetaState(localNode, bigArrays); + String randomRepoName = "randomRepoName"; + String stateRepoTypeAttributeKey = String.format( + Locale.getDefault(), + "node.attr." + REMOTE_STORE_REPOSITORY_TYPE_ATTRIBUTE_KEY_FORMAT, + randomRepoName + ); + String stateRepoSettingsAttributeKeyPrefix = String.format( + Locale.getDefault(), + "node.attr." + REMOTE_STORE_REPOSITORY_SETTINGS_ATTRIBUTE_KEY_PREFIX, + randomRepoName + ); + Settings settingWithRemoteStateEnabled = Settings.builder() + .put(settings) + .put("node.attr." + REMOTE_STORE_CLUSTER_STATE_REPOSITORY_NAME_ATTRIBUTE_KEY, randomRepoName) + .put(stateRepoTypeAttributeKey, FsRepository.TYPE) + .put(stateRepoSettingsAttributeKeyPrefix + "location", "randomRepoPath") + .put(RemoteClusterStateService.REMOTE_CLUSTER_STATE_ENABLED_SETTING.getKey(), true) + .build(); + final TransportService transportService = mock(TransportService.class); + ClusterService clusterService = mock(ClusterService.class); + when(clusterService.getClusterSettings()).thenReturn( + new ClusterSettings(Settings.EMPTY, ClusterSettings.BUILT_IN_CLUSTER_SETTINGS) + ); + final PersistedClusterStateService persistedClusterStateService = new PersistedClusterStateService( + nodeEnvironment, + xContentRegistry(), + getBigArrays(), + new ClusterSettings(settings, ClusterSettings.BUILT_IN_CLUSTER_SETTINGS), + () -> 0L + ); + if (!ClusterState.EMPTY_STATE.equals(currentState)) { + Writer writer = persistedClusterStateService.createWriter(); + writer.writeFullStateAndCommit(currentState.term(), currentState); + writer.close(); + } + final MetaStateService metaStateService = mock(MetaStateService.class); + when(metaStateService.loadFullState()).thenReturn(new Tuple<>(Manifest.empty(), ClusterState.EMPTY_STATE.metadata())); + gateway.start( + settingWithRemoteStateEnabled, + transportService, + clusterService, + metaStateService, + null, + null, + persistedClusterStateService, + remoteClusterStateService, + persistedStateRegistry, + remoteStoreRestoreService + ); + return gateway; + } + private static BigArrays getBigArrays() { return usually() ? BigArrays.NON_RECYCLING_INSTANCE 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 156138340db10..8c6ccea940816 100644 --- a/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java +++ b/server/src/test/java/org/opensearch/gateway/remote/RemoteClusterStateServiceTests.java @@ -54,6 +54,7 @@ import java.util.List; import java.util.Locale; import java.util.Map; +import java.util.Optional; import java.util.Set; import java.util.function.Supplier; @@ -140,22 +141,21 @@ public void testFailInitializationWhenRemoteStateDisabled() throws IOException { ); } - public void testFailWriteFullMetadataWhenRepositoryNotSet() { - final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); + public void testFailInitializeWhenRepositoryNotSet() { doThrow(new RepositoryMissingException("repository missing")).when(repositoriesService).repository("remote_store_repository"); - assertThrows(RepositoryMissingException.class, () -> remoteClusterStateService.writeFullMetadata(clusterState)); + assertThrows(RepositoryMissingException.class, () -> remoteClusterStateService.start()); } public void testFailWriteFullMetadataWhenNotBlobRepository() { final FilterRepository filterRepository = mock(FilterRepository.class); when(repositoriesService.repository("remote_store_repository")).thenReturn(filterRepository); - final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); - assertThrows(AssertionError.class, () -> remoteClusterStateService.writeFullMetadata(clusterState)); + assertThrows(AssertionError.class, () -> remoteClusterStateService.start()); } public void testWriteFullMetadataSuccess() throws IOException { final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); mockBlobStoreObjects(); + remoteClusterStateService.start(); final ClusterMetadataManifest manifest = remoteClusterStateService.writeFullMetadata(clusterState); final UploadedIndexMetadata uploadedIndexMetadata = new UploadedIndexMetadata("test-index", "index-uuid", "metadata-filename"); List indices = List.of(uploadedIndexMetadata); @@ -191,6 +191,7 @@ public void testWriteFullMetadataInParallelSuccess() throws IOException { return null; }).when(container).asyncBlobUpload(writeContextArgumentCaptor.capture(), actionListenerArgumentCaptor.capture()); + remoteClusterStateService.start(); final ClusterMetadataManifest manifest = remoteClusterStateService.writeFullMetadata(clusterState); final UploadedIndexMetadata uploadedIndexMetadata = new UploadedIndexMetadata("test-index", "index-uuid", "metadata-filename"); @@ -245,6 +246,7 @@ public void testWriteFullMetadataInParallelFailure() throws IOException { return null; }).when(container).asyncBlobUpload(any(WriteContext.class), actionListenerArgumentCaptor.capture()); + remoteClusterStateService.start(); assertThrows( RemoteClusterStateService.IndexMetadataTransferException.class, () -> remoteClusterStateService.writeFullMetadata(clusterState) @@ -253,6 +255,7 @@ public void testWriteFullMetadataInParallelFailure() throws IOException { public void testFailWriteIncrementalMetadataNonClusterManagerNode() throws IOException { final ClusterState clusterState = generateClusterStateWithOneIndex().build(); + remoteClusterStateService.start(); final ClusterMetadataManifest manifest = remoteClusterStateService.writeIncrementalMetadata(clusterState, clusterState, null); Assert.assertThat(manifest, nullValue()); } @@ -279,7 +282,7 @@ public void testWriteIncrementalMetadataSuccess() throws IOException { final ClusterMetadataManifest previousManifest = ClusterMetadataManifest.builder().indices(Collections.emptyList()).build(); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); final ClusterMetadataManifest manifest = remoteClusterStateService.writeIncrementalMetadata( previousClusterState, clusterState, @@ -319,7 +322,7 @@ public void testReadLatestMetadataManifestFailedIOException() throws IOException ) ).thenThrow(IOException.class); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); Exception e = assertThrows( IllegalStateException.class, () -> remoteClusterStateService.getLatestClusterMetadataManifest( @@ -342,15 +345,12 @@ public void testReadLatestMetadataManifestFailedNoManifestFileInRemote() throws ) ).thenReturn(List.of()); - remoteClusterStateService.ensureRepositorySet(); - Exception e = assertThrows( - IllegalStateException.class, - () -> remoteClusterStateService.getLatestClusterMetadataManifest( - clusterState.getClusterName().value(), - clusterState.metadata().clusterUUID() - ) + remoteClusterStateService.start(); + Optional manifest = remoteClusterStateService.getLatestClusterMetadataManifest( + clusterState.getClusterName().value(), + clusterState.metadata().clusterUUID() ); - assertEquals(e.getMessage(), "Remote Cluster State not found - " + clusterState.metadata().clusterUUID()); + assertEquals(manifest, Optional.empty()); } public void testReadLatestMetadataManifestFailedManifestFileRemoveAfterFetchInRemote() throws IOException { @@ -367,7 +367,7 @@ public void testReadLatestMetadataManifestFailedManifestFileRemoveAfterFetchInRe ).thenReturn(Arrays.asList(blobMetadata)); when(blobContainer.readBlob("manifestFileName")).thenThrow(FileNotFoundException.class); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); Exception e = assertThrows( IllegalStateException.class, () -> remoteClusterStateService.getLatestClusterMetadataManifest( @@ -394,7 +394,7 @@ public void testReadLatestMetadataManifestSuccessButNoIndexMetadata() throws IOE BlobContainer blobContainer = mockBlobStoreObjects(); mockBlobContainer(blobContainer, expectedManifest, Map.of()); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); assertEquals( remoteClusterStateService.getLatestIndexMetadata(clusterState.getClusterName().value(), clusterState.metadata().clusterUUID()) .size(), @@ -421,7 +421,7 @@ public void testReadLatestMetadataManifestSuccessButIndexMetadataFetchIOExceptio mockBlobContainer(blobContainer, expectedManifest, Map.of()); when(blobContainer.readBlob(uploadedIndexMetadata.getUploadedFilename() + ".dat")).thenThrow(FileNotFoundException.class); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); Exception e = assertThrows( IllegalStateException.class, () -> remoteClusterStateService.getLatestIndexMetadata( @@ -449,11 +449,11 @@ public void testReadLatestMetadataManifestSuccess() throws IOException { .build(); mockBlobContainer(mockBlobStoreObjects(), expectedManifest, new HashMap<>()); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); final ClusterMetadataManifest manifest = remoteClusterStateService.getLatestClusterMetadataManifest( clusterState.getClusterName().value(), clusterState.metadata().clusterUUID() - ); + ).get(); assertThat(manifest.getIndices().size(), is(1)); assertThat(manifest.getIndices().get(0).getIndexName(), is(uploadedIndexMetadata.getIndexName())); @@ -467,7 +467,7 @@ public void testReadLatestMetadataManifestSuccess() throws IOException { public void testReadLatestIndexMetadataSuccess() throws IOException { final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); final Index index = new Index("test-index", "index-uuid"); String fileName = "metadata-" + index.getUUID(); @@ -509,7 +509,7 @@ public void testReadLatestIndexMetadataSuccess() throws IOException { public void testMarkLastStateAsCommittedSuccess() throws IOException { final ClusterState clusterState = generateClusterStateWithOneIndex().nodes(nodesWithLocalNodeClusterManager()).build(); mockBlobStoreObjects(); - remoteClusterStateService.ensureRepositorySet(); + remoteClusterStateService.start(); final UploadedIndexMetadata uploadedIndexMetadata = new UploadedIndexMetadata("test-index", "index-uuid", "metadata-filename"); List indices = List.of(uploadedIndexMetadata); final ClusterMetadataManifest previousManifest = ClusterMetadataManifest.builder().indices(indices).build(); @@ -547,8 +547,8 @@ public void testGetValidPreviousClusterUUID() throws IOException { ); mockObjectsForGettingPreviousClusterUUID(clusterUUIDsPointers); - remoteClusterStateService.ensureRepositorySet(); - String previousClusterUUID = remoteClusterStateService.getLatestClusterUUID("test-cluster"); + remoteClusterStateService.start(); + String previousClusterUUID = remoteClusterStateService.getLastKnownUUIDFromRemote("test-cluster"); assertThat(previousClusterUUID, equalTo("cluster-uuid3")); } @@ -563,8 +563,8 @@ public void testGetValidPreviousClusterUUIDForInvalidChain() throws IOException ); mockObjectsForGettingPreviousClusterUUID(clusterUUIDsPointers); - remoteClusterStateService.ensureRepositorySet(); - assertThrows(IllegalStateException.class, () -> remoteClusterStateService.getLatestClusterUUID("test-cluster")); + remoteClusterStateService.start(); + assertThrows(IllegalStateException.class, () -> remoteClusterStateService.getLastKnownUUIDFromRemote("test-cluster")); } private void mockObjectsForGettingPreviousClusterUUID(Map clusterUUIDsPointers) throws IOException { diff --git a/server/src/test/java/org/opensearch/repositories/blobstore/BlobStoreRepositoryRemoteIndexTests.java b/server/src/test/java/org/opensearch/repositories/blobstore/BlobStoreRepositoryRemoteIndexTests.java index 1fa623430379f..7bcc5aba6ce05 100644 --- a/server/src/test/java/org/opensearch/repositories/blobstore/BlobStoreRepositoryRemoteIndexTests.java +++ b/server/src/test/java/org/opensearch/repositories/blobstore/BlobStoreRepositoryRemoteIndexTests.java @@ -104,7 +104,7 @@ private Settings buildRemoteStoreNodeAttributes(String repoName, Path repoPath) .put("node.attr." + REMOTE_STORE_CLUSTER_STATE_REPOSITORY_NAME_ATTRIBUTE_KEY, repoName) .put(repoTypeAttributeKey, FsRepository.TYPE) .put(repoSettingsAttributeKeyPrefix + "location", repoPath) - .put(RemoteClusterStateService.REMOTE_CLUSTER_STATE_ENABLED_SETTING.getKey(), true) + .put(RemoteClusterStateService.REMOTE_CLUSTER_STATE_ENABLED_SETTING.getKey(), false) .build(); } diff --git a/test/framework/src/main/java/org/opensearch/gateway/MockGatewayMetaState.java b/test/framework/src/main/java/org/opensearch/gateway/MockGatewayMetaState.java index bf59e9ff06cbe..dea205619ce95 100644 --- a/test/framework/src/main/java/org/opensearch/gateway/MockGatewayMetaState.java +++ b/test/framework/src/main/java/org/opensearch/gateway/MockGatewayMetaState.java @@ -46,16 +46,13 @@ import org.opensearch.core.xcontent.NamedXContentRegistry; import org.opensearch.env.NodeEnvironment; import org.opensearch.gateway.remote.RemoteClusterStateService; +import org.opensearch.index.recovery.RemoteStoreRestoreService; import org.opensearch.plugins.MetadataUpgrader; -import org.opensearch.repositories.RepositoriesService; import org.opensearch.threadpool.ThreadPool; import org.opensearch.transport.TransportService; import java.io.IOException; -import java.util.Collections; -import java.util.function.Supplier; -import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.isRemoteStoreClusterStateEnabled; import static org.mockito.Mockito.mock; import static org.mockito.Mockito.when; @@ -68,10 +65,26 @@ public class MockGatewayMetaState extends GatewayMetaState { private final DiscoveryNode localNode; private final BigArrays bigArrays; + private final RemoteClusterStateService remoteClusterStateService; + private final RemoteStoreRestoreService remoteStoreRestoreService; public MockGatewayMetaState(DiscoveryNode localNode, BigArrays bigArrays) { this.localNode = localNode; this.bigArrays = bigArrays; + this.remoteClusterStateService = mock(RemoteClusterStateService.class); + this.remoteStoreRestoreService = mock(RemoteStoreRestoreService.class); + } + + public MockGatewayMetaState( + DiscoveryNode localNode, + BigArrays bigArrays, + RemoteClusterStateService remoteClusterStateService, + RemoteStoreRestoreService remoteStoreRestoreService + ) { + this.localNode = localNode; + this.bigArrays = bigArrays; + this.remoteClusterStateService = remoteClusterStateService; + this.remoteStoreRestoreService = remoteStoreRestoreService; } @Override @@ -108,26 +121,6 @@ public void start( } catch (IOException e) { throw new AssertionError(e); } - Supplier remoteClusterStateServiceSupplier = () -> { - if (isRemoteStoreClusterStateEnabled(settings)) { - return new RemoteClusterStateService( - nodeEnvironment.nodeId(), - () -> new RepositoriesService( - settings, - clusterService, - transportService, - Collections.emptyMap(), - Collections.emptyMap(), - transportService.getThreadPool() - ), - settings, - new ClusterSettings(settings, ClusterSettings.BUILT_IN_CLUSTER_SETTINGS), - () -> 0L - ); - } else { - return null; - } - }; start( settings, transportService, @@ -142,8 +135,9 @@ public void start( new ClusterSettings(settings, ClusterSettings.BUILT_IN_CLUSTER_SETTINGS), () -> 0L ), - remoteClusterStateServiceSupplier.get(), - persistedStateRegistry + remoteClusterStateService, + persistedStateRegistry, + remoteStoreRestoreService ); } }