Skip to content

Commit

Permalink
Add dummy methods in RemoteClusterStateService
Browse files Browse the repository at this point in the history
Signed-off-by: Sooraj Sinha <soosinha@amazon.com>
  • Loading branch information
soosinha committed Jun 9, 2024
1 parent f4c2a04 commit b7a4586
Show file tree
Hide file tree
Showing 14 changed files with 192 additions and 60 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -40,6 +40,7 @@
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.util.io.IOUtils;
import org.opensearch.node.remotestore.RemoteStoreNodeAttribute;

import java.io.Closeable;
import java.io.IOException;
Expand All @@ -50,10 +51,8 @@
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import org.opensearch.node.remotestore.RemoteStoreNodeAttribute;

import static org.opensearch.cluster.coordination.Coordinator.ZEN1_BWC_TERM;
import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.isRemoteRoutingTableEnabled;
import static org.opensearch.node.remotestore.RemoteStoreNodeAttribute.isRemoteStoreClusterStateEnabled;

/**
Expand Down Expand Up @@ -583,7 +582,7 @@ public void handlePrePublish(ClusterState clusterState) {
// This is to ensure the remote store is the single source of truth for current state. Even if the current node
// goes down after sending the cluster state to other nodes, we should be able to read the remote state and
// recover the cluster.
if (isRemotePublicationEnabled) {
if (isRemoteStateEnabled) {
assert persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE) != null : "Remote state has not been initialized";
persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE).setLastAcceptedState(clusterState);
}
Expand All @@ -594,7 +593,7 @@ public void handlePrePublish(ClusterState clusterState) {
*/
public void handlePreCommit() {
// Publishing the committed state to remote store before sending apply commit to other nodes.
if (isRemotePublicationEnabled) {
if (isRemoteStateEnabled) {
assert persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE) != null : "Remote state has not been initialized";
persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE).markLastAcceptedStateAsCommitted();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -49,7 +49,6 @@
import org.opensearch.cluster.coordination.CoordinationState.VoteCollection;
import org.opensearch.cluster.coordination.FollowersChecker.FollowerCheckRequest;
import org.opensearch.cluster.coordination.JoinHelper.InitialJoinAccumulator;
import org.opensearch.cluster.coordination.PersistedStateRegistry.PersistedStateType;
import org.opensearch.cluster.metadata.Metadata;
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.cluster.node.DiscoveryNodes;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,9 +31,6 @@

package org.opensearch.cluster.coordination;

import java.util.Locale;
import java.util.Optional;
import java.util.function.Supplier;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
import org.apache.logging.log4j.message.ParameterizedMessage;
Expand All @@ -43,7 +40,6 @@
import org.opensearch.cluster.ClusterState;
import org.opensearch.cluster.Diff;
import org.opensearch.cluster.IncompatibleClusterStateVersionException;
import org.opensearch.cluster.coordination.CoordinationState.PersistedState;
import org.opensearch.cluster.coordination.PersistedStateRegistry.PersistedStateType;
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.cluster.node.DiscoveryNodes;
Expand All @@ -65,9 +61,7 @@

import java.io.IOException;
import java.util.HashMap;
import java.util.Locale;
import java.util.Map;
import java.util.Optional;
import java.util.concurrent.atomic.AtomicLong;
import java.util.concurrent.atomic.AtomicReference;
import java.util.function.BiConsumer;
Expand Down Expand Up @@ -238,7 +232,10 @@ private PublishWithJoinResponse handleIncomingRemotePublishRequest(RemotePublish
if (transportService.getLocalNode().equals(request.getSourceNode())) {
return acceptStateOnLocalNode(request);
}
ClusterMetadataManifest manifest = remoteClusterStateService.getClusterMetadataManifestByFileName(request.getClusterUUID(), request.getManifestFile());
ClusterMetadataManifest manifest = remoteClusterStateService.getClusterMetadataManifestByFileName(
request.getClusterUUID(),
request.getManifestFile()
);
boolean applyFullState = false;
final ClusterState lastSeen = lastSeenClusterState.get();
if (lastSeen == null) {
Expand All @@ -253,17 +250,40 @@ private PublishWithJoinResponse handleIncomingRemotePublishRequest(RemotePublish
}

if (applyFullState == true) {
logger.debug(() -> new ParameterizedMessage("Downloading full cluster state for term {}, version {}, stateUUID {}", manifest.getClusterTerm(), manifest.getStateVersion(),
manifest.getStateUUID()));
ClusterState clusterState = remoteClusterStateService.getClusterStateForManifest(request.getClusterName(), manifest, transportService.getLocalNode().getId(), true);
logger.debug(
() -> new ParameterizedMessage(
"Downloading full cluster state for term {}, version {}, stateUUID {}",
manifest.getClusterTerm(),
manifest.getStateVersion(),
manifest.getStateUUID()
)
);
ClusterState clusterState = remoteClusterStateService.getClusterStateForManifest(
request.getClusterName(),
manifest,
transportService.getLocalNode().getId(),
true
);
fullClusterStateReceivedCount.incrementAndGet();
final PublishWithJoinResponse response = acceptState(clusterState);
lastSeenClusterState.set(clusterState);
return response;
} else {
logger.debug(() -> new ParameterizedMessage("Downloading diff cluster state for term {}, version {}, previousUUID {}, current UUID {}", manifest.getClusterTerm(),
manifest.getStateVersion(), manifest.getDiffManifest().getFromStateUUID(), manifest.getStateUUID()));
ClusterState clusterState = remoteClusterStateService.getClusterStateUsingDiff(request.getClusterName(), manifest, lastSeen, transportService.getLocalNode().getId());
logger.debug(
() -> new ParameterizedMessage(
"Downloading diff cluster state for term {}, version {}, previousUUID {}, current UUID {}",
manifest.getClusterTerm(),
manifest.getStateVersion(),
manifest.getDiffManifest().getFromStateUUID(),
manifest.getStateUUID()
)
);
ClusterState clusterState = remoteClusterStateService.getClusterStateUsingDiff(
request.getClusterName(),
manifest,
lastSeen,
transportService.getLocalNode().getId()
);
compatibleClusterStateDiffReceivedCount.incrementAndGet();
final PublishWithJoinResponse response = acceptState(clusterState);
lastSeenClusterState.compareAndSet(lastSeen, clusterState);
Expand All @@ -286,7 +306,8 @@ private PublishWithJoinResponse acceptState(ClusterState incomingState) {

private PublishWithJoinResponse acceptStateOnLocalNode(RemotePublishRequest remotePublishRequest) {
final PublishRequest publishRequest = currentPublishRequestToSelf.get();
if (publishRequest == null || publishRequest.getAcceptedState().coordinationMetadata().term() != remotePublishRequest.term
if (publishRequest == null
|| publishRequest.getAcceptedState().coordinationMetadata().term() != remotePublishRequest.term
|| publishRequest.getAcceptedState().version() != remotePublishRequest.version) {
throw new IllegalStateException("publication to self failed for " + remotePublishRequest);
}
Expand All @@ -295,9 +316,16 @@ private PublishWithJoinResponse acceptStateOnLocalNode(RemotePublishRequest remo
return publishWithJoinResponse;
}

public PublicationContext newPublicationContext(ClusterChangedEvent clusterChangedEvent, boolean isRemotePublicationEnabled,
PersistedStateRegistry persistedStateRegistry) {
final PublicationContext publicationContext = new PublicationContext(clusterChangedEvent, isRemotePublicationEnabled, persistedStateRegistry);
public PublicationContext newPublicationContext(
ClusterChangedEvent clusterChangedEvent,
boolean isRemotePublicationEnabled,
PersistedStateRegistry persistedStateRegistry
) {
final PublicationContext publicationContext = new PublicationContext(
clusterChangedEvent,
isRemotePublicationEnabled,
persistedStateRegistry
);

// Build the serializations we expect to need now, early in the process, so that an error during serialization fails the publication
// straight away. This isn't watertight since we send diffs on a best-effort basis and may fall back to sending a full state (and
Expand Down Expand Up @@ -345,7 +373,11 @@ public class PublicationContext {
private final boolean sendRemoteState;
private final PersistedStateRegistry persistedStateRegistry;

PublicationContext(ClusterChangedEvent clusterChangedEvent, boolean isRemotePublicationEnabled, PersistedStateRegistry persistedStateRegistry) {
PublicationContext(
ClusterChangedEvent clusterChangedEvent,
boolean isRemotePublicationEnabled,
PersistedStateRegistry persistedStateRegistry
) {
discoveryNodes = clusterChangedEvent.state().nodes();
newState = clusterChangedEvent.state();
previousState = clusterChangedEvent.previousState();
Expand Down Expand Up @@ -462,11 +494,22 @@ public String executor() {
);
}

private void sendRemoteClusterState(final DiscoveryNode destination, final ClusterState clusterState, final ActionListener<PublishWithJoinResponse> listener) {
private void sendRemoteClusterState(
final DiscoveryNode destination,
final ClusterState clusterState,
final ActionListener<PublishWithJoinResponse> listener
) {
try {
final String manifestFileName = ((RemotePersistedState) persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE)).getLastUploadedManifestFile();
final RemotePublishRequest remotePublishRequest = new RemotePublishRequest(discoveryNodes.getLocalNode(), clusterState.term(),
clusterState.getVersion(), clusterState.getClusterName().value(), clusterState.metadata().clusterUUID(), manifestFileName);
final String manifestFileName = ((RemotePersistedState) persistedStateRegistry.getPersistedState(PersistedStateType.REMOTE))
.getLastUploadedManifestFile();
final RemotePublishRequest remotePublishRequest = new RemotePublishRequest(
discoveryNodes.getLocalNode(),
clusterState.term(),
clusterState.getVersion(),
clusterState.getClusterName().value(),
clusterState.metadata().clusterUUID(),
manifestFileName
);
final Consumer<TransportException> transportExceptionHandler = exp -> {
logger.debug(() -> new ParameterizedMessage("failed to send remote cluster state to {}", destination), exp);
listener.onFailure(exp);
Expand All @@ -493,7 +536,13 @@ public String executor() {
return ThreadPool.Names.GENERIC;
}
};
transportService.sendRequest(destination, PUBLISH_REMOTE_STATE_ACTION_NAME, remotePublishRequest, stateRequestOptions, responseHandler);
transportService.sendRequest(
destination,
PUBLISH_REMOTE_STATE_ACTION_NAME,
remotePublishRequest,
stateRequestOptions,
responseHandler
);
} catch (Exception e) {
logger.warn(() -> new ParameterizedMessage("error sending remote cluster state to {}", destination), e);
listener.onFailure(e);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -14,13 +14,24 @@

import java.io.IOException;

/**
* Send the publish request with the remote cluster state details
* @opensearch.internal
*/
public class RemotePublishRequest extends TermVersionRequest {

private final String clusterName;
private final String clusterUUID;
private final String manifestFile;

public RemotePublishRequest(DiscoveryNode sourceNode, long term, long version, String clusterName, String clusterUUID, String manifestFile) {
public RemotePublishRequest(
DiscoveryNode sourceNode,
long term,
long version,
String clusterName,
String clusterUUID,
String manifestFile
) {
super(sourceNode, term, version);
this.clusterName = clusterName;
this.clusterUUID = clusterUUID;
Expand All @@ -44,8 +55,20 @@ public void writeTo(StreamOutput out) throws IOException {

@Override
public String toString() {
return "RemotePublishRequest{" + "term=" + term + ", version=" + version + ", clusterName=" + clusterName + ", clusterUUID=" + clusterUUID
+ ", sourceNode=" + sourceNode + ", manifestFile=" + manifestFile + '}';
return "RemotePublishRequest{"
+ "term="
+ term
+ ", version="
+ version
+ ", clusterName="
+ clusterName
+ ", clusterUUID="
+ clusterUUID
+ ", sourceNode="
+ sourceNode
+ ", manifestFile="
+ manifestFile
+ '}';
}

public String getClusterName() {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,7 +33,6 @@
package org.opensearch.cluster.node;

import org.opensearch.Version;
import org.opensearch.cluster.metadata.Metadata;
import org.opensearch.common.UUIDs;
import org.opensearch.common.annotation.PublicApi;
import org.opensearch.common.settings.Setting;
Expand All @@ -44,7 +43,6 @@
import org.opensearch.core.common.transport.TransportAddress;
import org.opensearch.core.xcontent.ToXContentFragment;
import org.opensearch.core.xcontent.XContentBuilder;
import org.opensearch.core.xcontent.XContentParser;
import org.opensearch.node.Node;

import java.io.IOException;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -77,7 +77,6 @@
import org.opensearch.cluster.routing.allocation.decider.SameShardAllocationDecider;
import org.opensearch.cluster.routing.allocation.decider.ShardsLimitAllocationDecider;
import org.opensearch.cluster.routing.allocation.decider.ThrottlingAllocationDecider;
import org.opensearch.cluster.routing.remote.RemoteRoutingTableService;
import org.opensearch.cluster.service.ClusterApplierService;
import org.opensearch.cluster.service.ClusterManagerService;
import org.opensearch.cluster.service.ClusterManagerTaskThrottler;
Expand Down Expand Up @@ -716,6 +715,9 @@ public void apply(Settings value, Settings current, Settings previous) {
// Remote cluster state settings
RemoteClusterStateCleanupManager.REMOTE_CLUSTER_STATE_CLEANUP_INTERVAL_SETTING,
RemoteClusterStateService.REMOTE_CLUSTER_STATE_ENABLED_SETTING,
RemoteClusterStateService.INDEX_METADATA_UPLOAD_TIMEOUT_SETTING,
RemoteClusterStateService.GLOBAL_METADATA_UPLOAD_TIMEOUT_SETTING,
RemoteClusterStateService.METADATA_MANIFEST_UPLOAD_TIMEOUT_SETTING,
RemoteStoreNodeService.REMOTE_STORE_COMPATIBILITY_MODE_SETTING,
RemoteStoreNodeService.MIGRATION_DIRECTION_SETTING,
IndicesService.CLUSTER_REMOTE_INDEX_RESTRICT_ASYNC_DURABILITY_SETTING,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -722,9 +722,14 @@ public void setLastAcceptedState(ClusterState clusterState) {
} else {
assert verifyManifestAndClusterState(lastAcceptedManifest, lastAcceptedState) == true
: "Previous manifest and previous ClusterState are not in sync";
manifestDetails = remoteClusterStateService.writeIncrementalMetadata(lastAcceptedState, clusterState, lastAcceptedManifest);
manifestDetails = remoteClusterStateService.writeIncrementalMetadata(
lastAcceptedState,
clusterState,
lastAcceptedManifest
);
}
assert verifyManifestAndClusterState(manifestDetails.getClusterMetadataManifest(), clusterState) == true : "Manifest and ClusterState are not in sync";
assert verifyManifestAndClusterState(manifestDetails.getClusterMetadataManifest(), clusterState) == true
: "Manifest and ClusterState are not in sync";
lastAcceptedManifest = manifestDetails.getClusterMetadataManifest();
lastAcceptedState = clusterState;
lastUploadedManifestFile = manifestDetails.getManifestFileName();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -332,6 +332,11 @@ public Map<String, UploadedMetadataAttribute> getCustomMetadataMap() {
return uploadedCustomMetadataMap;
}

// TODO https://github.com/opensearch-project/OpenSearch/pull/14089
public ClusterStateDiffManifest getDiffManifest() {
return new ClusterStateDiffManifest();
}

public boolean hasMetadataAttributesFiles() {
return uploadedCoordinationMetadata != null
|| uploadedSettingsMetadata != null
Expand Down Expand Up @@ -991,4 +996,11 @@ public String toString() {
+ '}';
}
}

// TODO https://github.com/opensearch-project/OpenSearch/pull/14089
public static class ClusterStateDiffManifest {
public String getFromStateUUID() {
return null;
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -1092,6 +1092,31 @@ public ClusterState getLatestClusterState(String clusterName, String clusterUUID
.build();
}

public ClusterState getClusterStateForManifest(
String clusterName,
ClusterMetadataManifest manifest,
String localNodeId,
boolean includeEphemeral
) {
// TODO https://github.com/opensearch-project/OpenSearch/pull/14089
return null;
}

public ClusterState getClusterStateUsingDiff(
String clusterName,
ClusterMetadataManifest manifest,
ClusterState previousClusterState,
String localNodeId
) {
// TODO https://github.com/opensearch-project/OpenSearch/pull/14089
return null;
}

public ClusterMetadataManifest getClusterMetadataManifestByFileName(String clusterUUID, String manifestFileName) {
// TODO https://github.com/opensearch-project/OpenSearch/pull/14089
return null;
}

private Metadata getGlobalMetadata(String clusterName, String clusterUUID, ClusterMetadataManifest clusterMetadataManifest) {
String globalMetadataFileName = clusterMetadataManifest.getGlobalMetadataFileName();
try {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -202,7 +202,8 @@ private static boolean isRemoteRoutingTableAttributePresent(Settings settings) {
}

public static boolean isRemotePublicationEnabled(Settings settings) {
return FeatureFlags.isEnabled(REMOTE_PUBLICATION_EXPERIMENTAL) && isRemoteRoutingTableAttributePresent(settings)
return FeatureFlags.isEnabled(REMOTE_PUBLICATION_EXPERIMENTAL)
&& isRemoteRoutingTableAttributePresent(settings)
&& isRemoteStoreClusterStateEnabled(settings);
}

Expand Down
Loading

0 comments on commit b7a4586

Please sign in to comment.