[main] Move system indices migration to migrate plugin (#125437)

* [main] Move system indices migration to migrate plugin

It seems the best way to fix #122949 is to use existing data stream reindex API. However, this API is located in the migrate x-pack plugin. This commit moves the system indices migration logic (REST handlers, transport actions, and task) to the migrate plugin.

Port of #123551

* [CI] Auto commit changes from spotless

* Fix compilation

* Fix tests

* Fix test

---------

Co-authored-by: elasticsearchmachine <infra-root+elasticsearchmachine@elastic.co>
This commit is contained in:
Alexey Ivanov 2025-04-04 18:49:38 +01:00 committed by GitHub
parent 5706e075ad
commit fd7efe587e
No known key found for this signature in database
GPG key ID: B5690EEEBB952194
55 changed files with 696 additions and 519 deletions

View file

@ -44,9 +44,6 @@ dependencies {
clusterModules project(':modules:lang-painless')
clusterModules project(':modules:parent-join')
clusterModules project(":modules:rest-root")
internalClusterTestImplementation project(':modules:lang-painless')
internalClusterTestImplementation project(':modules:lang-painless:spi')
}
restResources {

View file

@ -10,9 +10,6 @@
grant {
// reindex opens socket connections using the rest client
permission java.net.SocketPermission "*", "connect";
// needed for Painless to generate runtime classes
permission java.lang.RuntimePermission "createClassLoader";
};
grant codeBase "${codebase.elasticsearch-rest-client}" {

View file

@ -11,9 +11,9 @@ package org.elasticsearch.upgrades;
import com.carrotsearch.randomizedtesting.annotations.Name;
import org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.client.Request;
import org.elasticsearch.client.ResponseException;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.test.XContentTestUtils;
import org.junit.BeforeClass;
@ -112,7 +112,7 @@ public class FeatureUpgradeIT extends AbstractRollingUpgradeTestCase {
// for the next major version upgrade (see e.g. #93666). Trying to express this with features may be problematic, so we
// want to keep using versions here. We also assume that for non-semantic version migrations are not required.
boolean migrationNeeded = parseLegacyVersion(getOldClusterVersion()).map(
v -> v.before(TransportGetFeatureUpgradeStatusAction.NO_UPGRADE_REQUIRED_VERSION)
v -> v.before(SystemIndices.NO_UPGRADE_REQUIRED_VERSION)
).orElse(false);
if (migrationNeeded) {
assertThat(feature.get("migration_status"), equalTo("MIGRATION_NEEDED"));

View file

@ -17,3 +17,8 @@ esplugin {
licenseFile = layout.settingsDirectory.file('licenses/AGPL-3.0+SSPL-1.0+ELASTIC-LICENSE-2.0.txt').asFile
noticeFile = layout.settingsDirectory.file('NOTICE.txt').asFile
}
dependencies {
clusterModules project(':modules:ingest-common')
clusterModules project(':x-pack:plugin:migrate')
}

View file

@ -25,6 +25,8 @@ public abstract class AbstractSystemIndicesIT extends ESRestTestCase {
@ClassRule
public static ElasticsearchCluster cluster = ElasticsearchCluster.local()
.plugin("system-indices-qa")
.module("ingest-common")
.module("x-pack-migrate")
.setting("xpack.security.enabled", "true")
.setting("xpack.security.autoconfiguration.enabled", "false")
.user("rest_user", "rest-user-password")

View file

@ -62,7 +62,6 @@ module org.elasticsearch.server {
exports org.elasticsearch.action.admin.cluster.coordination;
exports org.elasticsearch.action.admin.cluster.desirednodes;
exports org.elasticsearch.action.admin.cluster.health;
exports org.elasticsearch.action.admin.cluster.migration;
exports org.elasticsearch.action.admin.cluster.node.capabilities;
exports org.elasticsearch.action.admin.cluster.node.hotthreads;
exports org.elasticsearch.action.admin.cluster.node.info;
@ -382,7 +381,6 @@ module org.elasticsearch.server {
exports org.elasticsearch.tasks;
exports org.elasticsearch.threadpool;
exports org.elasticsearch.transport;
exports org.elasticsearch.upgrades;
exports org.elasticsearch.usage;
exports org.elasticsearch.watcher;

View file

@ -26,10 +26,6 @@ import org.elasticsearch.action.admin.cluster.desirednodes.TransportGetDesiredNo
import org.elasticsearch.action.admin.cluster.desirednodes.TransportUpdateDesiredNodesAction;
import org.elasticsearch.action.admin.cluster.desirednodes.UpdateDesiredNodesAction;
import org.elasticsearch.action.admin.cluster.health.TransportClusterHealthAction;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.TransportPostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.node.capabilities.TransportNodesCapabilitiesAction;
import org.elasticsearch.action.admin.cluster.node.features.TransportNodesFeaturesAction;
import org.elasticsearch.action.admin.cluster.node.hotthreads.TransportNodesHotThreadsAction;
@ -286,7 +282,6 @@ import org.elasticsearch.rest.action.admin.cluster.RestDeleteSnapshotAction;
import org.elasticsearch.rest.action.admin.cluster.RestDeleteStoredScriptAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetDesiredBalanceAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetDesiredNodesAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetFeatureUpgradeStatusAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetRepositoriesAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetScriptContextAction;
import org.elasticsearch.rest.action.admin.cluster.RestGetScriptLanguageAction;
@ -300,7 +295,6 @@ import org.elasticsearch.rest.action.admin.cluster.RestNodesInfoAction;
import org.elasticsearch.rest.action.admin.cluster.RestNodesStatsAction;
import org.elasticsearch.rest.action.admin.cluster.RestNodesUsageAction;
import org.elasticsearch.rest.action.admin.cluster.RestPendingClusterTasksAction;
import org.elasticsearch.rest.action.admin.cluster.RestPostFeatureUpgradeAction;
import org.elasticsearch.rest.action.admin.cluster.RestPrevalidateNodeRemovalAction;
import org.elasticsearch.rest.action.admin.cluster.RestPutRepositoryAction;
import org.elasticsearch.rest.action.admin.cluster.RestPutStoredScriptAction;
@ -681,10 +675,7 @@ public class ActionModule extends AbstractModule {
actions.register(TransportSnapshotsStatusAction.TYPE, TransportSnapshotsStatusAction.class);
actions.register(SnapshottableFeaturesAction.INSTANCE, TransportSnapshottableFeaturesAction.class);
actions.register(ResetFeatureStateAction.INSTANCE, TransportResetFeatureStateAction.class);
actions.register(GetFeatureUpgradeStatusAction.INSTANCE, TransportGetFeatureUpgradeStatusAction.class);
actions.register(PostFeatureUpgradeAction.INSTANCE, TransportPostFeatureUpgradeAction.class);
actions.register(TransportGetShardSnapshotAction.TYPE, TransportGetShardSnapshotAction.class);
actions.register(IndicesStatsAction.INSTANCE, TransportIndicesStatsAction.class);
actions.register(IndicesSegmentsAction.INSTANCE, TransportIndicesSegmentsAction.class);
actions.register(TransportIndicesShardStoresAction.TYPE, TransportIndicesShardStoresAction.class);
@ -882,8 +873,6 @@ public class ActionModule extends AbstractModule {
registerHandler.accept(new RestSnapshotsStatusAction());
registerHandler.accept(new RestSnapshottableFeaturesAction());
registerHandler.accept(new RestResetFeatureStateAction());
registerHandler.accept(new RestGetFeatureUpgradeStatusAction());
registerHandler.accept(new RestPostFeatureUpgradeAction());
registerHandler.accept(new RestGetIndicesAction());
registerHandler.accept(new RestIndicesStatsAction());
registerHandler.accept(new RestIndicesSegmentsAction());

View file

@ -148,7 +148,7 @@ public class TransportCreateIndexAction extends TransportMasterNodeAction<Create
listener.onFailure(new IllegalStateException(message));
return;
}
updateRequest = buildSystemIndexUpdateRequest(request, cause, descriptor, projectId);
updateRequest = buildManagedSystemIndexUpdateRequest(request, cause, descriptor, projectId);
} else {
updateRequest = buildUpdateRequest(request, cause, indexName, resolvedAt, projectId);
}
@ -181,31 +181,55 @@ public class TransportCreateIndexAction extends TransportMasterNodeAction<Create
.waitForActiveShards(request.waitForActiveShards());
}
private static CreateIndexClusterStateUpdateRequest buildSystemIndexUpdateRequest(
private static CreateIndexClusterStateUpdateRequest buildManagedSystemIndexUpdateRequest(
CreateIndexRequest request,
String cause,
SystemIndexDescriptor descriptor,
ProjectId projectId
) {
final Settings settings = Objects.requireNonNullElse(descriptor.getSettings(), Settings.EMPTY);
boolean indexMigrationInProgress = cause.equals(SystemIndices.MIGRATE_SYSTEM_INDEX_CAUSE)
&& request.index().endsWith(SystemIndices.UPGRADED_INDEX_SUFFIX);
final Settings settings;
final String mappings;
final Set<Alias> aliases;
if (descriptor.getAliasName() == null) {
final String indexName;
// if we are migrating a system index to a new index, we use settings/mappings/index name from the request,
// since it was created by SystemIndexMigrator
if (indexMigrationInProgress) {
settings = request.settings();
mappings = request.mappings();
indexName = request.index();
// we will update alias later on
aliases = Set.of();
} else {
aliases = Set.of(new Alias(descriptor.getAliasName()).isHidden(true).writeIndex(true));
settings = Objects.requireNonNullElse(descriptor.getSettings(), Settings.EMPTY);
mappings = descriptor.getMappings();
if (descriptor.getAliasName() == null) {
aliases = Set.of();
} else {
aliases = Set.of(new Alias(descriptor.getAliasName()).isHidden(true).writeIndex(true));
}
// Throw an error if we are trying to directly create a system index other
// than the primary system index (or the alias, or we are migrating the index)
if (request.index().equals(descriptor.getPrimaryIndex()) == false
&& request.index().equals(descriptor.getAliasName()) == false) {
throw new IllegalArgumentException(
"Cannot create system index with name "
+ request.index()
+ "; descriptor primary index is "
+ descriptor.getPrimaryIndex()
);
}
indexName = descriptor.getPrimaryIndex();
}
// Throw an error if we are trying to directly create a system index other than the primary system index (or the alias)
if (request.index().equals(descriptor.getPrimaryIndex()) == false && request.index().equals(descriptor.getAliasName()) == false) {
throw new IllegalArgumentException(
"Cannot create system index with name " + request.index() + "; descriptor primary index is " + descriptor.getPrimaryIndex()
);
}
return new CreateIndexClusterStateUpdateRequest(cause, projectId, descriptor.getPrimaryIndex(), request.index()).aliases(aliases)
return new CreateIndexClusterStateUpdateRequest(cause, projectId, indexName, request.index()).aliases(aliases)
.waitForActiveShards(ActiveShardCount.ALL)
.mappings(descriptor.getMappings())
.mappings(mappings)
.settings(settings);
}
}

View file

@ -89,7 +89,6 @@ import org.elasticsearch.tasks.Task;
import org.elasticsearch.tasks.TaskResultsService;
import org.elasticsearch.telemetry.TelemetryProvider;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.upgrades.FeatureMigrationResults;
import org.elasticsearch.xcontent.NamedXContentRegistry;
import org.elasticsearch.xcontent.ParseField;
@ -257,7 +256,6 @@ public class ClusterModule extends AbstractModule {
ComposableIndexTemplateMetadata::readDiffFrom
);
registerProjectCustom(entries, DataStreamMetadata.TYPE, DataStreamMetadata::new, DataStreamMetadata::readDiffFrom);
registerProjectCustom(entries, FeatureMigrationResults.TYPE, FeatureMigrationResults::new, FeatureMigrationResults::readDiffFrom);
registerMetadataCustom(entries, NodesShutdownMetadata.TYPE, NodesShutdownMetadata::new, NodesShutdownMetadata::readDiffFrom);
registerMetadataCustom(entries, DesiredNodesMetadata.TYPE, DesiredNodesMetadata::new, DesiredNodesMetadata::readDiffFrom);
registerProjectCustom(

View file

@ -13,7 +13,6 @@ import org.apache.lucene.util.automaton.Automaton;
import org.apache.lucene.util.automaton.CharacterRunAutomaton;
import org.apache.lucene.util.automaton.Operations;
import org.apache.lucene.util.automaton.RegExp;
import org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.indices.create.AutoCreateAction;
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
import org.elasticsearch.cluster.metadata.IndexMetadata;
@ -148,7 +147,7 @@ public class SystemIndexDescriptor implements IndexPatternMatcher, Comparable<Sy
/**
* An optional reindexing script to use when migrating an index created
* before {@link TransportGetFeatureUpgradeStatusAction#NO_UPGRADE_REQUIRED_INDEX_VERSION}.
* before {@link SystemIndices#NO_UPGRADE_REQUIRED_INDEX_VERSION}.
* This script can be used to modify documents before they are added to the new index.
* For example, it can be used to remove deprecated fields from the index.
* <br>

View file

@ -15,8 +15,8 @@ import org.apache.lucene.util.automaton.Automata;
import org.apache.lucene.util.automaton.Automaton;
import org.apache.lucene.util.automaton.CharacterRunAutomaton;
import org.apache.lucene.util.automaton.Operations;
import org.elasticsearch.Version;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.snapshots.features.ResetFeatureStateResponse.ResetFeatureStateStatus;
import org.elasticsearch.action.admin.indices.delete.DeleteIndexRequest;
import org.elasticsearch.action.admin.indices.delete.TransportDeleteIndexAction;
@ -36,7 +36,10 @@ import org.elasticsearch.core.CheckedConsumer;
import org.elasticsearch.core.Nullable;
import org.elasticsearch.core.Predicates;
import org.elasticsearch.core.Tuple;
import org.elasticsearch.core.UpdateForV10;
import org.elasticsearch.index.Index;
import org.elasticsearch.index.IndexVersion;
import org.elasticsearch.index.IndexVersions;
import org.elasticsearch.plugins.SystemIndexPlugin;
import org.elasticsearch.snapshots.SnapshotsService;
@ -111,7 +114,16 @@ import static org.elasticsearch.tasks.TaskResultsService.TASKS_FEATURE_NAME;
public class SystemIndices {
public static final String SYSTEM_INDEX_ACCESS_CONTROL_HEADER_KEY = "_system_index_access_allowed";
public static final String EXTERNAL_SYSTEM_INDEX_ACCESS_CONTROL_HEADER_KEY = "_external_system_index_access_origin";
private static final int UPGRADED_TO_VERSION = TransportGetFeatureUpgradeStatusAction.NO_UPGRADE_REQUIRED_VERSION.major + 1;
/**
* These versions should be set to current major and current major's index version
*/
@UpdateForV10(owner = UpdateForV10.Owner.CORE_INFRA)
public static final Version NO_UPGRADE_REQUIRED_VERSION = Version.V_9_0_0;
public static final IndexVersion NO_UPGRADE_REQUIRED_INDEX_VERSION = IndexVersions.UPGRADE_TO_LUCENE_10_0_0;
public static final String MIGRATE_SYSTEM_INDEX_CAUSE = "migrate-system-index";
private static final int UPGRADED_TO_VERSION = NO_UPGRADE_REQUIRED_VERSION.major + 1;
public static final String UPGRADED_INDEX_SUFFIX = "-reindexed-for-" + UPGRADED_TO_VERSION;
private static final Automaton EMPTY = Automata.makeEmpty();
@ -346,6 +358,7 @@ public class SystemIndices {
/**
* Finds a single matching {@link SystemIndexDescriptor}, if any, for the given index name.
* Does not take into account system data streams and their backing indices.
* @param name the name of the index
* @return The matching {@link SystemIndexDescriptor} or {@code null} if no descriptor is found
*/
@ -354,7 +367,7 @@ public class SystemIndices {
}
@Nullable
static SystemIndexDescriptor findMatchingDescriptor(SystemIndexDescriptor[] indexDescriptors, String name) {
private static SystemIndexDescriptor findMatchingDescriptor(SystemIndexDescriptor[] indexDescriptors, String name) {
SystemIndexDescriptor matchingDescriptor = null;
for (SystemIndexDescriptor systemIndexDescriptor : indexDescriptors) {
if (systemIndexDescriptor.matchesIndexPattern(name)) {

View file

@ -219,7 +219,6 @@ import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.threadpool.internal.BuiltInExecutorBuilders;
import org.elasticsearch.transport.Transport;
import org.elasticsearch.transport.TransportService;
import org.elasticsearch.upgrades.SystemIndexMigrationExecutor;
import org.elasticsearch.usage.UsageService;
import org.elasticsearch.watcher.ResourceWatcherService;
import org.elasticsearch.xcontent.NamedXContentRegistry;
@ -577,8 +576,7 @@ class NodeConstruction {
IndicesModule.getNamedWriteables().stream(),
searchModule.getNamedWriteables().stream(),
pluginsService.flatMap(Plugin::getNamedWriteables),
ClusterModule.getNamedWriteables().stream(),
SystemIndexMigrationExecutor.getNamedWriteables().stream()
ClusterModule.getNamedWriteables().stream()
).flatMap(Function.identity()).toList()
);
xContentRegistry = new NamedXContentRegistry(
@ -588,7 +586,6 @@ class NodeConstruction {
searchModule.getNamedXContents().stream(),
pluginsService.flatMap(Plugin::getNamedXContent),
ClusterModule.getNamedXWriteables().stream(),
SystemIndexMigrationExecutor.getNamedXContentParsers().stream(),
HealthNodeTaskExecutor.getNamedXContentParsers().stream()
).flatMap(Function.identity()).toList()
);
@ -1183,17 +1180,7 @@ class NodeConstruction {
final ShutdownPrepareService shutdownPrepareService = new ShutdownPrepareService(settings, httpServerTransport, terminationHandler);
modules.add(
loadPersistentTasksService(
settingsModule,
clusterService,
threadPool,
systemIndices,
clusterModule.getIndexNameExpressionResolver(),
metadataUpdateSettingsService,
metadataCreateIndexService
)
);
modules.add(loadPersistentTasksService(settingsModule, clusterService, threadPool, clusterModule.getIndexNameExpressionResolver()));
modules.add(
loadPluginShutdownService(clusterService),
@ -1696,27 +1683,16 @@ class NodeConstruction {
SettingsModule settingsModule,
ClusterService clusterService,
ThreadPool threadPool,
SystemIndices systemIndices,
IndexNameExpressionResolver indexNameExpressionResolver,
MetadataUpdateSettingsService metadataUpdateSettingsService,
MetadataCreateIndexService metadataCreateIndexService
IndexNameExpressionResolver indexNameExpressionResolver
) {
PersistentTasksService persistentTasksService = new PersistentTasksService(clusterService, threadPool, client);
SystemIndexMigrationExecutor systemIndexMigrationExecutor = new SystemIndexMigrationExecutor(
client,
clusterService,
systemIndices,
metadataUpdateSettingsService,
metadataCreateIndexService,
settingsModule.getIndexScopedSettings()
);
HealthNodeTaskExecutor healthNodeTaskExecutor = HealthNodeTaskExecutor.create(
clusterService,
persistentTasksService,
settingsModule.getSettings(),
clusterService.getClusterSettings()
);
Stream<PersistentTasksExecutor<?>> builtinTaskExecutors = Stream.of(systemIndexMigrationExecutor, healthNodeTaskExecutor);
Stream<PersistentTasksExecutor<?>> builtinTaskExecutors = Stream.of(healthNodeTaskExecutor);
Stream<PersistentTasksExecutor<?>> pluginTaskExecutors = pluginsService.filterPlugins(PersistentTaskPlugin.class)
.map(p -> p.getPersistentTasksExecutor(clusterService, threadPool, client, settingsModule, indexNameExpressionResolver))

View file

@ -48,7 +48,7 @@ import java.util.Map;
* <p>An implementation may also override {@link #prepareForIndicesMigration(ClusterService, Client, ActionListener)} and
* {@link #indicesMigrationComplete(Map, ClusterService, Client, ActionListener)} in order to take special action before and after a
* feature migration, which will temporarily block access to system indices. For example, a plugin might want to enter a safe mode and
* reject certain requests while the migration is in progress. See {@link org.elasticsearch.upgrades.SystemIndexMigrationExecutor} for
* reject certain requests while the migration is in progress. See org.elasticsearch.upgrades.SystemIndexMigrationExecutor for
* more details.
*
* <p>After plugins are loaded, the {@link SystemIndices} class will provide the rest of the system with access to the feature's

View file

@ -276,4 +276,34 @@ public class TransportCreateIndexActionTests extends ESTestCase {
);
assertThat(e.getMessage(), containsString("Could not find project with id [unknown_project_id]"));
}
public void testCreatingSystemIndexForMigration() {
CreateIndexRequest request = new CreateIndexRequest();
String path = "/test"; // just to test that we pass settings
Settings settings = Settings.builder().put(SETTING_INDEX_HIDDEN, true).put(IndexMetadata.SETTING_DATA_PATH, path).build();
request.index(MANAGED_SYSTEM_INDEX_NAME + SystemIndices.UPGRADED_INDEX_SUFFIX)
.cause(SystemIndices.MIGRATE_SYSTEM_INDEX_CAUSE)
.settings(settings);
@SuppressWarnings("unchecked")
ActionListener<CreateIndexResponse> mockListener = mock(ActionListener.class);
action.masterOperation(mock(Task.class), request, CLUSTER_STATE, mockListener);
ArgumentCaptor<CreateIndexClusterStateUpdateRequest> createRequestArgumentCaptor = ArgumentCaptor.forClass(
CreateIndexClusterStateUpdateRequest.class
);
verify(mockListener, times(0)).onFailure(any());
verify(metadataCreateIndexService, times(1)).createIndex(
any(TimeValue.class),
any(TimeValue.class),
any(TimeValue.class),
createRequestArgumentCaptor.capture(),
any()
);
CreateIndexClusterStateUpdateRequest processedRequest = createRequestArgumentCaptor.getValue();
assertTrue(processedRequest.settings().getAsBoolean(SETTING_INDEX_HIDDEN, false));
assertThat(processedRequest.settings().get(IndexMetadata.SETTING_DATA_PATH, ""), is(path));
}
}

View file

@ -13,7 +13,6 @@ import org.elasticsearch.ResourceNotFoundException;
import org.elasticsearch.TransportVersion;
import org.elasticsearch.TransportVersions;
import org.elasticsearch.action.admin.indices.alias.get.GetAliasesRequest;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.cluster.ClusterModule;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.Diff;
@ -31,7 +30,6 @@ import org.elasticsearch.common.io.stream.NamedWriteableRegistry;
import org.elasticsearch.common.io.stream.StreamInput;
import org.elasticsearch.common.io.stream.StreamOutput;
import org.elasticsearch.common.settings.ClusterSettings;
import org.elasticsearch.common.settings.IndexScopedSettings;
import org.elasticsearch.common.settings.Setting;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.util.Maps;
@ -53,7 +51,6 @@ import org.elasticsearch.index.IndexVersions;
import org.elasticsearch.index.alias.RandomAliasActionsGenerator;
import org.elasticsearch.index.mapper.MapperService;
import org.elasticsearch.indices.IndicesModule;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.ingest.IngestMetadata;
import org.elasticsearch.persistent.ClusterPersistentTasksCustomMetadata;
import org.elasticsearch.persistent.PersistentTasks;
@ -68,8 +65,6 @@ import org.elasticsearch.test.TransportVersionUtils;
import org.elasticsearch.test.index.IndexVersionUtils;
import org.elasticsearch.test.rest.ObjectPath;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.upgrades.SystemIndexMigrationExecutor;
import org.elasticsearch.upgrades.SystemIndexMigrationTaskParams;
import org.elasticsearch.xcontent.NamedXContentRegistry;
import org.elasticsearch.xcontent.ToXContent;
import org.elasticsearch.xcontent.XContentBuilder;
@ -770,10 +765,6 @@ public class MetadataTests extends ESTestCase {
{
"id": "health-node",
"task":{ "health-node": {"params":{}} }
},
{
"id": "upgrade-system-indices",
"task":{ "upgrade-system-indices": {"params":{}} }
}
]
},
@ -843,12 +834,6 @@ public class MetadataTests extends ESTestCase {
metadata.getProject().customs().keySet(),
containsInAnyOrder("persistent_tasks", "index-graveyard", "component_template", "repositories")
);
final var projectTasks = PersistentTasksCustomMetadata.get(metadata.getProject());
assertThat(
projectTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getTaskName).toList(),
containsInAnyOrder("upgrade-system-indices")
);
assertThat(clusterTasks.getLastAllocationId(), equalTo(projectTasks.getLastAllocationId()));
assertThat(metadata.customs(), not(hasKey("repositories")));
final var repositoriesMetadata = RepositoriesMetadata.get(metadata.getProject(ProjectId.DEFAULT));
assertThat(
@ -955,7 +940,6 @@ public class MetadataTests extends ESTestCase {
registry.addAll(ClusterModule.getNamedXWriteables());
registry.addAll(IndicesModule.getNamedXContents());
registry.addAll(HealthNodeTaskExecutor.getNamedXContentParsers());
registry.addAll(SystemIndexMigrationExecutor.getNamedXContentParsers());
final var clusterService = mock(ClusterService.class);
when(clusterService.threadPool()).thenReturn(mock(ThreadPool.class));
@ -965,15 +949,7 @@ public class MetadataTests extends ESTestCase {
Settings.EMPTY,
ClusterSettings.createBuiltInClusterSettings()
);
final var systemIndexMigrationExecutor = new SystemIndexMigrationExecutor(
mock(Client.class),
clusterService,
mock(SystemIndices.class),
mock(MetadataUpdateSettingsService.class),
mock(MetadataCreateIndexService.class),
IndexScopedSettings.DEFAULT_SCOPED_SETTINGS
);
new PersistentTasksExecutorRegistry(List.of(healthNodeTaskExecutor, systemIndexMigrationExecutor));
new PersistentTasksExecutorRegistry(List.of(healthNodeTaskExecutor));
XContentParserConfiguration config = XContentParserConfiguration.EMPTY.withRegistry(new NamedXContentRegistry(registry));
try (XContentParser parser = JsonXContent.jsonXContent.createParser(config, json)) {
@ -2709,22 +2685,6 @@ public class MetadataTests extends ESTestCase {
.stream()
.map(
project -> ProjectMetadata.builder(project)
.putCustom(
PersistentTasksCustomMetadata.TYPE,
new PersistentTasksCustomMetadata(
lastAllocationId,
Map.of(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new SystemIndexMigrationTaskParams(),
lastAllocationId,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
)
.putCustom(
RepositoriesMetadata.TYPE,
new RepositoriesMetadata(
@ -2784,22 +2744,9 @@ public class MetadataTests extends ESTestCase {
equalTo(projects.stream().map(pp -> pp.id().id()).collect(Collectors.toUnmodifiableSet()))
);
for (int i = 0; i < projects.size(); i++) {
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks"), notNullValue());
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.last_allocation_id"), equalTo(lastAllocationId));
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.tasks"), hasSize(1));
assertThat(
objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.tasks.0.id"),
equalTo(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)
);
}
Metadata fromXContentMeta = fromJsonXContentStringWithPersistentTasks(bytes.utf8ToString());
assertThat(fromXContentMeta.projects().keySet(), equalTo(originalMeta.projects().keySet()));
for (var project : fromXContentMeta.projects().values()) {
final var projectTasks = PersistentTasksCustomMetadata.get(project);
assertThat(projectTasks.getLastAllocationId(), equalTo(lastAllocationId));
assertThat(projectTasks.taskMap().keySet(), equalTo(Set.of(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)));
assertThat(
RepositoriesMetadata.get(project).repositories(),
equalTo(
@ -2824,25 +2771,6 @@ public class MetadataTests extends ESTestCase {
final var originalMeta = Metadata.builder()
.clusterUUID(randomUUID())
.clusterUUIDCommitted(true)
.put(
ProjectMetadata.builder(ProjectId.DEFAULT)
.putCustom(
PersistentTasksCustomMetadata.TYPE,
new PersistentTasksCustomMetadata(
lastAllocationId,
Map.of(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new SystemIndexMigrationTaskParams(),
lastAllocationId,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
)
)
.putCustom(
ClusterPersistentTasksCustomMetadata.TYPE,
new ClusterPersistentTasksCustomMetadata(
@ -2872,14 +2800,7 @@ public class MetadataTests extends ESTestCase {
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks"), nullValue());
// The combined lastAllocationId is the max between cluster and project tasks
assertThat(objectPath.evaluate("meta-data.persistent_tasks.last_allocation_id"), equalTo(lastAllocationId + 1));
assertThat(objectPath.evaluate("meta-data.persistent_tasks.tasks"), hasSize(2));
assertThat(
Set.of(
objectPath.evaluate("meta-data.persistent_tasks.tasks.0.id"),
objectPath.evaluate("meta-data.persistent_tasks.tasks.1.id")
),
equalTo(Set.of(HealthNode.TASK_NAME, SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME))
);
assertThat(objectPath.evaluate("meta-data.persistent_tasks.tasks"), hasSize(1));
// Deserialize from the XContent should separate cluster and project tasks
final Metadata fromXContentMeta = fromJsonXContentStringWithPersistentTasks(bytes.utf8ToString());
@ -2891,13 +2812,6 @@ public class MetadataTests extends ESTestCase {
clusterTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getId).toList(),
contains(HealthNode.TASK_NAME)
);
final var projectTasks = PersistentTasksCustomMetadata.get(fromXContentMeta.getProject(ProjectId.DEFAULT));
assertThat(projectTasks, notNullValue());
assertThat(projectTasks.getLastAllocationId(), equalTo(lastAllocationId + 1));
assertThat(
projectTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getId).toList(),
contains(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)
);
}
public void testSingleNonDefaultProjectXContent() throws IOException {

View file

@ -18,7 +18,6 @@ import org.elasticsearch.ingest.IngestMetadata;
import org.elasticsearch.persistent.PersistentTasksCustomMetadata;
import org.elasticsearch.test.AbstractChunkedSerializingTestCase;
import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.upgrades.FeatureMigrationResults;
import org.elasticsearch.xcontent.ToXContent;
import org.elasticsearch.xcontent.XContentType;
@ -363,8 +362,6 @@ public class ProjectMetadataTests extends ESTestCase {
params,
4 + dataStreamMetadata.dataStreams().size() + dataStreamMetadata.getDataStreamAliases().size()
);
} else if (custom instanceof FeatureMigrationResults featureMigrationResults) {
chunkCount += checkChunkSize(custom, params, 2 + featureMigrationResults.getFeatureStatuses().size());
} else if (custom instanceof IndexGraveyard indexGraveyard) {
chunkCount += checkChunkSize(custom, params, 2 + indexGraveyard.getTombstones().size());
} else if (custom instanceof IngestMetadata ingestMetadata) {

View file

@ -6,7 +6,9 @@
*/
package org.elasticsearch.xpack.core.security.authz.privilege;
import org.elasticsearch.action.admin.indices.create.TransportCreateIndexAction;
import org.elasticsearch.action.admin.indices.readonly.TransportAddIndexBlockAction;
import org.elasticsearch.action.admin.indices.settings.put.TransportUpdateSettingsAction;
import org.elasticsearch.action.search.TransportSearchShardsAction;
import org.elasticsearch.index.seqno.RetentionLeaseActions;
import org.elasticsearch.index.seqno.RetentionLeaseBackgroundSyncAction;
@ -45,7 +47,9 @@ public final class SystemPrivilege extends Privilege {
"indices:data/read/*", // needed for SystemIndexMigrator
"indices:admin/refresh", // needed for SystemIndexMigrator
"indices:admin/aliases", // needed for SystemIndexMigrator
TransportCreateIndexAction.TYPE.name() + "*", // needed for SystemIndexMigrator
TransportAddIndexBlockAction.TYPE.name() + "*", // needed for SystemIndexMigrator
TransportUpdateSettingsAction.TYPE.name() + "*", // needed for SystemIndexMigrator
TransportSearchShardsAction.TYPE.name(), // added so this API can be called with the system user by other APIs
ActionTypes.RELOAD_REMOTE_CLUSTER_CREDENTIALS_ACTION.name() // needed for Security plugin reload of remote cluster credentials
);

View file

@ -21,6 +21,10 @@ dependencies {
testImplementation project(':modules:data-streams')
testImplementation project(path: ':modules:reindex')
testImplementation project(path: ':modules:ingest-common')
testImplementation project(path: ':modules:lang-painless')
internalClusterTestImplementation project(path: ':modules:lang-painless')
internalClusterTestImplementation project(path: ':modules:lang-painless:spi')
}
addQaCheckDependencies(project)

View file

@ -1,19 +1,16 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.Version;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.ActionRequest;
import org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.indices.create.CreateIndexRequestBuilder;
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
import org.elasticsearch.action.admin.indices.stats.IndexStats;
@ -31,14 +28,18 @@ import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.index.IndexVersion;
import org.elasticsearch.indices.AssociatedIndexDescriptor;
import org.elasticsearch.indices.SystemIndexDescriptor;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.ingest.common.IngestCommonPlugin;
import org.elasticsearch.plugins.ActionPlugin;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.plugins.PluginsService;
import org.elasticsearch.plugins.SystemIndexPlugin;
import org.elasticsearch.reindex.ReindexPlugin;
import org.elasticsearch.test.ESIntegTestCase;
import org.elasticsearch.test.index.IndexVersionUtils;
import org.elasticsearch.xcontent.XContentBuilder;
import org.elasticsearch.xcontent.json.JsonXContent;
import org.elasticsearch.xpack.migrate.MigratePlugin;
import org.junit.Assert;
import org.junit.Before;
@ -78,9 +79,9 @@ public abstract class AbstractFeatureMigrationIntegTest extends ESIntegTestCase
static final int INTERNAL_MANAGED_FLAG_VALUE = 1;
static final String FIELD_NAME = "some_field";
protected static final IndexVersion NEEDS_UPGRADE_INDEX_VERSION = IndexVersionUtils.getPreviousMajorVersion(
TransportGetFeatureUpgradeStatusAction.NO_UPGRADE_REQUIRED_INDEX_VERSION
SystemIndices.NO_UPGRADE_REQUIRED_INDEX_VERSION
);
protected static final int UPGRADED_TO_VERSION = TransportGetFeatureUpgradeStatusAction.NO_UPGRADE_REQUIRED_VERSION.major + 1;
protected static final int UPGRADED_TO_VERSION = SystemIndices.NO_UPGRADE_REQUIRED_VERSION.major + 1;
static final SystemIndexDescriptor EXTERNAL_UNMANAGED = SystemIndexDescriptor.builder()
.setIndexPattern(".ext-unman-*")
@ -154,6 +155,16 @@ public abstract class AbstractFeatureMigrationIntegTest extends ESIntegTestCase
return pluginsService.filterPlugins(type).findFirst().get();
}
@Override
protected Collection<Class<? extends Plugin>> nodePlugins() {
List<Class<? extends Plugin>> plugins = new ArrayList<>(super.nodePlugins());
plugins.add(MigratePlugin.class);
plugins.add(ReindexPlugin.class);
plugins.add(TestPlugin.class);
plugins.add(IngestCommonPlugin.class);
return plugins;
}
protected void createSystemIndexForDescriptor(SystemIndexDescriptor descriptor) {
assertThat(
"the strategy used below to create index names for descriptors without a primary index name only works for simple patterns",

View file

@ -1,21 +1,13 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.migration;
package org.elasticsearch.system_indices.action;
import org.apache.lucene.util.SetOnce;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusRequest;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeRequest;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeResponse;
import org.elasticsearch.action.admin.indices.alias.Alias;
import org.elasticsearch.action.admin.indices.alias.TransportIndicesAliasesAction;
import org.elasticsearch.action.admin.indices.create.CreateIndexRequestBuilder;
@ -39,14 +31,13 @@ import org.elasticsearch.common.compress.CompressedXContent;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.index.query.QueryBuilders;
import org.elasticsearch.indices.SystemIndexDescriptor;
import org.elasticsearch.migration.AbstractFeatureMigrationIntegTest.TestPlugin.BlockingActionFilter;
import org.elasticsearch.painless.PainlessPlugin;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.plugins.SystemIndexPlugin;
import org.elasticsearch.reindex.ReindexPlugin;
import org.elasticsearch.system_indices.action.AbstractFeatureMigrationIntegTest.TestPlugin.BlockingActionFilter;
import org.elasticsearch.system_indices.task.FeatureMigrationResults;
import org.elasticsearch.system_indices.task.SingleFeatureMigrationResult;
import org.elasticsearch.test.InternalTestCluster;
import org.elasticsearch.upgrades.FeatureMigrationResults;
import org.elasticsearch.upgrades.SingleFeatureMigrationResult;
import java.util.ArrayList;
import java.util.Arrays;
@ -105,9 +96,7 @@ public class FeatureMigrationIT extends AbstractFeatureMigrationIntegTest {
@Override
protected Collection<Class<? extends Plugin>> nodePlugins() {
List<Class<? extends Plugin>> plugins = new ArrayList<>(super.nodePlugins());
plugins.add(TestPlugin.class);
plugins.add(SecondTestPlugin.class);
plugins.add(ReindexPlugin.class);
plugins.add(PainlessPlugin.class);
return plugins;
}

View file

@ -1,22 +1,14 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.migration;
package org.elasticsearch.system_indices.action;
import org.apache.lucene.util.SetOnce;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusRequest;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeRequest;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeResponse;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.Metadata;
@ -27,8 +19,7 @@ import org.elasticsearch.index.IndexVersions;
import org.elasticsearch.indices.SystemIndexDescriptor;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.plugins.SystemIndexPlugin;
import org.elasticsearch.reindex.ReindexPlugin;
import org.elasticsearch.upgrades.FeatureMigrationResults;
import org.elasticsearch.system_indices.task.FeatureMigrationResults;
import java.util.ArrayList;
import java.util.Arrays;
@ -71,9 +62,7 @@ public class MultiFeatureMigrationIT extends AbstractFeatureMigrationIntegTest {
@Override
protected Collection<Class<? extends Plugin>> nodePlugins() {
List<Class<? extends Plugin>> plugins = new ArrayList<>(super.nodePlugins());
plugins.add(TestPlugin.class);
plugins.add(SecondPlugin.class);
plugins.add(ReindexPlugin.class);
return plugins;
}

View file

@ -1,37 +1,25 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.migration;
package org.elasticsearch.system_indices.action;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusRequest;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeRequest;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.ClusterStateListener;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.persistent.PersistentTasksCustomMetadata;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.reindex.ReindexPlugin;
import org.elasticsearch.test.InternalTestCluster;
import java.util.ArrayList;
import java.util.Collection;
import java.util.List;
import java.util.concurrent.CyclicBarrier;
import java.util.concurrent.atomic.AtomicBoolean;
import static org.elasticsearch.upgrades.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.hamcrest.Matchers.equalTo;
/**
@ -51,14 +39,6 @@ public class SystemIndexMigrationIT extends AbstractFeatureMigrationIntegTest {
return false;
}
@Override
protected Collection<Class<? extends Plugin>> nodePlugins() {
List<Class<? extends Plugin>> plugins = new ArrayList<>(super.nodePlugins());
plugins.add(TestPlugin.class);
plugins.add(ReindexPlugin.class);
return plugins;
}
public void testSystemIndexMigrationCanBeInterruptedWithShutdown() throws Exception {
CyclicBarrier taskCreated = new CyclicBarrier(2);
CyclicBarrier shutdownCompleted = new CyclicBarrier(2);

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.action.ActionType;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.action.ActionRequestValidationException;
import org.elasticsearch.action.support.master.MasterNodeRequest;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.action.ActionResponse;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.action.ActionType;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.action.ActionRequestValidationException;
import org.elasticsearch.action.support.master.MasterNodeRequest;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.action.ActionResponse;

View file

@ -1,15 +1,12 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.Version;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.support.ActionFilters;
import org.elasticsearch.action.support.master.TransportMasterNodeAction;
@ -18,31 +15,30 @@ import org.elasticsearch.cluster.block.ClusterBlockException;
import org.elasticsearch.cluster.block.ClusterBlockLevel;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.util.concurrent.EsExecutors;
import org.elasticsearch.core.UpdateForV10;
import org.elasticsearch.index.IndexVersion;
import org.elasticsearch.index.IndexVersions;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.injection.guice.Inject;
import org.elasticsearch.persistent.PersistentTasksCustomMetadata;
import org.elasticsearch.system_indices.task.FeatureMigrationResults;
import org.elasticsearch.system_indices.task.SingleFeatureMigrationResult;
import org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams;
import org.elasticsearch.system_indices.task.SystemIndexMigrationTaskState;
import org.elasticsearch.tasks.Task;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.transport.TransportService;
import org.elasticsearch.upgrades.FeatureMigrationResults;
import org.elasticsearch.upgrades.SingleFeatureMigrationResult;
import org.elasticsearch.upgrades.SystemIndexMigrationTaskParams;
import org.elasticsearch.upgrades.SystemIndexMigrationTaskState;
import java.util.Comparator;
import java.util.List;
import java.util.Optional;
import java.util.stream.Stream;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.ERROR;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.IN_PROGRESS;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.NO_MIGRATION_NEEDED;
import static org.elasticsearch.indices.SystemIndices.NO_UPGRADE_REQUIRED_INDEX_VERSION;
import static org.elasticsearch.indices.SystemIndices.UPGRADED_INDEX_SUFFIX;
import static org.elasticsearch.upgrades.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.ERROR;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.IN_PROGRESS;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.NO_MIGRATION_NEEDED;
import static org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
/**
* Transport class for the get feature upgrade status action
@ -51,13 +47,6 @@ public class TransportGetFeatureUpgradeStatusAction extends TransportMasterNodeA
GetFeatureUpgradeStatusRequest,
GetFeatureUpgradeStatusResponse> {
/**
* These versions should be set to current major and current major's index version
*/
@UpdateForV10(owner = UpdateForV10.Owner.CORE_INFRA)
public static final Version NO_UPGRADE_REQUIRED_VERSION = Version.V_9_0_0;
public static final IndexVersion NO_UPGRADE_REQUIRED_INDEX_VERSION = IndexVersions.UPGRADE_TO_LUCENE_10_0_0;
private final SystemIndices systemIndices;
@Inject

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
@ -24,18 +22,18 @@ import org.elasticsearch.core.TimeValue;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.injection.guice.Inject;
import org.elasticsearch.persistent.PersistentTasksService;
import org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams;
import org.elasticsearch.tasks.Task;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.transport.TransportService;
import org.elasticsearch.upgrades.SystemIndexMigrationTaskParams;
import java.util.Comparator;
import java.util.EnumSet;
import java.util.List;
import java.util.Set;
import static org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction.getFeatureUpgradeStatus;
import static org.elasticsearch.upgrades.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.system_indices.action.TransportGetFeatureUpgradeStatusAction.getFeatureUpgradeStatus;
import static org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
/**
* Transport action for post feature upgrade action

View file

@ -1,20 +1,18 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.rest.action.admin.cluster;
package org.elasticsearch.system_indices.rest;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusAction;
import org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusRequest;
import org.elasticsearch.client.internal.node.NodeClient;
import org.elasticsearch.rest.BaseRestHandler;
import org.elasticsearch.rest.RestRequest;
import org.elasticsearch.rest.action.RestToXContentListener;
import org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusAction;
import org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusRequest;
import java.io.IOException;
import java.util.List;

View file

@ -1,20 +1,18 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.rest.action.admin.cluster;
package org.elasticsearch.system_indices.rest;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeAction;
import org.elasticsearch.action.admin.cluster.migration.PostFeatureUpgradeRequest;
import org.elasticsearch.client.internal.node.NodeClient;
import org.elasticsearch.rest.BaseRestHandler;
import org.elasticsearch.rest.RestRequest;
import org.elasticsearch.rest.action.RestToXContentListener;
import org.elasticsearch.system_indices.action.PostFeatureUpgradeAction;
import org.elasticsearch.system_indices.action.PostFeatureUpgradeRequest;
import java.io.IOException;
import java.util.List;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.TransportVersion;
import org.elasticsearch.TransportVersions;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.cluster.SimpleDiffable;

View file

@ -1,18 +1,14 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.MetadataCreateIndexService;
import org.elasticsearch.cluster.metadata.MetadataUpdateSettingsService;
import org.elasticsearch.cluster.node.DiscoveryNode;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.NamedWriteableRegistry;
@ -31,7 +27,7 @@ import java.util.Collection;
import java.util.List;
import java.util.Map;
import static org.elasticsearch.upgrades.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
/**
* Starts the process of migrating system indices. See {@link SystemIndexMigrator} for the actual migration logic.
@ -40,24 +36,18 @@ public class SystemIndexMigrationExecutor extends PersistentTasksExecutor<System
private final Client client; // NOTE: *NOT* an OriginSettingClient. We have to do that later.
private final ClusterService clusterService;
private final SystemIndices systemIndices;
private final MetadataUpdateSettingsService metadataUpdateSettingsService;
private final MetadataCreateIndexService metadataCreateIndexService;
private final IndexScopedSettings indexScopedSettings;
public SystemIndexMigrationExecutor(
Client client,
ClusterService clusterService,
SystemIndices systemIndices,
MetadataUpdateSettingsService metadataUpdateSettingsService,
MetadataCreateIndexService metadataCreateIndexService,
IndexScopedSettings indexScopedSettings
) {
super(SYSTEM_INDEX_UPGRADE_TASK_NAME, clusterService.threadPool().generic());
this.client = client;
this.clusterService = clusterService;
this.systemIndices = systemIndices;
this.metadataUpdateSettingsService = metadataUpdateSettingsService;
this.metadataCreateIndexService = metadataCreateIndexService;
this.indexScopedSettings = indexScopedSettings;
}
@ -77,20 +67,7 @@ public class SystemIndexMigrationExecutor extends PersistentTasksExecutor<System
PersistentTasksCustomMetadata.PersistentTask<SystemIndexMigrationTaskParams> taskInProgress,
Map<String, String> headers
) {
return new SystemIndexMigrator(
client,
id,
type,
action,
parentTaskId,
taskInProgress.getParams(),
headers,
clusterService,
systemIndices,
metadataUpdateSettingsService,
metadataCreateIndexService,
indexScopedSettings
);
return new SystemIndexMigrator(client, id, type, action, parentTaskId, headers, clusterService, systemIndices, indexScopedSettings);
}
@Override

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.TransportVersion;
import org.elasticsearch.common.io.stream.StreamInput;
@ -19,7 +17,7 @@ import org.elasticsearch.xcontent.XContentParser;
import java.io.IOException;
import static org.elasticsearch.upgrades.FeatureMigrationResults.MIGRATION_ADDED_VERSION;
import static org.elasticsearch.system_indices.task.FeatureMigrationResults.MIGRATION_ADDED_VERSION;
/**
* The params used to initialize {@link SystemIndexMigrator} when it's initially kicked off.

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.client.internal.Client;
@ -25,7 +23,7 @@ import java.util.HashMap;
import java.util.Map;
import java.util.Objects;
import static org.elasticsearch.upgrades.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.system_indices.task.SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME;
import static org.elasticsearch.xcontent.ConstructingObjectParser.constructorArg;
/**

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
@ -16,13 +14,14 @@ import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequest;
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesRequestBuilder;
import org.elasticsearch.action.admin.indices.alias.IndicesAliasesResponse;
import org.elasticsearch.action.admin.indices.create.CreateIndexClusterStateUpdateRequest;
import org.elasticsearch.action.admin.indices.create.CreateIndexRequest;
import org.elasticsearch.action.admin.indices.create.CreateIndexResponse;
import org.elasticsearch.action.admin.indices.readonly.AddIndexBlockRequest;
import org.elasticsearch.action.admin.indices.settings.put.UpdateSettingsClusterStateUpdateRequest;
import org.elasticsearch.action.admin.indices.settings.put.TransportUpdateSettingsAction;
import org.elasticsearch.action.admin.indices.settings.put.UpdateSettingsRequest;
import org.elasticsearch.action.support.ActiveShardCount;
import org.elasticsearch.action.support.master.AcknowledgedResponse;
import org.elasticsearch.action.support.master.MasterNodeRequest;
import org.elasticsearch.action.support.master.ShardsAcknowledgedResponse;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.client.internal.ParentTaskAssigningClient;
import org.elasticsearch.cluster.ClusterState;
@ -30,9 +29,7 @@ import org.elasticsearch.cluster.ClusterStateUpdateTask;
import org.elasticsearch.cluster.metadata.IndexMetadata;
import org.elasticsearch.cluster.metadata.IndexTemplateMetadata;
import org.elasticsearch.cluster.metadata.Metadata;
import org.elasticsearch.cluster.metadata.MetadataCreateIndexService;
import org.elasticsearch.cluster.metadata.MetadataIndexTemplateService;
import org.elasticsearch.cluster.metadata.MetadataUpdateSettingsService;
import org.elasticsearch.cluster.metadata.ProjectMetadata;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.Strings;
@ -61,10 +58,10 @@ import java.util.concurrent.atomic.AtomicReference;
import java.util.function.Consumer;
import java.util.stream.Collectors;
import static org.elasticsearch.action.admin.cluster.migration.TransportGetFeatureUpgradeStatusAction.NO_UPGRADE_REQUIRED_INDEX_VERSION;
import static org.elasticsearch.cluster.metadata.IndexMetadata.APIBlock.WRITE;
import static org.elasticsearch.cluster.metadata.IndexMetadata.State.CLOSE;
import static org.elasticsearch.core.Strings.format;
import static org.elasticsearch.indices.SystemIndices.NO_UPGRADE_REQUIRED_INDEX_VERSION;
/**
* This is where the logic to actually perform the migration lives - {@link SystemIndexMigrator#run(SystemIndexMigrationTaskState)} will
@ -79,8 +76,6 @@ public class SystemIndexMigrator extends AllocatedPersistentTask {
private final ParentTaskAssigningClient baseClient;
private final ClusterService clusterService;
private final SystemIndices systemIndices;
private final MetadataUpdateSettingsService metadataUpdateSettingsService;
private final MetadataCreateIndexService metadataCreateIndexService;
private final IndexScopedSettings indexScopedSettings;
// In-memory state
@ -96,20 +91,15 @@ public class SystemIndexMigrator extends AllocatedPersistentTask {
String type,
String action,
TaskId parentTask,
SystemIndexMigrationTaskParams params,
Map<String, String> headers,
ClusterService clusterService,
SystemIndices systemIndices,
MetadataUpdateSettingsService metadataUpdateSettingsService,
MetadataCreateIndexService metadataCreateIndexService,
IndexScopedSettings indexScopedSettings
) {
super(id, type, action, "system-index-migrator", parentTask, headers);
this.baseClient = new ParentTaskAssigningClient(client, parentTask);
this.clusterService = clusterService;
this.systemIndices = systemIndices;
this.metadataUpdateSettingsService = metadataUpdateSettingsService;
this.metadataCreateIndexService = metadataCreateIndexService;
this.indexScopedSettings = indexScopedSettings;
}
@ -471,15 +461,10 @@ public class SystemIndexMigrator extends AllocatedPersistentTask {
}
}
private void createIndex(SystemIndexMigrationInfo migrationInfo, ActionListener<ShardsAcknowledgedResponse> listener) {
private void createIndex(SystemIndexMigrationInfo migrationInfo, ActionListener<CreateIndexResponse> listener) {
logger.info("creating new system index [{}] from feature [{}]", migrationInfo.getNextIndexName(), migrationInfo.getFeatureName());
final CreateIndexClusterStateUpdateRequest createRequest = new CreateIndexClusterStateUpdateRequest(
"migrate-system-index",
migrationInfo.getNextIndexName(),
migrationInfo.getNextIndexName()
);
CreateIndexRequest createIndexRequest = new CreateIndexRequest(migrationInfo.getNextIndexName());
Settings.Builder settingsBuilder = Settings.builder();
if (Objects.nonNull(migrationInfo.getSettings())) {
settingsBuilder.put(migrationInfo.getSettings());
@ -487,28 +472,32 @@ public class SystemIndexMigrator extends AllocatedPersistentTask {
settingsBuilder.remove("index.blocks.read");
settingsBuilder.remove("index.blocks.metadata");
}
createRequest.waitForActiveShards(ActiveShardCount.ALL)
.mappings(migrationInfo.getMappings())
createIndexRequest.cause(SystemIndices.MIGRATE_SYSTEM_INDEX_CAUSE)
.ackTimeout(TimeValue.ZERO)
.masterNodeTimeout(MasterNodeRequest.INFINITE_MASTER_NODE_TIMEOUT)
.waitForActiveShards(ActiveShardCount.ALL)
.mapping(migrationInfo.getMappings())
.settings(Objects.requireNonNullElse(settingsBuilder.build(), Settings.EMPTY));
metadataCreateIndexService.createIndex(
MasterNodeRequest.INFINITE_MASTER_NODE_TIMEOUT,
TimeValue.ZERO,
null,
createRequest,
listener
);
baseClient.admin().indices().create(createIndexRequest, listener);
}
private void createIndexRetryOnFailure(SystemIndexMigrationInfo migrationInfo, ActionListener<ShardsAcknowledgedResponse> listener) {
private void createIndexRetryOnFailure(SystemIndexMigrationInfo migrationInfo, ActionListener<CreateIndexResponse> listener) {
createIndex(migrationInfo, listener.delegateResponse((l, e) -> {
logger.warn("createIndex failed, retrying after removing index [{}] from previous attempt", migrationInfo.getNextIndexName());
logger.warn(
"createIndex failed with \"{}\", retrying after removing index [{}] from previous attempt",
e.getMessage(),
migrationInfo.getNextIndexName()
);
deleteIndex(migrationInfo, ActionListener.wrap(cleanupResponse -> createIndex(migrationInfo, l.delegateResponse((l3, e3) -> {
e3.addSuppressed(e);
logger.error(
"createIndex failed after retrying, aborting system index migration. index: " + migrationInfo.getNextIndexName(),
e3
);
l.onFailure(e3);
})), e2 -> {
e2.addSuppressed(e);
logger.error("deleteIndex failed, aborting system index migration. index: " + migrationInfo.getNextIndexName(), e2);
l.onFailure(e2);
}));
@ -575,18 +564,10 @@ public class SystemIndexMigrator extends AllocatedPersistentTask {
} else {
// The only way to remove a Block is via a settings update.
final Settings readOnlySettings = Settings.builder().put(IndexMetadata.INDEX_BLOCKS_WRITE_SETTING.getKey(), false).build();
metadataUpdateSettingsService.updateSettings(
new UpdateSettingsClusterStateUpdateRequest(
Metadata.DEFAULT_PROJECT_ID,
MasterNodeRequest.INFINITE_MASTER_NODE_TIMEOUT,
TimeValue.ZERO,
readOnlySettings,
UpdateSettingsClusterStateUpdateRequest.OnExisting.OVERWRITE,
UpdateSettingsClusterStateUpdateRequest.OnStaticSetting.REJECT,
index
),
listener
);
UpdateSettingsRequest updateSettingsRequest = new UpdateSettingsRequest(readOnlySettings, index.getName()).setPreserveExisting(
false
).masterNodeTimeout(MasterNodeRequest.INFINITE_MASTER_NODE_TIMEOUT).ackTimeout(TimeValue.ZERO);
baseClient.execute(TransportUpdateSettingsAction.TYPE, updateSettingsRequest, listener);
}
}

View file

@ -7,11 +7,14 @@
package org.elasticsearch.xpack.migrate;
import org.apache.lucene.util.SetOnce;
import org.elasticsearch.action.ActionRequest;
import org.elasticsearch.action.ActionResponse;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.client.internal.OriginSettingClient;
import org.elasticsearch.cluster.NamedDiff;
import org.elasticsearch.cluster.metadata.IndexNameExpressionResolver;
import org.elasticsearch.cluster.metadata.Metadata;
import org.elasticsearch.cluster.node.DiscoveryNodes;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.io.stream.NamedWriteableRegistry;
@ -22,6 +25,7 @@ import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.settings.SettingsFilter;
import org.elasticsearch.common.settings.SettingsModule;
import org.elasticsearch.features.NodeFeature;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.persistent.PersistentTaskParams;
import org.elasticsearch.persistent.PersistentTaskState;
import org.elasticsearch.persistent.PersistentTasksExecutor;
@ -30,6 +34,14 @@ import org.elasticsearch.plugins.PersistentTaskPlugin;
import org.elasticsearch.plugins.Plugin;
import org.elasticsearch.rest.RestController;
import org.elasticsearch.rest.RestHandler;
import org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusAction;
import org.elasticsearch.system_indices.action.PostFeatureUpgradeAction;
import org.elasticsearch.system_indices.action.TransportGetFeatureUpgradeStatusAction;
import org.elasticsearch.system_indices.action.TransportPostFeatureUpgradeAction;
import org.elasticsearch.system_indices.rest.RestGetFeatureUpgradeStatusAction;
import org.elasticsearch.system_indices.rest.RestPostFeatureUpgradeAction;
import org.elasticsearch.system_indices.task.FeatureMigrationResults;
import org.elasticsearch.system_indices.task.SystemIndexMigrationExecutor;
import org.elasticsearch.tasks.Task;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.xcontent.NamedXContentRegistry;
@ -61,14 +73,19 @@ import java.util.Collection;
import java.util.List;
import java.util.function.Predicate;
import java.util.function.Supplier;
import java.util.stream.Stream;
import static org.elasticsearch.xpack.core.ClientHelper.REINDEX_DATA_STREAM_ORIGIN;
import static org.elasticsearch.xpack.migrate.action.ReindexDataStreamIndexTransportAction.REINDEX_MAX_REQUESTS_PER_SECOND_SETTING;
import static org.elasticsearch.xpack.migrate.task.ReindexDataStreamPersistentTaskExecutor.MAX_CONCURRENT_INDICES_REINDEXED_PER_DATA_STREAM_SETTING;
public class MigratePlugin extends Plugin implements ActionPlugin, PersistentTaskPlugin {
private final SetOnce<SystemIndices> systemIndices = new SetOnce<>();
@Override
public Collection<?> createComponents(PluginServices services) {
systemIndices.set(services.systemIndices());
var registry = new MigrateTemplateRegistry(
services.environment().settings(),
services.clusterService(),
@ -97,6 +114,9 @@ public class MigratePlugin extends Plugin implements ActionPlugin, PersistentTas
handlers.add(new RestGetMigrationReindexStatusAction());
handlers.add(new RestCancelReindexDataStreamAction());
handlers.add(new RestCreateIndexFromSourceAction());
handlers.add(new RestGetFeatureUpgradeStatusAction());
handlers.add(new RestPostFeatureUpgradeAction());
return handlers;
}
@ -109,40 +129,51 @@ public class MigratePlugin extends Plugin implements ActionPlugin, PersistentTas
actions.add(new ActionHandler<>(ReindexDataStreamIndexAction.INSTANCE, ReindexDataStreamIndexTransportAction.class));
actions.add(new ActionHandler<>(CreateIndexFromSourceAction.INSTANCE, CreateIndexFromSourceTransportAction.class));
actions.add(new ActionHandler<>(CopyLifecycleIndexMetadataAction.INSTANCE, CopyLifecycleIndexMetadataTransportAction.class));
actions.add(new ActionHandler<>(GetFeatureUpgradeStatusAction.INSTANCE, TransportGetFeatureUpgradeStatusAction.class));
actions.add(new ActionHandler<>(PostFeatureUpgradeAction.INSTANCE, TransportPostFeatureUpgradeAction.class));
return actions;
}
@Override
public List<NamedXContentRegistry.Entry> getNamedXContent() {
return List.of(
new NamedXContentRegistry.Entry(
PersistentTaskState.class,
new ParseField(ReindexDataStreamPersistentTaskState.NAME),
ReindexDataStreamPersistentTaskState::fromXContent
),
new NamedXContentRegistry.Entry(
PersistentTaskParams.class,
new ParseField(ReindexDataStreamTaskParams.NAME),
ReindexDataStreamTaskParams::fromXContent
return Stream.concat(
SystemIndexMigrationExecutor.getNamedXContentParsers().stream(),
Stream.of(
new NamedXContentRegistry.Entry(
PersistentTaskState.class,
new ParseField(ReindexDataStreamPersistentTaskState.NAME),
ReindexDataStreamPersistentTaskState::fromXContent
),
new NamedXContentRegistry.Entry(
PersistentTaskParams.class,
new ParseField(ReindexDataStreamTaskParams.NAME),
ReindexDataStreamTaskParams::fromXContent
)
)
);
).toList();
}
@Override
public List<NamedWriteableRegistry.Entry> getNamedWriteables() {
return List.of(
new NamedWriteableRegistry.Entry(
PersistentTaskState.class,
ReindexDataStreamPersistentTaskState.NAME,
ReindexDataStreamPersistentTaskState::new
),
new NamedWriteableRegistry.Entry(
PersistentTaskParams.class,
ReindexDataStreamTaskParams.NAME,
ReindexDataStreamTaskParams::new
),
new NamedWriteableRegistry.Entry(Task.Status.class, ReindexDataStreamStatus.NAME, ReindexDataStreamStatus::new)
);
return Stream.concat(
SystemIndexMigrationExecutor.getNamedWriteables().stream(),
Stream.of(
new NamedWriteableRegistry.Entry(Metadata.ProjectCustom.class, FeatureMigrationResults.TYPE, FeatureMigrationResults::new),
new NamedWriteableRegistry.Entry(NamedDiff.class, FeatureMigrationResults.TYPE, FeatureMigrationResults::readDiffFrom),
new NamedWriteableRegistry.Entry(
PersistentTaskState.class,
ReindexDataStreamPersistentTaskState.NAME,
ReindexDataStreamPersistentTaskState::new
),
new NamedWriteableRegistry.Entry(
PersistentTaskParams.class,
ReindexDataStreamTaskParams.NAME,
ReindexDataStreamTaskParams::new
),
new NamedWriteableRegistry.Entry(Task.Status.class, ReindexDataStreamStatus.NAME, ReindexDataStreamStatus::new)
)
).toList();
}
@Override
@ -154,6 +185,7 @@ public class MigratePlugin extends Plugin implements ActionPlugin, PersistentTas
IndexNameExpressionResolver expressionResolver
) {
return List.of(
new SystemIndexMigrationExecutor(client, clusterService, systemIndices.get(), settingsModule.getIndexScopedSettings()),
new ReindexDataStreamPersistentTaskExecutor(
new OriginSettingClient(client, REINDEX_DATA_STREAM_ORIGIN),
clusterService,

View file

@ -0,0 +1,13 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
*/
grant {
// needed for Painless to generate runtime classes
permission java.lang.RuntimePermission "createClassLoader";
};

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.common.io.stream.Writeable;
import org.elasticsearch.index.IndexVersion;
@ -16,10 +14,10 @@ import org.elasticsearch.test.AbstractWireSerializingTestCase;
import java.util.Collections;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.ERROR;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.IN_PROGRESS;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.NO_MIGRATION_NEEDED;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.ERROR;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.IN_PROGRESS;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.NO_MIGRATION_NEEDED;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.notNullValue;
@ -37,7 +35,7 @@ public class GetFeatureUpgradeStatusResponseTests extends AbstractWireSerializin
protected GetFeatureUpgradeStatusResponse createTestInstance() {
return new GetFeatureUpgradeStatusResponse(
randomList(8, GetFeatureUpgradeStatusResponseTests::createFeatureStatus),
randomFrom(org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.values())
randomFrom(GetFeatureUpgradeStatusResponse.UpgradeStatus.values())
);
}
@ -51,10 +49,7 @@ public class GetFeatureUpgradeStatusResponseTests extends AbstractWireSerializin
GetFeatureUpgradeStatusResponseTests::createFeatureStatus
)
),
randomValueOtherThan(
instance.getUpgradeStatus(),
() -> randomFrom(org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.values())
)
randomValueOtherThan(instance.getUpgradeStatus(), () -> randomFrom(GetFeatureUpgradeStatusResponse.UpgradeStatus.values()))
);
}
@ -92,7 +87,7 @@ public class GetFeatureUpgradeStatusResponseTests extends AbstractWireSerializin
return new GetFeatureUpgradeStatusResponse.FeatureUpgradeStatus(
randomAlphaOfLengthBetween(3, 20),
randomFrom(IndexVersion.current(), IndexVersions.MINIMUM_COMPATIBLE),
randomFrom(org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.values()),
randomFrom(GetFeatureUpgradeStatusResponse.UpgradeStatus.values()),
randomList(4, GetFeatureUpgradeStatusResponseTests::getIndexInfo)
);
}

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.common.io.stream.Writeable;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.action.admin.cluster.migration;
package org.elasticsearch.system_indices.action;
import org.elasticsearch.cluster.ClusterState;
import org.elasticsearch.cluster.metadata.IndexMetadata;
@ -24,7 +22,7 @@ import java.util.ArrayList;
import java.util.List;
import java.util.Map;
import static org.elasticsearch.action.admin.cluster.migration.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.elasticsearch.system_indices.action.GetFeatureUpgradeStatusResponse.UpgradeStatus.MIGRATION_NEEDED;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.hasSize;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.ElasticsearchException;
import org.elasticsearch.cluster.Diff;

View file

@ -0,0 +1,317 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.system_indices.task;
import org.elasticsearch.client.internal.Client;
import org.elasticsearch.cluster.ClusterModule;
import org.elasticsearch.cluster.metadata.Metadata;
import org.elasticsearch.cluster.metadata.ProjectId;
import org.elasticsearch.cluster.metadata.ProjectMetadata;
import org.elasticsearch.cluster.service.ClusterService;
import org.elasticsearch.common.bytes.BytesReference;
import org.elasticsearch.common.settings.ClusterSettings;
import org.elasticsearch.common.settings.IndexScopedSettings;
import org.elasticsearch.common.settings.Settings;
import org.elasticsearch.common.xcontent.ChunkedToXContent;
import org.elasticsearch.health.node.selection.HealthNode;
import org.elasticsearch.health.node.selection.HealthNodeTaskExecutor;
import org.elasticsearch.health.node.selection.HealthNodeTaskParams;
import org.elasticsearch.index.IndexVersion;
import org.elasticsearch.indices.IndicesModule;
import org.elasticsearch.indices.SystemIndices;
import org.elasticsearch.persistent.ClusterPersistentTasksCustomMetadata;
import org.elasticsearch.persistent.PersistentTasks;
import org.elasticsearch.persistent.PersistentTasksCustomMetadata;
import org.elasticsearch.persistent.PersistentTasksExecutorRegistry;
import org.elasticsearch.persistent.PersistentTasksService;
import org.elasticsearch.test.ESTestCase;
import org.elasticsearch.test.rest.ObjectPath;
import org.elasticsearch.threadpool.ThreadPool;
import org.elasticsearch.xcontent.NamedXContentRegistry;
import org.elasticsearch.xcontent.ToXContent;
import org.elasticsearch.xcontent.XContentBuilder;
import org.elasticsearch.xcontent.XContentParser;
import org.elasticsearch.xcontent.XContentParserConfiguration;
import org.elasticsearch.xcontent.json.JsonXContent;
import java.io.IOException;
import java.io.UncheckedIOException;
import java.util.ArrayList;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.stream.Collectors;
import java.util.stream.IntStream;
import static org.elasticsearch.cluster.metadata.Metadata.CONTEXT_MODE_SNAPSHOT;
import static org.hamcrest.Matchers.contains;
import static org.hamcrest.Matchers.containsInAnyOrder;
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.hasSize;
import static org.hamcrest.Matchers.is;
import static org.hamcrest.Matchers.notNullValue;
import static org.hamcrest.Matchers.nullValue;
import static org.mockito.Mockito.mock;
import static org.mockito.Mockito.when;
public class SystemIndexMigrationMetadataTests extends ESTestCase {
public void testParseXContentFormatBeforeMultiProject() throws IOException {
final String json = org.elasticsearch.core.Strings.format("""
{
"meta-data": {
"version": 54321,
"cluster_uuid":"aba1aa1ababbbaabaabaab",
"cluster_uuid_committed":false,
"persistent_tasks": {
"last_allocation_id": 1,
"tasks": [
{
"id": "health-node",
"task":{ "health-node": {"params":{}} }
},
{
"id": "upgrade-system-indices",
"task":{ "upgrade-system-indices": {"params":{}} }
}
]
},
"reserved_state":{ }
}
}
""", IndexVersion.current(), IndexVersion.current());
final var metadata = fromJsonXContentStringWithPersistentTasks(json);
assertThat(metadata, notNullValue());
assertThat(metadata.clusterUUID(), is("aba1aa1ababbbaabaabaab"));
assertThat(metadata.customs().keySet(), contains("cluster_persistent_tasks"));
final var clusterTasks = ClusterPersistentTasksCustomMetadata.get(metadata);
assertThat(clusterTasks.tasks(), hasSize(1));
assertThat(
clusterTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getTaskName).toList(),
containsInAnyOrder("health-node")
);
assertThat(metadata.getProject().customs().keySet(), containsInAnyOrder("persistent_tasks", "index-graveyard"));
final var projectTasks = PersistentTasksCustomMetadata.get(metadata.getProject());
assertThat(
projectTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getTaskName).toList(),
containsInAnyOrder("upgrade-system-indices")
);
assertThat(clusterTasks.getLastAllocationId(), equalTo(projectTasks.getLastAllocationId()));
}
private Metadata fromJsonXContentStringWithPersistentTasks(String json) throws IOException {
List<NamedXContentRegistry.Entry> registry = new ArrayList<>();
registry.addAll(ClusterModule.getNamedXWriteables());
registry.addAll(IndicesModule.getNamedXContents());
registry.addAll(HealthNodeTaskExecutor.getNamedXContentParsers());
registry.addAll(SystemIndexMigrationExecutor.getNamedXContentParsers());
final var clusterService = mock(ClusterService.class);
when(clusterService.threadPool()).thenReturn(mock(ThreadPool.class));
final var healthNodeTaskExecutor = HealthNodeTaskExecutor.create(
clusterService,
mock(PersistentTasksService.class),
Settings.EMPTY,
ClusterSettings.createBuiltInClusterSettings()
);
final var systemIndexMigrationExecutor = new SystemIndexMigrationExecutor(
mock(Client.class),
clusterService,
mock(SystemIndices.class),
IndexScopedSettings.DEFAULT_SCOPED_SETTINGS
);
new PersistentTasksExecutorRegistry(List.of(healthNodeTaskExecutor, systemIndexMigrationExecutor));
XContentParserConfiguration config = XContentParserConfiguration.EMPTY.withRegistry(new NamedXContentRegistry(registry));
try (XContentParser parser = JsonXContent.jsonXContent.createParser(config, json)) {
return Metadata.fromXContent(parser);
}
}
public void testMultiProjectXContent() throws IOException {
final long lastAllocationId = randomNonNegativeLong();
final List<ProjectMetadata> projects = randomList(
1,
5,
() -> ProjectMetadata.builder(randomUniqueProjectId())
.putCustom(
PersistentTasksCustomMetadata.TYPE,
new PersistentTasksCustomMetadata(
lastAllocationId,
Map.of(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new SystemIndexMigrationTaskParams(),
lastAllocationId,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
)
.build()
);
Metadata.Builder metadataBuilder = Metadata.builder()
.putCustom(
ClusterPersistentTasksCustomMetadata.TYPE,
new ClusterPersistentTasksCustomMetadata(
lastAllocationId + 1,
Map.of(
HealthNode.TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
HealthNode.TASK_NAME,
HealthNode.TASK_NAME,
HealthNodeTaskParams.INSTANCE,
lastAllocationId + 1,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
);
for (ProjectMetadata project : projects) {
metadataBuilder.put(project);
}
final Metadata originalMeta = metadataBuilder.build();
ToXContent.Params p = new ToXContent.MapParams(
Map.of("multi-project", "true", Metadata.CONTEXT_MODE_PARAM, Metadata.CONTEXT_MODE_GATEWAY)
);
final BytesReference bytes = toXContentBytes(originalMeta, p);
// XContent with multi-project=true has separate cluster and project tasks
final var objectPath = ObjectPath.createFromXContent(JsonXContent.jsonXContent, bytes);
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks"), notNullValue());
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks.last_allocation_id"), equalTo(lastAllocationId + 1));
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks.tasks"), hasSize(1));
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks.tasks.0.id"), equalTo(HealthNode.TASK_NAME));
assertThat(objectPath.evaluate("meta-data.projects"), hasSize(projects.size()));
assertThat(IntStream.range(0, projects.size()).mapToObj(i -> {
try {
return (String) objectPath.evaluate("meta-data.projects." + i + ".id");
} catch (IOException e) {
throw new UncheckedIOException(e);
}
}).collect(Collectors.toUnmodifiableSet()),
equalTo(projects.stream().map(pp -> pp.id().id()).collect(Collectors.toUnmodifiableSet()))
);
for (int i = 0; i < projects.size(); i++) {
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks"), notNullValue());
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.last_allocation_id"), equalTo(lastAllocationId));
assertThat(objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.tasks"), hasSize(1));
assertThat(
objectPath.evaluate("meta-data.projects." + i + ".persistent_tasks.tasks.0.id"),
equalTo(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)
);
}
Metadata fromXContentMeta = fromJsonXContentStringWithPersistentTasks(bytes.utf8ToString());
assertThat(fromXContentMeta.projects().keySet(), equalTo(originalMeta.projects().keySet()));
for (var project : fromXContentMeta.projects().values()) {
final var projectTasks = PersistentTasksCustomMetadata.get(project);
assertThat(projectTasks.getLastAllocationId(), equalTo(lastAllocationId));
assertThat(projectTasks.taskMap().keySet(), equalTo(Set.of(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)));
}
final var clusterTasks = ClusterPersistentTasksCustomMetadata.get(fromXContentMeta);
assertThat(clusterTasks.getLastAllocationId(), equalTo(lastAllocationId + 1));
assertThat(clusterTasks.taskMap().keySet(), equalTo(Set.of(HealthNode.TASK_NAME)));
}
public void testDefaultProjectXContentWithPersistentTasks() throws IOException {
final long lastAllocationId = randomNonNegativeLong();
final var originalMeta = Metadata.builder()
.clusterUUID(randomUUID())
.clusterUUIDCommitted(true)
.put(
ProjectMetadata.builder(ProjectId.DEFAULT)
.putCustom(
PersistentTasksCustomMetadata.TYPE,
new PersistentTasksCustomMetadata(
lastAllocationId,
Map.of(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME,
new SystemIndexMigrationTaskParams(),
lastAllocationId,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
)
)
.putCustom(
ClusterPersistentTasksCustomMetadata.TYPE,
new ClusterPersistentTasksCustomMetadata(
lastAllocationId + 1,
Map.of(
HealthNode.TASK_NAME,
new PersistentTasksCustomMetadata.PersistentTask<>(
HealthNode.TASK_NAME,
HealthNode.TASK_NAME,
HealthNodeTaskParams.INSTANCE,
lastAllocationId + 1,
PersistentTasks.INITIAL_ASSIGNMENT
)
)
)
)
.build();
// For single project metadata, XContent output should combine the cluster and project tasks
final ToXContent.Params p = new ToXContent.MapParams(
Map.ofEntries(Map.entry("multi-project", "false"), Map.entry(Metadata.CONTEXT_MODE_PARAM, CONTEXT_MODE_SNAPSHOT))
);
final BytesReference bytes = toXContentBytes(originalMeta, p);
final var objectPath = ObjectPath.createFromXContent(JsonXContent.jsonXContent, bytes);
// No cluster_persistent_tasks for single project output, it is combined with persistent_tasks
assertThat(objectPath.evaluate("meta-data.cluster_persistent_tasks"), nullValue());
// The combined lastAllocationId is the max between cluster and project tasks
assertThat(objectPath.evaluate("meta-data.persistent_tasks.last_allocation_id"), equalTo(lastAllocationId + 1));
assertThat(objectPath.evaluate("meta-data.persistent_tasks.tasks"), hasSize(2));
assertThat(
Set.of(
objectPath.evaluate("meta-data.persistent_tasks.tasks.0.id"),
objectPath.evaluate("meta-data.persistent_tasks.tasks.1.id")
),
equalTo(Set.of(HealthNode.TASK_NAME, SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME))
);
// Deserialize from the XContent should separate cluster and project tasks
final Metadata fromXContentMeta = fromJsonXContentStringWithPersistentTasks(bytes.utf8ToString());
assertThat(fromXContentMeta.projects().keySet(), equalTo(Set.of(ProjectId.DEFAULT)));
final var clusterTasks = ClusterPersistentTasksCustomMetadata.get(fromXContentMeta);
assertThat(clusterTasks, notNullValue());
assertThat(clusterTasks.getLastAllocationId(), equalTo(lastAllocationId + 1));
assertThat(
clusterTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getId).toList(),
contains(HealthNode.TASK_NAME)
);
final var projectTasks = PersistentTasksCustomMetadata.get(fromXContentMeta.getProject(ProjectId.DEFAULT));
assertThat(projectTasks, notNullValue());
assertThat(projectTasks.getLastAllocationId(), equalTo(lastAllocationId + 1));
assertThat(
projectTasks.tasks().stream().map(PersistentTasksCustomMetadata.PersistentTask::getId).toList(),
contains(SystemIndexMigrationTaskParams.SYSTEM_INDEX_UPGRADE_TASK_NAME)
);
}
private static BytesReference toXContentBytes(Metadata metadata, ToXContent.Params params) throws IOException {
XContentBuilder builder = JsonXContent.contentBuilder();
builder.startObject();
ChunkedToXContent.wrapAsToXContent(metadata).toXContent(builder, params);
builder.endObject();
return BytesReference.bytes(builder);
}
}

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.TransportVersion;
import org.elasticsearch.common.io.stream.NamedWriteableRegistry;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.test.AbstractXContentTestCase;
import org.elasticsearch.xcontent.NamedXContentRegistry;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.TransportVersion;
import org.elasticsearch.common.io.stream.NamedWriteableRegistry;

View file

@ -1,13 +1,11 @@
/*
* Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
* or more contributor license agreements. Licensed under the "Elastic License
* 2.0", the "GNU Affero General Public License v3.0 only", and the "Server Side
* Public License v 1"; you may not use this file except in compliance with, at
* your election, the "Elastic License 2.0", the "GNU Affero General Public
* License v3.0 only", or the "Server Side Public License, v 1".
* or more contributor license agreements. Licensed under the Elastic License
* 2.0; you may not use this file except in compliance with the Elastic License
* 2.0.
*/
package org.elasticsearch.upgrades;
package org.elasticsearch.system_indices.task;
import org.elasticsearch.test.AbstractXContentTestCase;
import org.elasticsearch.xcontent.NamedXContentRegistry;

View file

@ -339,7 +339,7 @@ public class SearchableSnapshotsBlobStoreCacheMaintenanceIntegTests extends Base
/**
* Mimics migration of the {@link SearchableSnapshots#SNAPSHOT_BLOB_CACHE_INDEX} as done in
* {@link org.elasticsearch.upgrades.SystemIndexMigrator}, where the index is re-indexed, and replaced by an alias.
* org.elasticsearch.upgrades.SystemIndexMigrator, where the index is re-indexed, and replaced by an alias.
*/
private void migrateTheSystemIndex() {
final var migratedSnapshotBlobCache = SNAPSHOT_BLOB_CACHE_INDEX + SystemIndices.UPGRADED_INDEX_SUFFIX;

View file

@ -1,11 +1,10 @@
"Get feature upgrade status":
"Start feature upgrade":
- requires:
cluster_features: ["gte_v7.16.0"]
reason: "Endpoint added in 7.16.0"
- do:
migration.get_feature_upgrade_status: {}
migration.post_feature_upgrade: {}
- is_false: accepted
- is_true: features

View file

@ -58,6 +58,8 @@ tasks.named("yamlRestTest").configure {
'^migrate/10_reindex/*',
'^migrate/20_reindex_status/*',
'^migrate/30_create_from/*',
'^migration/10_get_feature_upgrade_status/*',
'^migration/20_post_feature_upgrade/*',
'^ml/3rd_party_deployment/*',
'^ml/bucket_correlation_agg/*',
'^ml/bucket_count_ks_test_agg/*',