Skip to content

Commit

Permalink
Remove RoutingTable#version
Browse files Browse the repository at this point in the history
  • Loading branch information
nielsbauman committed Sep 12, 2024
1 parent 0ab2afb commit 4e4d56b
Show file tree
Hide file tree
Showing 18 changed files with 38 additions and 91 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -152,7 +152,6 @@ public void testClusterStateDiffSerialization() throws Exception {
}

// Check routing table
assertThat(clusterStateFromDiffs.routingTable().version(), equalTo(clusterState.routingTable().version()));
assertThat(clusterStateFromDiffs.routingTable().indicesRouting(), equalTo(clusterState.routingTable().indicesRouting()));

// Check cluster blocks
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -208,6 +208,7 @@ static TransportVersion def(int id) {
public static final TransportVersion ESQL_AGGREGATE_EXEC_TRACKS_INTERMEDIATE_ATTRS = def(8_738_00_0);
public static final TransportVersion CCS_TELEMETRY_STATS = def(8_739_00_0);
public static final TransportVersion GLOBAL_RETENTION_TELEMETRY = def(8_740_00_0);
public static final TransportVersion ROUTING_TABLE_VERSION_REMOVED = def(8_741_00_0);

/*
* STOP! READ THIS FIRST! No, really,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -8,6 +8,7 @@

package org.elasticsearch.cluster.routing;

import org.elasticsearch.TransportVersions;
import org.elasticsearch.cluster.Diff;
import org.elasticsearch.cluster.Diffable;
import org.elasticsearch.cluster.DiffableUtils;
Expand Down Expand Up @@ -45,22 +46,15 @@
*/
public class RoutingTable implements Iterable<IndexRoutingTable>, Diffable<RoutingTable> {

public static final RoutingTable EMPTY_ROUTING_TABLE = new RoutingTable(0, ImmutableOpenMap.of());

private final long version;
public static final RoutingTable EMPTY_ROUTING_TABLE = new RoutingTable(ImmutableOpenMap.of());

// index to IndexRoutingTable map
private final ImmutableOpenMap<String, IndexRoutingTable> indicesRouting;

private RoutingTable(long version, ImmutableOpenMap<String, IndexRoutingTable> indicesRouting) {
this.version = version;
private RoutingTable(ImmutableOpenMap<String, IndexRoutingTable> indicesRouting) {
this.indicesRouting = indicesRouting;
}

public RoutingTable withIncrementedVersion() {
return new RoutingTable(version + 1, indicesRouting);
}

/**
* Get's the {@link IndexShardRoutingTable} for the given shard id from the given {@link IndexRoutingTable}
* or throws a {@link ShardNotFoundException} if no shard by the given id is found in the IndexRoutingTable.
Expand All @@ -77,15 +71,6 @@ public static IndexShardRoutingTable shardRoutingTable(IndexRoutingTable indexRo
return indexShard;
}

/**
* Returns the version of the {@link RoutingTable}.
*
* @return version of the {@link RoutingTable}
*/
public long version() {
return this.version;
}

@Override
public Iterator<IndexRoutingTable> iterator() {
return indicesRouting.values().iterator();
Expand Down Expand Up @@ -331,7 +316,9 @@ public static Diff<RoutingTable> readDiffFrom(StreamInput in) throws IOException

public static RoutingTable readFrom(StreamInput in) throws IOException {
Builder builder = new Builder();
builder.version = in.readLong();
if (in.getTransportVersion().before(TransportVersions.ROUTING_TABLE_VERSION_REMOVED)) {
in.readLong();
}
int size = in.readVInt();
for (int i = 0; i < size; i++) {
IndexRoutingTable index = IndexRoutingTable.readFrom(in);
Expand All @@ -343,46 +330,45 @@ public static RoutingTable readFrom(StreamInput in) throws IOException {

@Override
public void writeTo(StreamOutput out) throws IOException {
out.writeLong(version);
out.writeLong(0);
out.writeCollection(indicesRouting.values());
}

private static class RoutingTableDiff implements Diff<RoutingTable> {

private final long version;

private final Diff<ImmutableOpenMap<String, IndexRoutingTable>> indicesRouting;

RoutingTableDiff(RoutingTable before, RoutingTable after) {
version = after.version;
indicesRouting = DiffableUtils.diff(before.indicesRouting, after.indicesRouting, DiffableUtils.getStringKeySerializer());
}

private static final DiffableUtils.DiffableValueReader<String, IndexRoutingTable> DIFF_VALUE_READER =
new DiffableUtils.DiffableValueReader<>(IndexRoutingTable::readFrom, IndexRoutingTable::readDiffFrom);

RoutingTableDiff(StreamInput in) throws IOException {
version = in.readLong();
if (in.getTransportVersion().before(TransportVersions.ROUTING_TABLE_VERSION_REMOVED)) {
in.readLong();
}
indicesRouting = DiffableUtils.readImmutableOpenMapDiff(in, DiffableUtils.getStringKeySerializer(), DIFF_VALUE_READER);
}

@Override
public RoutingTable apply(RoutingTable part) {
final ImmutableOpenMap<String, IndexRoutingTable> updatedRouting = indicesRouting.apply(part.indicesRouting);
if (part.version == version && updatedRouting == part.indicesRouting) {
if (updatedRouting == part.indicesRouting) {
return part;
}
return new RoutingTable(version, updatedRouting);
return new RoutingTable(updatedRouting);
}

@Override
public void writeTo(StreamOutput out) throws IOException {
out.writeLong(version);
out.writeLong(0);
indicesRouting.writeTo(out);
}
}

public static RoutingTable of(long version, RoutingNodes routingNodes) {
public static RoutingTable of(RoutingNodes routingNodes) {
Map<String, IndexRoutingTable.Builder> indexRoutingTableBuilders = new HashMap<>();
for (RoutingNode routingNode : routingNodes) {
for (ShardRouting shardRoutingEntry : routingNode) {
Expand All @@ -404,7 +390,7 @@ public static RoutingTable of(long version, RoutingNodes routingNodes) {
IndexRoutingTable indexRoutingTable = indexBuilder.build();
indicesRouting.put(indexRoutingTable.getIndex().getName(), indexRoutingTable);
}
return new RoutingTable(version, indicesRouting.build());
return new RoutingTable(indicesRouting.build());
}

public static Builder builder() {
Expand All @@ -429,7 +415,6 @@ public static Builder builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy,
public static class Builder {

private final ShardRoutingRoleStrategy shardRoutingRoleStrategy;
private long version;
private ImmutableOpenMap.Builder<String, IndexRoutingTable> indicesRouting;

public Builder() {
Expand All @@ -447,7 +432,6 @@ public Builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy) {

public Builder(ShardRoutingRoleStrategy shardRoutingRoleStrategy, RoutingTable routingTable) {
this.shardRoutingRoleStrategy = shardRoutingRoleStrategy;
this.version = routingTable.version;
this.indicesRouting = ImmutableOpenMap.builder(routingTable.indicesRouting);
}

Expand Down Expand Up @@ -591,16 +575,6 @@ public Builder remove(String index) {
return this;
}

public Builder version(long version) {
this.version = version;
return this;
}

public Builder incrementVersion() {
this.version++;
return this;
}

/**
* Builds the routing table. Note that once this is called the builder
* must be thrown away. If you need to build a new RoutingTable as a
Expand All @@ -610,15 +584,15 @@ public RoutingTable build() {
if (indicesRouting == null) {
throw new IllegalStateException("once build is called the builder cannot be reused");
}
RoutingTable table = new RoutingTable(version, indicesRouting.build());
RoutingTable table = new RoutingTable(indicesRouting.build());
indicesRouting = null;
return table;
}
}

@Override
public String toString() {
StringBuilder sb = new StringBuilder("routing_table (version ").append(version).append("):\n");
StringBuilder sb = new StringBuilder("routing_table:\n");
for (IndexRoutingTable entry : indicesRouting.values()) {
sb.append(entry.prettyPrint()).append('\n');
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -159,7 +159,7 @@ public ClusterState applyStartedShards(ClusterState clusterState, List<ShardRout
private static ClusterState buildResultAndLogHealthChange(ClusterState oldState, RoutingAllocation allocation, String reason) {
final RoutingTable oldRoutingTable = oldState.routingTable();
final RoutingNodes newRoutingNodes = allocation.routingNodes();
final RoutingTable newRoutingTable = RoutingTable.of(oldRoutingTable.version(), newRoutingNodes);
final RoutingTable newRoutingTable = RoutingTable.of(newRoutingNodes);
final Metadata newMetadata = allocation.updateMetadataWithRoutingChanges(newRoutingTable);
assert newRoutingTable.validate(newMetadata); // validates the routing table is coherent with the cluster state metadata

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -431,7 +431,7 @@ public RoutingAllocation immutableClone() {
deciders,
routingNodesChanged()
? ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), routingNodes))
.routingTable(RoutingTable.of(routingNodes))
.build()
: clusterState,
clusterInfo,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -505,9 +505,6 @@ private ClusterState patchVersions(ClusterState previousClusterState, ClusterSta
if (previousClusterState != newClusterState) {
// only the master controls the version numbers
Builder builder = incrementVersion(newClusterState);
if (previousClusterState.routingTable() != newClusterState.routingTable()) {
builder.routingTable(newClusterState.routingTable().withIncrementedVersion());
}
if (previousClusterState.metadata() != newClusterState.metadata()) {
builder.metadata(newClusterState.metadata().withIncrementedVersion());
}
Expand Down Expand Up @@ -535,10 +532,6 @@ private static boolean versionNumbersPreserved(ClusterState oldState, ClusterSta
if (oldState.metadata().version() != newState.metadata().version()) {
return false;
}
if (oldState.routingTable().version() != newState.routingTable().version()) {
// GatewayService is special and for odd legacy reasons gets to do this:
return oldState.clusterRecovered() == false && newState.clusterRecovered() && newState.routingTable().version() == 0;
}
return true;
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -98,8 +98,6 @@ static ClusterState updateRoutingTable(final ClusterState state, ShardRoutingRol
for (final IndexMetadata indexMetadata : state.metadata().indices().values()) {
routingTableBuilder.addAsRecovery(indexMetadata);
}
// start with 0 based versions for routing table
routingTableBuilder.version(0);
return ClusterState.builder(state).routingTable(routingTableBuilder.build()).build();
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -343,7 +343,7 @@ private static ClusterState createState(final int numNodes, final boolean isLoca
return ClusterState.builder(TEST_CLUSTER_NAME)
.nodes(createDiscoveryNodes(numNodes, isLocalMaster))
.metadata(metadata)
.routingTable(createRoutingTable(1, metadata))
.routingTable(createRoutingTable(metadata))
.build();
}

Expand Down Expand Up @@ -498,8 +498,8 @@ private static IndexMetadata createIndexMetadata(final Index index, final long v
}

// Create the routing table for a cluster state.
private static RoutingTable createRoutingTable(final long version, final Metadata metadata) {
final RoutingTable.Builder builder = RoutingTable.builder(TestShardRoutingRoleStrategies.DEFAULT_ROLE_ONLY).version(version);
private static RoutingTable createRoutingTable(final Metadata metadata) {
final RoutingTable.Builder builder = RoutingTable.builder(TestShardRoutingRoleStrategies.DEFAULT_ROLE_ONLY);
for (IndexMetadata indexMetadata : metadata.indices().values()) {
builder.addAsNew(indexMetadata);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -120,7 +120,7 @@ private void applyRerouteResult(ClusterState newClusterState) {
ClusterState.Builder builder = ClusterState.builder(newClusterState).incrementVersion();
if (previousClusterState.routingTable() != newClusterState.routingTable()) {
builder.routingTable(
RoutingTable.builder(newClusterState.routingTable()).version(newClusterState.routingTable().version() + 1).build()
RoutingTable.builder(newClusterState.routingTable()).build()
);
}
if (previousClusterState.metadata() != newClusterState.metadata()) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -497,7 +497,7 @@ public void testRemoveReplicasInRightOrder() {
public void testRoutingNodesRoundtrip() {
final RoutingTable originalTable = clusterState.getRoutingTable();
final RoutingNodes routingNodes = clusterState.getRoutingNodes();
final RoutingTable fromNodes = RoutingTable.of(originalTable.version(), routingNodes);
final RoutingTable fromNodes = RoutingTable.of(routingNodes);
// we don't have an equals implementation for the routing table so we assert equality by checking for a noop diff
final Diff<RoutingTable> routingTableDiff = fromNodes.diff(originalTable);
assertSame(originalTable, routingTableDiff.apply(originalTable));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -320,7 +320,7 @@ private static ClusterState updateClusterState(ClusterState state, RoutingAlloca
if (allocation.routingNodesChanged() == false) {
return state;
}
final RoutingTable newRoutingTable = RoutingTable.of(state.routingTable().version(), allocation.routingNodes());
final RoutingTable newRoutingTable = RoutingTable.of(allocation.routingNodes());
final Metadata newMetadata = allocation.updateMetadataWithRoutingChanges(newRoutingTable);
assert newRoutingTable.validate(newMetadata);

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -235,7 +235,7 @@ public void testAssignShardsToTheirPreviousLocationIfAvailable() {
}
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), routingNodes))
.routingTable(RoutingTable.of(routingNodes))
.build();

var ignored = randomBoolean()
Expand Down Expand Up @@ -285,7 +285,7 @@ public void testRespectsAssignmentOfUnknownPrimaries() {
}
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), routingNodes))
.routingTable(RoutingTable.of(routingNodes))
.build();

var desiredBalance = desiredBalanceComputer.compute(DesiredBalance.INITIAL, createInput(clusterState), queue(), input -> true);
Expand Down Expand Up @@ -334,7 +334,7 @@ public void testRespectsAssignmentOfUnknownReplicas() {
}
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), routingNodes))
.routingTable(RoutingTable.of(routingNodes))
.build();

var desiredBalance = desiredBalanceComputer.compute(DesiredBalance.INITIAL, createInput(clusterState), queue(), input -> true);
Expand Down Expand Up @@ -432,7 +432,7 @@ public ShardAllocationDecision decideShardAllocation(ShardRouting shard, Routing
);
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), desiredRoutingNodes))
.routingTable(RoutingTable.of(desiredRoutingNodes))
.build();

var desiredBalance1 = desiredBalanceComputer.compute(DesiredBalance.INITIAL, createInput(clusterState), queue(), input -> true);
Expand Down Expand Up @@ -501,7 +501,7 @@ public ShardAllocationDecision decideShardAllocation(ShardRouting shard, Routing
}
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), randomRoutingNodes))
.routingTable(RoutingTable.of(randomRoutingNodes))
.build();

allocateCalled.set(false);
Expand Down Expand Up @@ -540,7 +540,7 @@ public void testAppliesMoveCommands() {
routingNodes.startShard(iterator.initialize(shardRouting.primary() ? "node-0" : "node-1", null, 0L, changes), changes, 0L);
}
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(clusterState.routingTable().version(), routingNodes))
.routingTable(RoutingTable.of(routingNodes))
.build();

var desiredBalance = desiredBalanceComputer.compute(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1250,7 +1250,7 @@ public void testRebalanceDoesNotCauseHotSpots() {

totalOutgoingMoves.keySet().removeIf(nodeId -> isReconciled(allocation.routingNodes().node(nodeId), balance));
clusterState = ClusterState.builder(clusterState)
.routingTable(RoutingTable.of(allocation.routingTable().version(), allocation.routingNodes()))
.routingTable(RoutingTable.of(allocation.routingNodes()))
.build();
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2607,16 +2607,6 @@ public void testVersionNumberProtection() {
b -> b.version(randomFrom(currentState.metadata().version() - 1, currentState.metadata().version() + 1))
)
);

runVersionNumberProtectionTest(
currentState -> ClusterState.builder(currentState)
.routingTable(
RoutingTable.builder(currentState.routingTable())
.version(randomFrom(currentState.routingTable().version() - 1, currentState.routingTable().version() + 1))
.build()
)
.build()
);
}

private void runVersionNumberProtectionTest(UnaryOperator<ClusterState> updateOperator) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -142,7 +142,6 @@ public void testUpdateRoutingTable() {
{
final ClusterState newState = updateRoutingTable(initialState, TestShardRoutingRoleStrategies.DEFAULT_ROLE_ONLY);
assertTrue(newState.routingTable().hasIndex(index));
assertThat(newState.routingTable().version(), is(0L));
assertThat(newState.routingTable().allShards(index.getName()).size(), is(numOfShards));
}
{
Expand Down Expand Up @@ -179,7 +178,6 @@ public void testUpdateRoutingTable() {
TestShardRoutingRoleStrategies.DEFAULT_ROLE_ONLY
);
assertTrue(newState.routingTable().hasIndex(index));
assertThat(newState.routingTable().version(), is(0L));
assertThat(newState.routingTable().allShards(index.getName()).size(), is(numOfShards));
}
}
Expand Down
Loading

0 comments on commit 4e4d56b

Please sign in to comment.