RoutingTable.java 26 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627
  1. /*
  2. * Copyright Elasticsearch B.V. and/or licensed to Elasticsearch B.V. under one
  3. * or more contributor license agreements. Licensed under the Elastic License
  4. * 2.0 and the Server Side Public License, v 1; you may not use this file except
  5. * in compliance with, at your election, the Elastic License 2.0 or the Server
  6. * Side Public License, v 1.
  7. */
  8. package org.elasticsearch.cluster.routing;
  9. import org.elasticsearch.cluster.Diff;
  10. import org.elasticsearch.cluster.Diffable;
  11. import org.elasticsearch.cluster.DiffableUtils;
  12. import org.elasticsearch.cluster.metadata.IndexMetadata;
  13. import org.elasticsearch.cluster.metadata.Metadata;
  14. import org.elasticsearch.cluster.routing.RecoverySource.SnapshotRecoverySource;
  15. import org.elasticsearch.common.collect.ImmutableOpenMap;
  16. import org.elasticsearch.common.io.stream.StreamInput;
  17. import org.elasticsearch.common.io.stream.StreamOutput;
  18. import org.elasticsearch.core.Nullable;
  19. import org.elasticsearch.index.Index;
  20. import org.elasticsearch.index.IndexNotFoundException;
  21. import org.elasticsearch.index.shard.ShardId;
  22. import org.elasticsearch.index.shard.ShardNotFoundException;
  23. import java.io.IOException;
  24. import java.util.ArrayList;
  25. import java.util.Collections;
  26. import java.util.HashMap;
  27. import java.util.Iterator;
  28. import java.util.List;
  29. import java.util.Map;
  30. import java.util.Set;
  31. import java.util.function.Predicate;
  32. import java.util.stream.Stream;
  33. import static org.elasticsearch.cluster.metadata.MetadataIndexStateService.isIndexVerifiedBeforeClosed;
  34. /**
  35. * Represents a global cluster-wide routing table for all indices including the
  36. * version of the current routing state.
  37. *
  38. * @see IndexRoutingTable
  39. */
  40. public class RoutingTable implements Iterable<IndexRoutingTable>, Diffable<RoutingTable> {
  41. public static final RoutingTable EMPTY_ROUTING_TABLE = new RoutingTable(0, ImmutableOpenMap.of());
  42. private final long version;
  43. // index to IndexRoutingTable map
  44. private final ImmutableOpenMap<String, IndexRoutingTable> indicesRouting;
  45. private RoutingTable(long version, ImmutableOpenMap<String, IndexRoutingTable> indicesRouting) {
  46. this.version = version;
  47. this.indicesRouting = indicesRouting;
  48. }
  49. public RoutingTable withIncrementedVersion() {
  50. return new RoutingTable(version + 1, indicesRouting);
  51. }
  52. /**
  53. * Get's the {@link IndexShardRoutingTable} for the given shard id from the given {@link IndexRoutingTable}
  54. * or throws a {@link ShardNotFoundException} if no shard by the given id is found in the IndexRoutingTable.
  55. *
  56. * @param indexRouting IndexRoutingTable
  57. * @param shardId ShardId
  58. * @return IndexShardRoutingTable
  59. */
  60. public static IndexShardRoutingTable shardRoutingTable(IndexRoutingTable indexRouting, int shardId) {
  61. IndexShardRoutingTable indexShard = indexRouting.shard(shardId);
  62. if (indexShard == null) {
  63. throw new ShardNotFoundException(new ShardId(indexRouting.getIndex(), shardId));
  64. }
  65. return indexShard;
  66. }
  67. /**
  68. * Returns the version of the {@link RoutingTable}.
  69. *
  70. * @return version of the {@link RoutingTable}
  71. */
  72. public long version() {
  73. return this.version;
  74. }
  75. @Override
  76. public Iterator<IndexRoutingTable> iterator() {
  77. return indicesRouting.values().iterator();
  78. }
  79. public boolean hasIndex(String index) {
  80. return indicesRouting.containsKey(index);
  81. }
  82. public boolean hasIndex(Index index) {
  83. IndexRoutingTable indexRouting = index(index);
  84. return indexRouting != null;
  85. }
  86. public IndexRoutingTable index(String index) {
  87. return indicesRouting.get(index);
  88. }
  89. public IndexRoutingTable index(Index index) {
  90. IndexRoutingTable indexRouting = index(index.getName());
  91. return indexRouting != null && indexRouting.getIndex().equals(index) ? indexRouting : null;
  92. }
  93. public Map<String, IndexRoutingTable> indicesRouting() {
  94. return indicesRouting;
  95. }
  96. public Map<String, IndexRoutingTable> getIndicesRouting() {
  97. return indicesRouting();
  98. }
  99. /**
  100. * All shards for the provided index and shard id
  101. * @return All the shard routing entries for the given index and shard id
  102. * @throws IndexNotFoundException if provided index does not exist
  103. * @throws ShardNotFoundException if provided shard id is unknown
  104. */
  105. public IndexShardRoutingTable shardRoutingTable(String index, int shardId) {
  106. IndexRoutingTable indexRouting = index(index);
  107. if (indexRouting == null) {
  108. throw new IndexNotFoundException(index);
  109. }
  110. return shardRoutingTable(indexRouting, shardId);
  111. }
  112. /**
  113. * All shards for the provided {@link ShardId}
  114. * @return All the shard routing entries for the given index and shard id
  115. * @throws IndexNotFoundException if provided index does not exist
  116. * @throws ShardNotFoundException if provided shard id is unknown
  117. */
  118. public IndexShardRoutingTable shardRoutingTable(ShardId shardId) {
  119. IndexRoutingTable indexRouting = index(shardId.getIndex());
  120. if (indexRouting == null) {
  121. throw new IndexNotFoundException(shardId.getIndex());
  122. }
  123. IndexShardRoutingTable shard = indexRouting.shard(shardId.id());
  124. if (shard == null) {
  125. throw new ShardNotFoundException(shardId);
  126. }
  127. return shard;
  128. }
  129. @Nullable
  130. public ShardRouting getByAllocationId(ShardId shardId, String allocationId) {
  131. final IndexRoutingTable indexRoutingTable = index(shardId.getIndex());
  132. if (indexRoutingTable == null) {
  133. return null;
  134. }
  135. final IndexShardRoutingTable shardRoutingTable = indexRoutingTable.shard(shardId.getId());
  136. return shardRoutingTable == null ? null : shardRoutingTable.getByAllocationId(allocationId);
  137. }
  138. public boolean validate(Metadata metadata) {
  139. for (IndexRoutingTable indexRoutingTable : this) {
  140. if (indexRoutingTable.validate(metadata) == false) {
  141. return false;
  142. }
  143. }
  144. return true;
  145. }
  146. /**
  147. * All the shards (replicas) for all indices in this routing table.
  148. *
  149. * @return All the shards
  150. */
  151. public Stream<ShardRouting> allShards() {
  152. return indicesRouting.values().stream().flatMap(IndexRoutingTable::allShards).flatMap(IndexShardRoutingTable::allShards);
  153. }
  154. public Iterable<ShardRouting> allShardsIterator() {
  155. return () -> allShards().iterator();
  156. }
  157. /**
  158. * All the shards (replicas) for the provided index.
  159. *
  160. * @param index The index to return all the shards (replicas).
  161. * @return All the shards matching the specific index
  162. * @throws IndexNotFoundException If the index passed does not exists
  163. */
  164. public List<ShardRouting> allShards(String index) {
  165. List<ShardRouting> shards = new ArrayList<>();
  166. IndexRoutingTable indexRoutingTable = index(index);
  167. if (indexRoutingTable == null) {
  168. throw new IndexNotFoundException(index);
  169. }
  170. for (int shardId = 0; shardId < indexRoutingTable.size(); shardId++) {
  171. IndexShardRoutingTable indexShardRoutingTable = indexRoutingTable.shard(shardId);
  172. for (int copy = 0; copy < indexShardRoutingTable.size(); copy++) {
  173. shards.add(indexShardRoutingTable.shard(copy));
  174. }
  175. }
  176. return shards;
  177. }
  178. /**
  179. * Return GroupShardsIterator where each active shard routing has it's own shard iterator.
  180. *
  181. * @param includeEmpty if true, a shard iterator will be added for non-assigned shards as well
  182. */
  183. public GroupShardsIterator<ShardIterator> allActiveShardsGrouped(String[] indices, boolean includeEmpty) {
  184. return allSatisfyingPredicateShardsGrouped(indices, includeEmpty, ShardRouting::active);
  185. }
  186. /**
  187. * Return GroupShardsIterator where each assigned shard routing has it's own shard iterator.
  188. *
  189. * @param includeEmpty if true, a shard iterator will be added for non-assigned shards as well
  190. */
  191. public GroupShardsIterator<ShardIterator> allAssignedShardsGrouped(String[] indices, boolean includeEmpty) {
  192. return allSatisfyingPredicateShardsGrouped(indices, includeEmpty, ShardRouting::assignedToNode);
  193. }
  194. private GroupShardsIterator<ShardIterator> allSatisfyingPredicateShardsGrouped(
  195. String[] indices,
  196. boolean includeEmpty,
  197. Predicate<ShardRouting> predicate
  198. ) {
  199. // use list here since we need to maintain identity across shards
  200. ArrayList<ShardIterator> set = new ArrayList<>();
  201. for (String index : indices) {
  202. IndexRoutingTable indexRoutingTable = index(index);
  203. if (indexRoutingTable == null) {
  204. continue;
  205. // we simply ignore indices that don't exists (make sense for operations that use it currently)
  206. }
  207. for (int shardId = 0; shardId < indexRoutingTable.size(); shardId++) {
  208. IndexShardRoutingTable indexShardRoutingTable = indexRoutingTable.shard(shardId);
  209. for (int copy = 0; copy < indexShardRoutingTable.size(); copy++) {
  210. ShardRouting shardRouting = indexShardRoutingTable.shard(copy);
  211. if (predicate.test(shardRouting)) {
  212. set.add(shardRouting.shardsIt());
  213. } else if (includeEmpty) { // we need this for counting properly, just make it an empty one
  214. set.add(new PlainShardIterator(shardRouting.shardId(), Collections.emptyList()));
  215. }
  216. }
  217. }
  218. }
  219. return GroupShardsIterator.sortAndCreate(set);
  220. }
  221. public ShardsIterator allShards(String[] indices) {
  222. return allShardsSatisfyingPredicate(indices, shardRouting -> true, false);
  223. }
  224. public ShardsIterator allActiveShards(String[] indices) {
  225. return allShardsSatisfyingPredicate(indices, ShardRouting::active, false);
  226. }
  227. public ShardsIterator allShardsIncludingRelocationTargets(String[] indices) {
  228. return allShardsSatisfyingPredicate(indices, shardRouting -> true, true);
  229. }
  230. private ShardsIterator allShardsSatisfyingPredicate(
  231. String[] indices,
  232. Predicate<ShardRouting> predicate,
  233. boolean includeRelocationTargets
  234. ) {
  235. // use list here since we need to maintain identity across shards
  236. List<ShardRouting> shards = new ArrayList<>();
  237. for (String index : indices) {
  238. IndexRoutingTable indexRoutingTable = index(index);
  239. if (indexRoutingTable == null) {
  240. continue;
  241. // we simply ignore indices that don't exists (make sense for operations that use it currently)
  242. }
  243. for (int shardId = 0; shardId < indexRoutingTable.size(); shardId++) {
  244. IndexShardRoutingTable indexShardRoutingTable = indexRoutingTable.shard(shardId);
  245. for (int copy = 0; copy < indexShardRoutingTable.size(); copy++) {
  246. ShardRouting shardRouting = indexShardRoutingTable.shard(copy);
  247. if (predicate.test(shardRouting)) {
  248. shards.add(shardRouting);
  249. if (includeRelocationTargets && shardRouting.relocating()) {
  250. shards.add(shardRouting.getTargetRelocatingShard());
  251. }
  252. }
  253. }
  254. }
  255. }
  256. return new PlainShardsIterator(shards);
  257. }
  258. /**
  259. * All the *active* primary shards for the provided indices grouped (each group is a single element, consisting
  260. * of the primary shard). This is handy for components that expect to get group iterators, but still want in some
  261. * cases to iterate over all primary shards (and not just one shard in replication group).
  262. *
  263. * @param indices The indices to return all the shards (replicas)
  264. * @return All the primary shards grouped into a single shard element group each
  265. * @throws IndexNotFoundException If an index passed does not exists
  266. */
  267. public GroupShardsIterator<ShardIterator> activePrimaryShardsGrouped(String[] indices, boolean includeEmpty) {
  268. // use list here since we need to maintain identity across shards
  269. ArrayList<ShardIterator> set = new ArrayList<>();
  270. for (String index : indices) {
  271. IndexRoutingTable indexRoutingTable = index(index);
  272. if (indexRoutingTable == null) {
  273. throw new IndexNotFoundException(index);
  274. }
  275. for (int i = 0; i < indexRoutingTable.size(); i++) {
  276. IndexShardRoutingTable indexShardRoutingTable = indexRoutingTable.shard(i);
  277. ShardRouting primary = indexShardRoutingTable.primaryShard();
  278. if (primary.active()) {
  279. set.add(primary.shardsIt());
  280. } else if (includeEmpty) { // we need this for counting properly, just make it an empty one
  281. set.add(new PlainShardIterator(primary.shardId(), Collections.emptyList()));
  282. }
  283. }
  284. }
  285. return GroupShardsIterator.sortAndCreate(set);
  286. }
  287. @Override
  288. public Diff<RoutingTable> diff(RoutingTable previousState) {
  289. return new RoutingTableDiff(previousState, this);
  290. }
  291. public static Diff<RoutingTable> readDiffFrom(StreamInput in) throws IOException {
  292. return new RoutingTableDiff(in);
  293. }
  294. public static RoutingTable readFrom(StreamInput in) throws IOException {
  295. Builder builder = new Builder();
  296. builder.version = in.readLong();
  297. int size = in.readVInt();
  298. for (int i = 0; i < size; i++) {
  299. IndexRoutingTable index = IndexRoutingTable.readFrom(in);
  300. builder.add(index);
  301. }
  302. return builder.build();
  303. }
  304. @Override
  305. public void writeTo(StreamOutput out) throws IOException {
  306. out.writeLong(version);
  307. out.writeCollection(indicesRouting.values());
  308. }
  309. private static class RoutingTableDiff implements Diff<RoutingTable> {
  310. private final long version;
  311. private final Diff<ImmutableOpenMap<String, IndexRoutingTable>> indicesRouting;
  312. RoutingTableDiff(RoutingTable before, RoutingTable after) {
  313. version = after.version;
  314. indicesRouting = DiffableUtils.diff(before.indicesRouting, after.indicesRouting, DiffableUtils.getStringKeySerializer());
  315. }
  316. private static final DiffableUtils.DiffableValueReader<String, IndexRoutingTable> DIFF_VALUE_READER =
  317. new DiffableUtils.DiffableValueReader<>(IndexRoutingTable::readFrom, IndexRoutingTable::readDiffFrom);
  318. RoutingTableDiff(StreamInput in) throws IOException {
  319. version = in.readLong();
  320. indicesRouting = DiffableUtils.readImmutableOpenMapDiff(in, DiffableUtils.getStringKeySerializer(), DIFF_VALUE_READER);
  321. }
  322. @Override
  323. public RoutingTable apply(RoutingTable part) {
  324. final ImmutableOpenMap<String, IndexRoutingTable> updatedRouting = indicesRouting.apply(part.indicesRouting);
  325. if (part.version == version && updatedRouting == part.indicesRouting) {
  326. return part;
  327. }
  328. return new RoutingTable(version, updatedRouting);
  329. }
  330. @Override
  331. public void writeTo(StreamOutput out) throws IOException {
  332. out.writeLong(version);
  333. indicesRouting.writeTo(out);
  334. }
  335. }
  336. public static RoutingTable of(long version, RoutingNodes routingNodes) {
  337. Map<String, IndexRoutingTable.Builder> indexRoutingTableBuilders = new HashMap<>();
  338. for (RoutingNode routingNode : routingNodes) {
  339. for (ShardRouting shardRoutingEntry : routingNode) {
  340. // every relocating shard has a double entry, ignore the target one.
  341. if (shardRoutingEntry.initializing() && shardRoutingEntry.relocatingNodeId() != null) continue;
  342. Builder.addShard(indexRoutingTableBuilders, shardRoutingEntry);
  343. }
  344. }
  345. for (ShardRouting shardRoutingEntry : routingNodes.unassigned()) {
  346. Builder.addShard(indexRoutingTableBuilders, shardRoutingEntry);
  347. }
  348. for (ShardRouting shardRoutingEntry : routingNodes.unassigned().ignored()) {
  349. Builder.addShard(indexRoutingTableBuilders, shardRoutingEntry);
  350. }
  351. ImmutableOpenMap.Builder<String, IndexRoutingTable> indicesRouting = ImmutableOpenMap.builder(indexRoutingTableBuilders.size());
  352. for (IndexRoutingTable.Builder indexBuilder : indexRoutingTableBuilders.values()) {
  353. IndexRoutingTable indexRoutingTable = indexBuilder.build();
  354. indicesRouting.put(indexRoutingTable.getIndex().getName(), indexRoutingTable);
  355. }
  356. return new RoutingTable(version, indicesRouting.build());
  357. }
  358. public static Builder builder() {
  359. return new Builder();
  360. }
  361. public static Builder builder(RoutingTable routingTable) {
  362. return new Builder(routingTable);
  363. }
  364. public static Builder builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy) {
  365. return new Builder(shardRoutingRoleStrategy);
  366. }
  367. public static Builder builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy, RoutingTable routingTable) {
  368. return new Builder(shardRoutingRoleStrategy, routingTable);
  369. }
  370. /**
  371. * Builder for the routing table. Note that build can only be called one time.
  372. */
  373. public static class Builder {
  374. private final ShardRoutingRoleStrategy shardRoutingRoleStrategy;
  375. private long version;
  376. private ImmutableOpenMap.Builder<String, IndexRoutingTable> indicesRouting;
  377. public Builder() {
  378. this(ShardRoutingRoleStrategy.NO_SHARD_CREATION);
  379. }
  380. public Builder(RoutingTable routingTable) {
  381. this(ShardRoutingRoleStrategy.NO_SHARD_CREATION, routingTable);
  382. }
  383. public Builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy) {
  384. this.shardRoutingRoleStrategy = shardRoutingRoleStrategy;
  385. this.indicesRouting = ImmutableOpenMap.builder();
  386. }
  387. public Builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy, RoutingTable routingTable) {
  388. this.shardRoutingRoleStrategy = shardRoutingRoleStrategy;
  389. this.version = routingTable.version;
  390. this.indicesRouting = ImmutableOpenMap.builder(routingTable.indicesRouting);
  391. }
  392. private static void addShard(
  393. final Map<String, IndexRoutingTable.Builder> indexRoutingTableBuilders,
  394. final ShardRouting shardRoutingEntry
  395. ) {
  396. Index index = shardRoutingEntry.index();
  397. indexRoutingTableBuilders.computeIfAbsent(index.getName(), idxName -> IndexRoutingTable.builder(index))
  398. .addShard(shardRoutingEntry);
  399. }
  400. /**
  401. * Update the number of replicas for the specified indices.
  402. *
  403. * @param numberOfReplicas the number of replicas
  404. * @param indices the indices to update the number of replicas for
  405. * @return the builder
  406. */
  407. public Builder updateNumberOfReplicas(final int numberOfReplicas, final String[] indices) {
  408. if (indicesRouting == null) {
  409. throw new IllegalStateException("once build is called the builder cannot be reused");
  410. }
  411. for (String index : indices) {
  412. IndexRoutingTable indexRoutingTable = indicesRouting.get(index);
  413. if (indexRoutingTable == null) {
  414. // ignore index missing failure, its closed...
  415. continue;
  416. }
  417. int currentNumberOfReplicas = indexRoutingTable.shard(0).size() - 1; // remove the required primary
  418. IndexRoutingTable.Builder builder = new IndexRoutingTable.Builder(shardRoutingRoleStrategy, indexRoutingTable.getIndex());
  419. // re-add all the shards
  420. builder.ensureShardArray(indexRoutingTable.size());
  421. for (int i = 0; i < indexRoutingTable.size(); i++) {
  422. builder.addIndexShard(new IndexShardRoutingTable.Builder(indexRoutingTable.shard(i)));
  423. }
  424. if (currentNumberOfReplicas < numberOfReplicas) {
  425. // now, add "empty" ones
  426. for (int i = 0; i < (numberOfReplicas - currentNumberOfReplicas); i++) {
  427. builder.addReplica(shardRoutingRoleStrategy.newReplicaRole());
  428. }
  429. } else if (currentNumberOfReplicas > numberOfReplicas) {
  430. for (int i = 0; i < (currentNumberOfReplicas - numberOfReplicas); i++) {
  431. builder.removeReplica();
  432. }
  433. }
  434. indicesRouting.put(index, builder.build());
  435. }
  436. return this;
  437. }
  438. public Builder addAsNew(IndexMetadata indexMetadata) {
  439. if (indexMetadata.getState() == IndexMetadata.State.OPEN) {
  440. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  441. shardRoutingRoleStrategy,
  442. indexMetadata.getIndex()
  443. ).initializeAsNew(indexMetadata);
  444. add(indexRoutingBuilder);
  445. }
  446. return this;
  447. }
  448. public Builder addAsRecovery(IndexMetadata indexMetadata) {
  449. if (indexMetadata.getState() == IndexMetadata.State.OPEN || isIndexVerifiedBeforeClosed(indexMetadata)) {
  450. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  451. shardRoutingRoleStrategy,
  452. indexMetadata.getIndex()
  453. ).initializeAsRecovery(indexMetadata);
  454. add(indexRoutingBuilder);
  455. }
  456. return this;
  457. }
  458. public Builder addAsFromDangling(IndexMetadata indexMetadata) {
  459. if (indexMetadata.getState() == IndexMetadata.State.OPEN || isIndexVerifiedBeforeClosed(indexMetadata)) {
  460. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  461. shardRoutingRoleStrategy,
  462. indexMetadata.getIndex()
  463. ).initializeAsFromDangling(indexMetadata);
  464. add(indexRoutingBuilder);
  465. }
  466. return this;
  467. }
  468. public Builder addAsFromCloseToOpen(IndexMetadata indexMetadata) {
  469. if (indexMetadata.getState() == IndexMetadata.State.OPEN) {
  470. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  471. shardRoutingRoleStrategy,
  472. indexMetadata.getIndex()
  473. ).initializeAsFromCloseToOpen(indexMetadata, indicesRouting.get(indexMetadata.getIndex().getName()));
  474. add(indexRoutingBuilder);
  475. }
  476. return this;
  477. }
  478. public Builder addAsFromOpenToClose(IndexMetadata indexMetadata) {
  479. assert isIndexVerifiedBeforeClosed(indexMetadata);
  480. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  481. shardRoutingRoleStrategy,
  482. indexMetadata.getIndex()
  483. ).initializeAsFromOpenToClose(indexMetadata, indicesRouting.get(indexMetadata.getIndex().getName()));
  484. return add(indexRoutingBuilder);
  485. }
  486. public Builder addAsRestore(IndexMetadata indexMetadata, SnapshotRecoverySource recoverySource) {
  487. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  488. shardRoutingRoleStrategy,
  489. indexMetadata.getIndex()
  490. ).initializeAsRestore(indexMetadata, recoverySource, indicesRouting.get(indexMetadata.getIndex().getName()));
  491. add(indexRoutingBuilder);
  492. return this;
  493. }
  494. public Builder addAsNewRestore(IndexMetadata indexMetadata, SnapshotRecoverySource recoverySource, Set<Integer> ignoreShards) {
  495. IndexRoutingTable.Builder indexRoutingBuilder = new IndexRoutingTable.Builder(
  496. shardRoutingRoleStrategy,
  497. indexMetadata.getIndex()
  498. ).initializeAsNewRestore(indexMetadata, recoverySource, ignoreShards);
  499. add(indexRoutingBuilder);
  500. return this;
  501. }
  502. public Builder add(IndexRoutingTable indexRoutingTable) {
  503. if (indicesRouting == null) {
  504. throw new IllegalStateException("once build is called the builder cannot be reused");
  505. }
  506. indicesRouting.put(indexRoutingTable.getIndex().getName(), indexRoutingTable);
  507. return this;
  508. }
  509. public Builder add(IndexRoutingTable.Builder indexRoutingTableBuilder) {
  510. add(indexRoutingTableBuilder.build());
  511. return this;
  512. }
  513. public Builder remove(String index) {
  514. if (indicesRouting == null) {
  515. throw new IllegalStateException("once build is called the builder cannot be reused");
  516. }
  517. indicesRouting.remove(index);
  518. return this;
  519. }
  520. public Builder version(long version) {
  521. this.version = version;
  522. return this;
  523. }
  524. public Builder incrementVersion() {
  525. this.version++;
  526. return this;
  527. }
  528. /**
  529. * Builds the routing table. Note that once this is called the builder
  530. * must be thrown away. If you need to build a new RoutingTable as a
  531. * copy of this one you'll need to build a new RoutingTable.Builder.
  532. */
  533. public RoutingTable build() {
  534. if (indicesRouting == null) {
  535. throw new IllegalStateException("once build is called the builder cannot be reused");
  536. }
  537. RoutingTable table = new RoutingTable(version, indicesRouting.build());
  538. indicesRouting = null;
  539. return table;
  540. }
  541. }
  542. @Override
  543. public String toString() {
  544. StringBuilder sb = new StringBuilder("routing_table (version ").append(version).append("):\n");
  545. for (IndexRoutingTable entry : indicesRouting.values()) {
  546. sb.append(entry.prettyPrint()).append('\n');
  547. }
  548. return sb.toString();
  549. }
  550. }