mirror of
https://github.com/apache/druid.git
synced 2025-02-17 07:25:02 +00:00
Remove deprecated coordinator dynamic configs (#14923)
Changes: [A] Remove config `decommissioningMaxPercentOfMaxSegmentsToMove` - It is a complicated config 😅 , - It is always desirable to prioritize move from decommissioning servers so that they can be terminated quickly, so this should always be 100% - It is already handled by `smartSegmentLoading` (enabled by default) [B] Remove config `maxNonPrimaryReplicantsToLoad` This was added in #11135 to address two requirements: - Prevent coordinator runs from getting stuck assigning too many segments to historicals - Prevent load of replicas from competing with load of unavailable segments Both of these requirements are now already met thanks to: - Round-robin segment assignment - Prioritization in the new coordinator - Modifications to `replicationThrottleLimit` - `smartSegmentLoading` (enabled by default)
This commit is contained in:
parent
7f26b80e21
commit
ec630e3671
@ -940,10 +940,8 @@ A sample Coordinator dynamic config JSON object is shown below:
|
||||
"killTaskSlotRatio": 0.10,
|
||||
"maxKillTaskSlots": 5,
|
||||
"decommissioningNodes": ["localhost:8182", "localhost:8282"],
|
||||
"decommissioningMaxPercentOfMaxSegmentsToMove": 70,
|
||||
"pauseCoordination": false,
|
||||
"replicateAfterLoadTimeout": false,
|
||||
"maxNonPrimaryReplicantsToLoad": 2147483647
|
||||
"replicateAfterLoadTimeout": false
|
||||
}
|
||||
```
|
||||
|
||||
@ -965,11 +963,9 @@ Issuing a GET request at the same URL will return the spec that is currently in
|
||||
|`killPendingSegmentsSkipList`|List of data sources for which pendingSegments are _NOT_ cleaned up if property `druid.coordinator.kill.pendingSegments.on` is true. This can be a list of comma-separated data sources or a JSON array.|none|
|
||||
|`maxSegmentsInNodeLoadingQueue`|The maximum number of segments allowed in the load queue of any given server. Use this parameter to load segments faster if, for example, the cluster contains slow-loading nodes or if there are too many segments to be replicated to a particular node (when faster loading is preferred to better segments distribution). The optimal value depends on the loading speed of segments, acceptable replication time and number of nodes.|500|
|
||||
|`useRoundRobinSegmentAssignment`|Boolean flag for whether segments should be assigned to historicals in a round robin fashion. When disabled, segment assignment is done using the chosen balancer strategy. When enabled, this can speed up segment assignments leaving balancing to move the segments to their optimal locations (based on the balancer strategy) lazily.|true|
|
||||
|`decommissioningNodes`|List of historical servers to 'decommission'. Coordinator will not assign new segments to 'decommissioning' servers, and segments will be moved away from them to be placed on non-decommissioning servers at the maximum rate specified by `decommissioningMaxPercentOfMaxSegmentsToMove`.|none|
|
||||
|`decommissioningMaxPercentOfMaxSegmentsToMove`|Upper limit of segments the Coordinator can move from decommissioning servers to active non-decommissioning servers during a single run. This value is relative to the total maximum number of segments that can be moved at any given time based upon the value of `maxSegmentsToMove`.<br /><br />If `decommissioningMaxPercentOfMaxSegmentsToMove` is 0, the Coordinator does not move segments to decommissioning servers, effectively putting them in a type of "maintenance" mode. In this case, decommissioning servers do not participate in balancing or assignment by load rules. The Coordinator still considers segments on decommissioning servers as candidates to replicate on active servers.<br /><br />Decommissioning can stall if there are no available active servers to move the segments to. You can use the maximum percent of decommissioning segment movements to prioritize balancing or to decrease commissioning time to prevent active servers from being overloaded. The value must be between 0 and 100.|70|
|
||||
|`decommissioningNodes`|List of historical servers to 'decommission'. Coordinator will not assign new segments to 'decommissioning' servers, and segments will be moved away from them to be placed on non-decommissioning servers at the maximum rate specified by `maxSegmentsToMove`.|none|
|
||||
|`pauseCoordination`|Boolean flag for whether or not the coordinator should execute its various duties of coordinating the cluster. Setting this to true essentially pauses all coordination work while allowing the API to remain up. Duties that are paused include all classes that implement the `CoordinatorDuty` Interface. Such duties include: Segment balancing, Segment compaction, Submitting kill tasks for unused segments (if enabled), Logging of used segments in the cluster, Marking of newly unused or overshadowed segments, Matching and execution of load/drop rules for used segments, Unloading segments that are no longer marked as used from Historical servers. An example of when an admin may want to pause coordination would be if they are doing deep storage maintenance on HDFS Name Nodes with downtime and don't want the coordinator to be directing Historical Nodes to hit the Name Node with API requests until maintenance is done and the deep store is declared healthy for use again.|false|
|
||||
|`replicateAfterLoadTimeout`|Boolean flag for whether or not additional replication is needed for segments that have failed to load due to the expiry of `druid.coordinator.load.timeout`. If this is set to true, the coordinator will attempt to replicate the failed segment on a different historical server. This helps improve the segment availability if there are a few slow historicals in the cluster. However, the slow historical may still load the segment later and the coordinator may issue drop requests if the segment is over-replicated.|false|
|
||||
|`maxNonPrimaryReplicantsToLoad`|The maximum number of replicas that can be assigned across all tiers in a single Coordinator run. This parameter serves the same purpose as `replicationThrottleLimit` except this limit applies at the cluster-level instead of per tier. The default value does not apply a limit to the number of replicas assigned per coordination cycle. If you want to use a non-default value for this property, you may want to start with `~20%` of the number of segments found on the historical server with the most segments. Use the Druid metric, `coordinator/time` with the filter `duty=org.apache.druid.server.coordinator.duty.RunRules` to see how different values of this property impact your Coordinator execution time.|`Integer.MAX_VALUE` (no limit)|
|
||||
|
||||
##### Smart segment loading
|
||||
|
||||
@ -986,9 +982,7 @@ Druid computes the values to optimize Coordinator performance, based on the curr
|
||||
|`maxSegmentsInNodeLoadingQueue`|0|Removes the limit on load queue size.|
|
||||
|`replicationThrottleLimit`|5% of used segments, minimum value 100|Prevents aggressive replication when a historical disappears only intermittently.|
|
||||
|`replicantLifetime`|60|Allows segments to wait about an hour (assuming a Coordinator period of 1 minute) in the load queue before an alert is raised. In `smartSegmentLoading` mode, load queues are not limited by size. Segments might therefore assigned to a load queue even if the corresponding server is slow to load them.|
|
||||
|`maxNonPrimaryReplicantsToLoad`|`Integer.MAX_VALUE` (no limit)|This throttling is already handled by `replicationThrottleLimit`.|
|
||||
|`maxSegmentsToMove`|2% of used segments, minimum value 100, maximum value 1000|Ensures that some segments are always moving in the cluster to keep it well balanced. The maximum value keeps the Coordinator run times bounded.|
|
||||
|`decommissioningMaxPercentOfMaxSegmentsToMove`|100|Prioritizes the move of segments from decommissioning servers so that they can be terminated quickly.|
|
||||
|`balancerComputeThreads`|`num_cores` / 2|Ensures that there are enough threads to perform balancing computations without hogging all Coordinator resources.|
|
||||
|
||||
When `smartSegmentLoading` is disabled, Druid uses the configured values of these properties.
|
||||
|
@ -26,7 +26,6 @@ import com.google.common.base.Preconditions;
|
||||
import com.google.common.collect.ImmutableSet;
|
||||
import org.apache.druid.common.config.JacksonConfigManager;
|
||||
import org.apache.druid.error.InvalidInput;
|
||||
import org.apache.druid.java.util.common.logger.Logger;
|
||||
import org.apache.druid.server.coordinator.duty.KillUnusedSegments;
|
||||
import org.apache.druid.server.coordinator.loading.LoadQueuePeon;
|
||||
import org.apache.druid.server.coordinator.stats.Dimension;
|
||||
@ -34,8 +33,6 @@ import org.apache.druid.utils.JvmUtils;
|
||||
|
||||
import javax.annotation.Nonnull;
|
||||
import javax.annotation.Nullable;
|
||||
import javax.validation.constraints.Max;
|
||||
import javax.validation.constraints.Min;
|
||||
import javax.validation.constraints.NotNull;
|
||||
import java.util.Collection;
|
||||
import java.util.EnumMap;
|
||||
@ -79,9 +76,6 @@ public class CoordinatorDynamicConfig
|
||||
private final Map<String, String> debugDimensions;
|
||||
private final Map<Dimension, String> validDebugDimensions;
|
||||
|
||||
@Deprecated
|
||||
private final int decommissioningMaxPercentOfMaxSegmentsToMove;
|
||||
|
||||
/**
|
||||
* Stale pending segments belonging to the data sources in this list are not killed by {@link
|
||||
* KillStalePendingSegments}. In other words, segments in these data sources are "protected".
|
||||
@ -92,10 +86,10 @@ public class CoordinatorDynamicConfig
|
||||
private final Set<String> dataSourcesToNotKillStalePendingSegmentsIn;
|
||||
|
||||
/**
|
||||
* The maximum number of segments that could be queued for loading to any given server.
|
||||
* Default values is 0 with the meaning of "unbounded" (any number of
|
||||
* segments could be added to the loading queue for any server).
|
||||
* See {@link LoadQueuePeon}, {@link org.apache.druid.server.coordinator.rules.LoadRule#run}
|
||||
* The maximum number of segments that can be queued for loading to any given server.
|
||||
*
|
||||
* @see LoadQueuePeon
|
||||
* @see org.apache.druid.server.coordinator.rules.LoadRule#run
|
||||
*/
|
||||
private final int maxSegmentsInNodeLoadingQueue;
|
||||
private final boolean pauseCoordination;
|
||||
@ -108,16 +102,6 @@ public class CoordinatorDynamicConfig
|
||||
*/
|
||||
private final boolean replicateAfterLoadTimeout;
|
||||
|
||||
/**
|
||||
* This is the maximum number of non-primary segment replicants to load per Coordination run. This number can
|
||||
* be set to put a hard upper limit on the number of replicants loaded. It is a tool that can help prevent
|
||||
* long delays in new data loads after events such as a Historical server leaving the cluster.
|
||||
*/
|
||||
@Deprecated
|
||||
private final int maxNonPrimaryReplicantsToLoad;
|
||||
|
||||
private static final Logger log = new Logger(CoordinatorDynamicConfig.class);
|
||||
|
||||
@JsonCreator
|
||||
public CoordinatorDynamicConfig(
|
||||
// Keeping the legacy 'millisToWaitBeforeDeleting' property name for backward compatibility. When the project is
|
||||
@ -144,10 +128,8 @@ public class CoordinatorDynamicConfig
|
||||
@JsonProperty("killPendingSegmentsSkipList") Object dataSourcesToNotKillStalePendingSegmentsIn,
|
||||
@JsonProperty("maxSegmentsInNodeLoadingQueue") @Nullable Integer maxSegmentsInNodeLoadingQueue,
|
||||
@JsonProperty("decommissioningNodes") Object decommissioningNodes,
|
||||
@JsonProperty("decommissioningMaxPercentOfMaxSegmentsToMove") int decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
@JsonProperty("pauseCoordination") boolean pauseCoordination,
|
||||
@JsonProperty("replicateAfterLoadTimeout") boolean replicateAfterLoadTimeout,
|
||||
@JsonProperty("maxNonPrimaryReplicantsToLoad") @Nullable Integer maxNonPrimaryReplicantsToLoad,
|
||||
@JsonProperty("useRoundRobinSegmentAssignment") @Nullable Boolean useRoundRobinSegmentAssignment,
|
||||
@JsonProperty("smartSegmentLoading") @Nullable Boolean smartSegmentLoading,
|
||||
@JsonProperty("debugDimensions") @Nullable Map<String, String> debugDimensions
|
||||
@ -186,31 +168,10 @@ public class CoordinatorDynamicConfig
|
||||
Defaults.MAX_SEGMENTS_IN_NODE_LOADING_QUEUE
|
||||
);
|
||||
this.decommissioningNodes = parseJsonStringOrArray(decommissioningNodes);
|
||||
Preconditions.checkArgument(
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove >= 0 && decommissioningMaxPercentOfMaxSegmentsToMove <= 100,
|
||||
"'decommissioningMaxPercentOfMaxSegmentsToMove' should be in range [0, 100]"
|
||||
);
|
||||
this.decommissioningMaxPercentOfMaxSegmentsToMove = decommissioningMaxPercentOfMaxSegmentsToMove;
|
||||
|
||||
this.pauseCoordination = pauseCoordination;
|
||||
this.replicateAfterLoadTimeout = replicateAfterLoadTimeout;
|
||||
|
||||
if (maxNonPrimaryReplicantsToLoad == null) {
|
||||
log.debug(
|
||||
"maxNonPrimaryReplicantsToLoad was null! This is likely because your metastore does not "
|
||||
+ "reflect this configuration being added to Druid in a recent release. Druid is defaulting the value "
|
||||
+ "to the Druid default of %d. It is recommended that you re-submit your dynamic config with your "
|
||||
+ "desired value for maxNonPrimaryReplicantsToLoad",
|
||||
Defaults.MAX_NON_PRIMARY_REPLICANTS_TO_LOAD
|
||||
);
|
||||
maxNonPrimaryReplicantsToLoad = Defaults.MAX_NON_PRIMARY_REPLICANTS_TO_LOAD;
|
||||
}
|
||||
Preconditions.checkArgument(
|
||||
maxNonPrimaryReplicantsToLoad >= 0,
|
||||
"maxNonPrimaryReplicantsToLoad must be greater than or equal to 0."
|
||||
);
|
||||
this.maxNonPrimaryReplicantsToLoad = maxNonPrimaryReplicantsToLoad;
|
||||
|
||||
this.useRoundRobinSegmentAssignment = Builder.valueOrDefault(
|
||||
useRoundRobinSegmentAssignment,
|
||||
Defaults.USE_ROUND_ROBIN_ASSIGNMENT
|
||||
@ -362,8 +323,7 @@ public class CoordinatorDynamicConfig
|
||||
|
||||
/**
|
||||
* List of historical servers to 'decommission'. Coordinator will not assign new segments to 'decommissioning'
|
||||
* servers, and segments will be moved away from them to be placed on non-decommissioning servers at the maximum rate
|
||||
* specified by {@link CoordinatorDynamicConfig#getDecommissioningMaxPercentOfMaxSegmentsToMove}.
|
||||
* servers, and segments will be moved away from them to be placed on non-decommissioning servers.
|
||||
*
|
||||
* @return list of host:port entries
|
||||
*/
|
||||
@ -385,27 +345,6 @@ public class CoordinatorDynamicConfig
|
||||
return validDebugDimensions;
|
||||
}
|
||||
|
||||
/**
|
||||
* The percent of {@link CoordinatorDynamicConfig#getMaxSegmentsToMove()} that determines the maximum number of
|
||||
* segments that may be moved away from 'decommissioning' servers (specified by
|
||||
* {@link CoordinatorDynamicConfig#getDecommissioningNodes()}) to non-decommissioning servers during one Coordinator
|
||||
* balancer run. If this value is 0, segments will neither be moved from or to 'decommissioning' servers, effectively
|
||||
* putting them in a sort of "maintenance" mode that will not participate in balancing or assignment by load rules.
|
||||
* Decommissioning can also become stalled if there are no available active servers to place the segments. By
|
||||
* adjusting this value, an operator can prevent active servers from overload by prioritizing balancing, or
|
||||
* decrease decommissioning time instead.
|
||||
*
|
||||
* @return number in range [0, 100]
|
||||
*/
|
||||
@Min(0)
|
||||
@Max(100)
|
||||
@Deprecated
|
||||
@JsonProperty
|
||||
public int getDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
{
|
||||
return decommissioningMaxPercentOfMaxSegmentsToMove;
|
||||
}
|
||||
|
||||
@JsonProperty
|
||||
public boolean getPauseCoordination()
|
||||
{
|
||||
@ -418,14 +357,6 @@ public class CoordinatorDynamicConfig
|
||||
return replicateAfterLoadTimeout;
|
||||
}
|
||||
|
||||
@Min(0)
|
||||
@Deprecated
|
||||
@JsonProperty
|
||||
public int getMaxNonPrimaryReplicantsToLoad()
|
||||
{
|
||||
return maxNonPrimaryReplicantsToLoad;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String toString()
|
||||
{
|
||||
@ -444,10 +375,8 @@ public class CoordinatorDynamicConfig
|
||||
", dataSourcesToNotKillStalePendingSegmentsIn=" + dataSourcesToNotKillStalePendingSegmentsIn +
|
||||
", maxSegmentsInNodeLoadingQueue=" + maxSegmentsInNodeLoadingQueue +
|
||||
", decommissioningNodes=" + decommissioningNodes +
|
||||
", decommissioningMaxPercentOfMaxSegmentsToMove=" + decommissioningMaxPercentOfMaxSegmentsToMove +
|
||||
", pauseCoordination=" + pauseCoordination +
|
||||
", replicateAfterLoadTimeout=" + replicateAfterLoadTimeout +
|
||||
", maxNonPrimaryReplicantsToLoad=" + maxNonPrimaryReplicantsToLoad +
|
||||
'}';
|
||||
}
|
||||
|
||||
@ -467,13 +396,11 @@ public class CoordinatorDynamicConfig
|
||||
&& mergeBytesLimit == that.mergeBytesLimit
|
||||
&& mergeSegmentsLimit == that.mergeSegmentsLimit
|
||||
&& maxSegmentsToMove == that.maxSegmentsToMove
|
||||
&& decommissioningMaxPercentOfMaxSegmentsToMove == that.decommissioningMaxPercentOfMaxSegmentsToMove
|
||||
&& balancerComputeThreads == that.balancerComputeThreads
|
||||
&& replicantLifetime == that.replicantLifetime
|
||||
&& replicationThrottleLimit == that.replicationThrottleLimit
|
||||
&& replicateAfterLoadTimeout == that.replicateAfterLoadTimeout
|
||||
&& maxSegmentsInNodeLoadingQueue == that.maxSegmentsInNodeLoadingQueue
|
||||
&& maxNonPrimaryReplicantsToLoad == that.maxNonPrimaryReplicantsToLoad
|
||||
&& useRoundRobinSegmentAssignment == that.useRoundRobinSegmentAssignment
|
||||
&& pauseCoordination == that.pauseCoordination
|
||||
&& Objects.equals(
|
||||
@ -505,9 +432,7 @@ public class CoordinatorDynamicConfig
|
||||
maxKillTaskSlots,
|
||||
dataSourcesToNotKillStalePendingSegmentsIn,
|
||||
decommissioningNodes,
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
pauseCoordination,
|
||||
maxNonPrimaryReplicantsToLoad,
|
||||
debugDimensions
|
||||
);
|
||||
}
|
||||
@ -535,10 +460,8 @@ public class CoordinatorDynamicConfig
|
||||
static final int REPLICANT_LIFETIME = 15;
|
||||
static final int REPLICATION_THROTTLE_LIMIT = 500;
|
||||
static final int MAX_SEGMENTS_IN_NODE_LOADING_QUEUE = 500;
|
||||
static final int DECOMMISSIONING_MAX_SEGMENTS_TO_MOVE_PERCENT = 70;
|
||||
static final boolean PAUSE_COORDINATION = false;
|
||||
static final boolean REPLICATE_AFTER_LOAD_TIMEOUT = false;
|
||||
static final int MAX_NON_PRIMARY_REPLICANTS_TO_LOAD = Integer.MAX_VALUE;
|
||||
static final boolean USE_ROUND_ROBIN_ASSIGNMENT = true;
|
||||
static final boolean SMART_SEGMENT_LOADING = true;
|
||||
|
||||
@ -566,10 +489,8 @@ public class CoordinatorDynamicConfig
|
||||
private Integer maxSegmentsInNodeLoadingQueue;
|
||||
private Object decommissioningNodes;
|
||||
private Map<String, String> debugDimensions;
|
||||
private Integer decommissioningMaxPercentOfMaxSegmentsToMove;
|
||||
private Boolean pauseCoordination;
|
||||
private Boolean replicateAfterLoadTimeout;
|
||||
private Integer maxNonPrimaryReplicantsToLoad;
|
||||
private Boolean useRoundRobinSegmentAssignment;
|
||||
private Boolean smartSegmentLoading;
|
||||
|
||||
@ -592,11 +513,8 @@ public class CoordinatorDynamicConfig
|
||||
@JsonProperty("killPendingSegmentsSkipList") @Nullable Object dataSourcesToNotKillStalePendingSegmentsIn,
|
||||
@JsonProperty("maxSegmentsInNodeLoadingQueue") @Nullable Integer maxSegmentsInNodeLoadingQueue,
|
||||
@JsonProperty("decommissioningNodes") @Nullable Object decommissioningNodes,
|
||||
@JsonProperty("decommissioningMaxPercentOfMaxSegmentsToMove")
|
||||
@Nullable Integer decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
@JsonProperty("pauseCoordination") @Nullable Boolean pauseCoordination,
|
||||
@JsonProperty("replicateAfterLoadTimeout") @Nullable Boolean replicateAfterLoadTimeout,
|
||||
@JsonProperty("maxNonPrimaryReplicantsToLoad") @Nullable Integer maxNonPrimaryReplicantsToLoad,
|
||||
@JsonProperty("useRoundRobinSegmentAssignment") @Nullable Boolean useRoundRobinSegmentAssignment,
|
||||
@JsonProperty("smartSegmentLoading") @Nullable Boolean smartSegmentLoading,
|
||||
@JsonProperty("debugDimensions") @Nullable Map<String, String> debugDimensions
|
||||
@ -615,10 +533,8 @@ public class CoordinatorDynamicConfig
|
||||
this.dataSourcesToNotKillStalePendingSegmentsIn = dataSourcesToNotKillStalePendingSegmentsIn;
|
||||
this.maxSegmentsInNodeLoadingQueue = maxSegmentsInNodeLoadingQueue;
|
||||
this.decommissioningNodes = decommissioningNodes;
|
||||
this.decommissioningMaxPercentOfMaxSegmentsToMove = decommissioningMaxPercentOfMaxSegmentsToMove;
|
||||
this.pauseCoordination = pauseCoordination;
|
||||
this.replicateAfterLoadTimeout = replicateAfterLoadTimeout;
|
||||
this.maxNonPrimaryReplicantsToLoad = maxNonPrimaryReplicantsToLoad;
|
||||
this.useRoundRobinSegmentAssignment = useRoundRobinSegmentAssignment;
|
||||
this.smartSegmentLoading = smartSegmentLoading;
|
||||
this.debugDimensions = debugDimensions;
|
||||
@ -630,18 +546,6 @@ public class CoordinatorDynamicConfig
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withMergeBytesLimit(long mergeBytesLimit)
|
||||
{
|
||||
this.mergeBytesLimit = mergeBytesLimit;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withMergeSegmentsLimit(int mergeSegmentsLimit)
|
||||
{
|
||||
this.mergeSegmentsLimit = mergeSegmentsLimit;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withMaxSegmentsToMove(int maxSegmentsToMove)
|
||||
{
|
||||
this.maxSegmentsToMove = maxSegmentsToMove;
|
||||
@ -666,18 +570,6 @@ public class CoordinatorDynamicConfig
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withDebugDimensions(Map<String, String> debugDimensions)
|
||||
{
|
||||
this.debugDimensions = debugDimensions;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withBalancerComputeThreads(int balancerComputeThreads)
|
||||
{
|
||||
this.balancerComputeThreads = balancerComputeThreads;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withSpecificDataSourcesToKillUnusedSegmentsIn(Set<String> dataSources)
|
||||
{
|
||||
this.specificDataSourcesToKillUnusedSegmentsIn = dataSources;
|
||||
@ -708,12 +600,6 @@ public class CoordinatorDynamicConfig
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withDecommissioningMaxPercentOfMaxSegmentsToMove(Integer percent)
|
||||
{
|
||||
this.decommissioningMaxPercentOfMaxSegmentsToMove = percent;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withPauseCoordination(boolean pauseCoordination)
|
||||
{
|
||||
this.pauseCoordination = pauseCoordination;
|
||||
@ -726,12 +612,6 @@ public class CoordinatorDynamicConfig
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withMaxNonPrimaryReplicantsToLoad(int maxNonPrimaryReplicantsToLoad)
|
||||
{
|
||||
this.maxNonPrimaryReplicantsToLoad = maxNonPrimaryReplicantsToLoad;
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder withUseRoundRobinSegmentAssignment(boolean useRoundRobinSegmentAssignment)
|
||||
{
|
||||
this.useRoundRobinSegmentAssignment = useRoundRobinSegmentAssignment;
|
||||
@ -761,13 +641,8 @@ public class CoordinatorDynamicConfig
|
||||
dataSourcesToNotKillStalePendingSegmentsIn,
|
||||
valueOrDefault(maxSegmentsInNodeLoadingQueue, Defaults.MAX_SEGMENTS_IN_NODE_LOADING_QUEUE),
|
||||
decommissioningNodes,
|
||||
valueOrDefault(
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
Defaults.DECOMMISSIONING_MAX_SEGMENTS_TO_MOVE_PERCENT
|
||||
),
|
||||
valueOrDefault(pauseCoordination, Defaults.PAUSE_COORDINATION),
|
||||
valueOrDefault(replicateAfterLoadTimeout, Defaults.REPLICATE_AFTER_LOAD_TIMEOUT),
|
||||
valueOrDefault(maxNonPrimaryReplicantsToLoad, Defaults.MAX_NON_PRIMARY_REPLICANTS_TO_LOAD),
|
||||
valueOrDefault(useRoundRobinSegmentAssignment, Defaults.USE_ROUND_ROBIN_ASSIGNMENT),
|
||||
valueOrDefault(smartSegmentLoading, Defaults.SMART_SEGMENT_LOADING),
|
||||
debugDimensions
|
||||
@ -798,13 +673,8 @@ public class CoordinatorDynamicConfig
|
||||
valueOrDefault(dataSourcesToNotKillStalePendingSegmentsIn, defaults.getDataSourcesToNotKillStalePendingSegmentsIn()),
|
||||
valueOrDefault(maxSegmentsInNodeLoadingQueue, defaults.getMaxSegmentsInNodeLoadingQueue()),
|
||||
valueOrDefault(decommissioningNodes, defaults.getDecommissioningNodes()),
|
||||
valueOrDefault(
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
defaults.getDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
),
|
||||
valueOrDefault(pauseCoordination, defaults.getPauseCoordination()),
|
||||
valueOrDefault(replicateAfterLoadTimeout, defaults.getReplicateAfterLoadTimeout()),
|
||||
valueOrDefault(maxNonPrimaryReplicantsToLoad, defaults.getMaxNonPrimaryReplicantsToLoad()),
|
||||
valueOrDefault(useRoundRobinSegmentAssignment, defaults.isUseRoundRobinSegmentAssignment()),
|
||||
valueOrDefault(smartSegmentLoading, defaults.isSmartSegmentLoading()),
|
||||
valueOrDefault(debugDimensions, defaults.getDebugDimensions())
|
||||
|
@ -201,17 +201,13 @@ public class TierSegmentBalancer
|
||||
*/
|
||||
private int getNumDecommSegmentsToMove(int maxSegmentsToMove)
|
||||
{
|
||||
final CoordinatorDynamicConfig dynamicConfig = params.getCoordinatorDynamicConfig();
|
||||
if (decommissioningServers.isEmpty() || activeServers.isEmpty()) {
|
||||
return 0;
|
||||
} else if (dynamicConfig.isSmartSegmentLoading()) {
|
||||
} else {
|
||||
final int decommSegmentsToMove = decommissioningServers.stream().mapToInt(
|
||||
server -> server.getProjectedSegments().getTotalSegmentCount()
|
||||
).sum();
|
||||
return Math.min(decommSegmentsToMove, maxSegmentsToMove);
|
||||
} else {
|
||||
int maxPercentageToMove = dynamicConfig.getDecommissioningMaxPercentOfMaxSegmentsToMove();
|
||||
return (int) Math.ceil(maxSegmentsToMove * (maxPercentageToMove / 100.0));
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -31,21 +31,15 @@ import java.util.Map;
|
||||
* <ul>
|
||||
* <li>{@link CoordinatorDynamicConfig#getReplicationThrottleLimit()} - Maximum
|
||||
* number of replicas that can be assigned to a tier in a single run.</li>
|
||||
* <li>{@link CoordinatorDynamicConfig#getMaxNonPrimaryReplicantsToLoad()} -
|
||||
* Maximum number of total replicas that can be assigned across all tiers in a
|
||||
* single run.</li>
|
||||
* </ul>
|
||||
*/
|
||||
public class ReplicationThrottler
|
||||
{
|
||||
private final int replicationThrottleLimit;
|
||||
private final int maxReplicaAssignmentsInRun;
|
||||
|
||||
private final Object2IntOpenHashMap<String> tierToNumAssigned = new Object2IntOpenHashMap<>();
|
||||
private final Object2IntOpenHashMap<String> tierToMaxAssignments = new Object2IntOpenHashMap<>();
|
||||
|
||||
private int totalReplicasAssignedInRun;
|
||||
|
||||
/**
|
||||
* Creates a new ReplicationThrottler for use during a single coordinator run.
|
||||
* The number of replicas loading on a tier must always be within the current
|
||||
@ -53,22 +47,17 @@ public class ReplicationThrottler
|
||||
* replicas at the start of a coordinator run, it may be assigned only
|
||||
* {@code replicationThrottleLimit - k} more replicas during the run.
|
||||
*
|
||||
* @param tierToLoadingReplicaCount Map from tier name to number of replicas
|
||||
* already being loaded.
|
||||
* @param replicationThrottleLimit Maximum number of replicas that can be
|
||||
* assigned to a single tier in the current run.
|
||||
* @param maxReplicaAssignmentsInRun Max number of total replicas that can be
|
||||
* assigned across all tiers in the current run.
|
||||
* @param tierToLoadingReplicaCount Map from tier name to number of replicas
|
||||
* already being loaded.
|
||||
* @param replicationThrottleLimit Maximum number of replicas that can be
|
||||
* assigned to a single tier in the current run.
|
||||
*/
|
||||
public ReplicationThrottler(
|
||||
Map<String, Integer> tierToLoadingReplicaCount,
|
||||
int replicationThrottleLimit,
|
||||
int maxReplicaAssignmentsInRun
|
||||
int replicationThrottleLimit
|
||||
)
|
||||
{
|
||||
this.replicationThrottleLimit = replicationThrottleLimit;
|
||||
this.maxReplicaAssignmentsInRun = maxReplicaAssignmentsInRun;
|
||||
this.totalReplicasAssignedInRun = 0;
|
||||
|
||||
if (tierToLoadingReplicaCount != null) {
|
||||
tierToLoadingReplicaCount.forEach(
|
||||
@ -82,13 +71,12 @@ public class ReplicationThrottler
|
||||
|
||||
public boolean isReplicationThrottledForTier(String tier)
|
||||
{
|
||||
return tierToNumAssigned.getInt(tier) >= tierToMaxAssignments.getOrDefault(tier, replicationThrottleLimit)
|
||||
|| totalReplicasAssignedInRun >= maxReplicaAssignmentsInRun;
|
||||
return tierToNumAssigned.getOrDefault(tier, 0)
|
||||
>= tierToMaxAssignments.getOrDefault(tier, replicationThrottleLimit);
|
||||
}
|
||||
|
||||
public void incrementAssignedReplicas(String tier)
|
||||
{
|
||||
++totalReplicasAssignedInRun;
|
||||
tierToNumAssigned.addTo(tier, 1);
|
||||
}
|
||||
|
||||
|
@ -32,7 +32,6 @@ public class SegmentLoadingConfig
|
||||
|
||||
private final int maxSegmentsInLoadQueue;
|
||||
private final int replicationThrottleLimit;
|
||||
private final int maxReplicaAssignmentsInRun;
|
||||
private final int maxLifetimeInLoadQueue;
|
||||
|
||||
private final int balancerComputeThreads;
|
||||
@ -58,7 +57,6 @@ public class SegmentLoadingConfig
|
||||
return new SegmentLoadingConfig(
|
||||
0,
|
||||
replicationThrottleLimit,
|
||||
Integer.MAX_VALUE,
|
||||
60,
|
||||
true,
|
||||
CoordinatorDynamicConfig.getDefaultBalancerComputeThreads()
|
||||
@ -68,7 +66,6 @@ public class SegmentLoadingConfig
|
||||
return new SegmentLoadingConfig(
|
||||
dynamicConfig.getMaxSegmentsInNodeLoadingQueue(),
|
||||
dynamicConfig.getReplicationThrottleLimit(),
|
||||
dynamicConfig.getMaxNonPrimaryReplicantsToLoad(),
|
||||
dynamicConfig.getReplicantLifetime(),
|
||||
dynamicConfig.isUseRoundRobinSegmentAssignment(),
|
||||
dynamicConfig.getBalancerComputeThreads()
|
||||
@ -79,7 +76,6 @@ public class SegmentLoadingConfig
|
||||
private SegmentLoadingConfig(
|
||||
int maxSegmentsInLoadQueue,
|
||||
int replicationThrottleLimit,
|
||||
int maxReplicaAssignmentsInRun,
|
||||
int maxLifetimeInLoadQueue,
|
||||
boolean useRoundRobinSegmentAssignment,
|
||||
int balancerComputeThreads
|
||||
@ -87,7 +83,6 @@ public class SegmentLoadingConfig
|
||||
{
|
||||
this.maxSegmentsInLoadQueue = maxSegmentsInLoadQueue;
|
||||
this.replicationThrottleLimit = replicationThrottleLimit;
|
||||
this.maxReplicaAssignmentsInRun = maxReplicaAssignmentsInRun;
|
||||
this.maxLifetimeInLoadQueue = maxLifetimeInLoadQueue;
|
||||
this.useRoundRobinSegmentAssignment = useRoundRobinSegmentAssignment;
|
||||
this.balancerComputeThreads = balancerComputeThreads;
|
||||
@ -113,11 +108,6 @@ public class SegmentLoadingConfig
|
||||
return maxLifetimeInLoadQueue;
|
||||
}
|
||||
|
||||
public int getMaxReplicaAssignmentsInRun()
|
||||
{
|
||||
return maxReplicaAssignmentsInRun;
|
||||
}
|
||||
|
||||
public int getBalancerComputeThreads()
|
||||
{
|
||||
return balancerComputeThreads;
|
||||
|
@ -573,8 +573,7 @@ public class StrategicSegmentAssigner implements SegmentActionHandler
|
||||
);
|
||||
return new ReplicationThrottler(
|
||||
tierToLoadingReplicaCount,
|
||||
loadingConfig.getReplicationThrottleLimit(),
|
||||
loadingConfig.getMaxReplicaAssignmentsInRun()
|
||||
loadingConfig.getReplicationThrottleLimit()
|
||||
);
|
||||
}
|
||||
|
||||
|
@ -121,68 +121,6 @@ public class BalanceSegmentsTest
|
||||
Assert.assertEquals(2L, totalMoved);
|
||||
}
|
||||
|
||||
/**
|
||||
* Server 1 has 2 segments.
|
||||
* Server 2 (decommissioning) has 2 segments.
|
||||
* Server 3 is empty.
|
||||
* Decommissioning percent is 60.
|
||||
* Max segments to move is 3.
|
||||
* 2 (of 2) segments should be moved from Server 2 and 1 (of 2) from Server 1.
|
||||
*/
|
||||
@Test
|
||||
public void testMoveDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
{
|
||||
final ServerHolder serverHolder1 = createHolder(server1, true, segment1, segment2, segment3, segment4);
|
||||
final ServerHolder serverHolder2 = createHolder(server2, false);
|
||||
|
||||
// ceil(3 * 0.6) = 2 segments from decommissioning servers
|
||||
CoordinatorDynamicConfig dynamicConfig =
|
||||
CoordinatorDynamicConfig.builder()
|
||||
.withSmartSegmentLoading(false)
|
||||
.withMaxSegmentsToMove(3)
|
||||
.withDecommissioningMaxPercentOfMaxSegmentsToMove(60)
|
||||
.build();
|
||||
DruidCoordinatorRuntimeParams params =
|
||||
defaultRuntimeParamsBuilder(serverHolder1, serverHolder2)
|
||||
.withDynamicConfigs(dynamicConfig)
|
||||
.withBalancerStrategy(balancerStrategy)
|
||||
.withBroadcastDatasources(broadcastDatasources)
|
||||
.withSegmentAssignerUsing(loadQueueManager)
|
||||
.build();
|
||||
|
||||
CoordinatorRunStats stats = runBalancer(params);
|
||||
|
||||
// 2 segments are moved from the decommissioning server
|
||||
long totalMoved = stats.getSegmentStat(Stats.Segments.MOVED, "normal", "datasource1")
|
||||
+ stats.getSegmentStat(Stats.Segments.MOVED, "normal", "datasource2");
|
||||
Assert.assertEquals(2L, totalMoved);
|
||||
Set<DataSegment> segmentsMoved = serverHolder2.getPeon().getSegmentsToLoad();
|
||||
Assert.assertEquals(2, segmentsMoved.size());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testZeroDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
{
|
||||
final ServerHolder holder1 = createHolder(server1, false, segment1, segment2);
|
||||
final ServerHolder holder2 = createHolder(server2, true, segment3, segment4);
|
||||
final ServerHolder holder3 = createHolder(server3, false);
|
||||
|
||||
CoordinatorDynamicConfig dynamicConfig =
|
||||
CoordinatorDynamicConfig.builder()
|
||||
.withSmartSegmentLoading(false)
|
||||
.withDecommissioningMaxPercentOfMaxSegmentsToMove(0)
|
||||
.withMaxSegmentsToMove(1).build();
|
||||
DruidCoordinatorRuntimeParams params =
|
||||
defaultRuntimeParamsBuilder(holder1, holder2, holder3).withDynamicConfigs(dynamicConfig).build();
|
||||
|
||||
CoordinatorRunStats stats = runBalancer(params);
|
||||
|
||||
// Verify that either segment1 or segment2 is chosen for move
|
||||
Assert.assertEquals(1L, stats.getSegmentStat(Stats.Segments.MOVED, "normal", segment1.getDataSource()));
|
||||
DataSegment movingSegment = holder3.getPeon().getSegmentsToLoad().iterator().next();
|
||||
Assert.assertEquals(segment1.getDataSource(), movingSegment.getDataSource());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testMaxDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
{
|
||||
@ -193,7 +131,6 @@ public class BalanceSegmentsTest
|
||||
CoordinatorDynamicConfig dynamicConfig =
|
||||
CoordinatorDynamicConfig.builder()
|
||||
.withSmartSegmentLoading(false)
|
||||
.withDecommissioningMaxPercentOfMaxSegmentsToMove(100)
|
||||
.withMaxSegmentsToMove(1).build();
|
||||
DruidCoordinatorRuntimeParams params =
|
||||
defaultRuntimeParamsBuilder(holder1, holder2, holder3).withDynamicConfigs(dynamicConfig).build();
|
||||
@ -206,11 +143,8 @@ public class BalanceSegmentsTest
|
||||
Assert.assertEquals(segment3.getDataSource(), movingSegment.getDataSource());
|
||||
}
|
||||
|
||||
/**
|
||||
* Should balance segments as usual (ignoring percent) with empty decommissioningNodes.
|
||||
*/
|
||||
@Test
|
||||
public void testMoveDecommissioningMaxPercentOfMaxSegmentsToMoveWithNoDecommissioning()
|
||||
public void testMoveWithNoDecommissioning()
|
||||
{
|
||||
final ServerHolder serverHolder1 = createHolder(server1, segment1, segment2, segment3, segment4);
|
||||
final ServerHolder serverHolder2 = createHolder(server2);
|
||||
@ -219,7 +153,6 @@ public class BalanceSegmentsTest
|
||||
CoordinatorDynamicConfig.builder()
|
||||
.withSmartSegmentLoading(false)
|
||||
.withMaxSegmentsToMove(4)
|
||||
.withDecommissioningMaxPercentOfMaxSegmentsToMove(9)
|
||||
.build();
|
||||
DruidCoordinatorRuntimeParams params =
|
||||
defaultRuntimeParamsBuilder(serverHolder1, serverHolder2)
|
||||
|
@ -114,7 +114,7 @@ public class RunRulesTest
|
||||
/**
|
||||
* Nodes:
|
||||
* normal - 2 replicants
|
||||
* maxNonPrimaryReplicantsToLoad - 10
|
||||
* replicationThrottleLimit - 10
|
||||
* Expect only 34 segments to be loaded despite there being 48 primary + non-primary replicants to load!
|
||||
*/
|
||||
@Test
|
||||
@ -152,7 +152,7 @@ public class RunRulesTest
|
||||
.withDynamicConfigs(
|
||||
CoordinatorDynamicConfig
|
||||
.builder()
|
||||
.withMaxNonPrimaryReplicantsToLoad(10)
|
||||
.withReplicationThrottleLimit(10)
|
||||
.withSmartSegmentLoading(false)
|
||||
.build()
|
||||
)
|
||||
@ -171,7 +171,7 @@ public class RunRulesTest
|
||||
* Nodes:
|
||||
* normal - 2 replicants
|
||||
* hot - 2 replicants
|
||||
* maxNonPrimaryReplicantsToLoad - 48
|
||||
* replicationThrottleLimit - 48
|
||||
* Expect only 72 segments to be loaded despite there being 96 primary + non-primary replicants to load!
|
||||
*/
|
||||
@Test
|
||||
@ -216,7 +216,7 @@ public class RunRulesTest
|
||||
.withBalancerStrategy(new CostBalancerStrategy(balancerExecutor))
|
||||
.withDynamicConfigs(
|
||||
CoordinatorDynamicConfig.builder()
|
||||
.withMaxNonPrimaryReplicantsToLoad(10)
|
||||
.withReplicationThrottleLimit(10)
|
||||
.withSmartSegmentLoading(false)
|
||||
.build()
|
||||
)
|
||||
@ -225,7 +225,6 @@ public class RunRulesTest
|
||||
|
||||
CoordinatorRunStats stats = runDutyAndGetStats(params);
|
||||
|
||||
// maxNonPrimaryReplicantsToLoad takes effect on hot tier, but not normal tier
|
||||
Assert.assertEquals(10L, stats.getSegmentStat(Stats.Segments.ASSIGNED, "hot", DATASOURCE));
|
||||
Assert.assertEquals(48L, stats.getSegmentStat(Stats.Segments.ASSIGNED, "normal", DATASOURCE));
|
||||
|
||||
|
@ -34,8 +34,7 @@ public class ReplicationThrottlerTest
|
||||
final int replicationThrottleLimit = 10;
|
||||
ReplicationThrottler throttler = new ReplicationThrottler(
|
||||
ImmutableMap.of(),
|
||||
replicationThrottleLimit,
|
||||
1000
|
||||
replicationThrottleLimit
|
||||
);
|
||||
|
||||
// Verify that both the tiers can be assigned replicas upto the limit
|
||||
@ -54,8 +53,7 @@ public class ReplicationThrottlerTest
|
||||
final int replicationThrottleLimit = 10;
|
||||
ReplicationThrottler throttler = new ReplicationThrottler(
|
||||
ImmutableMap.of(TIER_1, 10, TIER_2, 7),
|
||||
replicationThrottleLimit,
|
||||
1000
|
||||
replicationThrottleLimit
|
||||
);
|
||||
|
||||
// T1 cannot be assigned any more replicas
|
||||
|
@ -56,10 +56,8 @@ public class CoordinatorDynamicConfigTest
|
||||
+ " \"maxKillTaskSlots\": 2,\n"
|
||||
+ " \"maxSegmentsInNodeLoadingQueue\": 1,\n"
|
||||
+ " \"decommissioningNodes\": [\"host1\", \"host2\"],\n"
|
||||
+ " \"decommissioningMaxPercentOfMaxSegmentsToMove\": 9,\n"
|
||||
+ " \"pauseCoordination\": false,\n"
|
||||
+ " \"replicateAfterLoadTimeout\": false,\n"
|
||||
+ " \"maxNonPrimaryReplicantsToLoad\": 2147483647\n"
|
||||
+ " \"replicateAfterLoadTimeout\": false\n"
|
||||
+ "}\n";
|
||||
|
||||
CoordinatorDynamicConfig actual = mapper.readValue(
|
||||
@ -88,10 +86,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
decommissioning,
|
||||
9,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withDecommissioningNodes(ImmutableSet.of("host1")).build(actual);
|
||||
@ -110,13 +106,11 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
9,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withDecommissioningMaxPercentOfMaxSegmentsToMove(5).build(actual);
|
||||
actual = CoordinatorDynamicConfig.builder().build(actual);
|
||||
assertConfig(
|
||||
actual,
|
||||
1,
|
||||
@ -132,10 +126,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withPauseCoordination(true).build(actual);
|
||||
@ -154,10 +146,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withReplicateAfterLoadTimeout(true).build(actual);
|
||||
@ -176,13 +166,11 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
Integer.MAX_VALUE
|
||||
true
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withMaxNonPrimaryReplicantsToLoad(10).build(actual);
|
||||
actual = CoordinatorDynamicConfig.builder().build(actual);
|
||||
assertConfig(
|
||||
actual,
|
||||
1,
|
||||
@ -198,10 +186,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
10
|
||||
true
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withKillTaskSlotRatio(1.0).build(actual);
|
||||
@ -220,10 +206,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
10
|
||||
true
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withMaxKillTaskSlots(5).build(actual);
|
||||
@ -242,10 +226,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
10
|
||||
true
|
||||
);
|
||||
}
|
||||
|
||||
@ -279,10 +261,8 @@ public class CoordinatorDynamicConfigTest
|
||||
null,
|
||||
null,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
10,
|
||||
false,
|
||||
false,
|
||||
null
|
||||
@ -308,10 +288,8 @@ public class CoordinatorDynamicConfigTest
|
||||
null,
|
||||
null,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
true,
|
||||
true,
|
||||
10,
|
||||
false,
|
||||
false,
|
||||
null
|
||||
@ -361,10 +339,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
decommissioning,
|
||||
0,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withDecommissioningNodes(ImmutableSet.of("host1")).build(actual);
|
||||
@ -383,13 +359,11 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
0,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
actual = CoordinatorDynamicConfig.builder().withDecommissioningMaxPercentOfMaxSegmentsToMove(5).build(actual);
|
||||
actual = CoordinatorDynamicConfig.builder().build(actual);
|
||||
assertConfig(
|
||||
actual,
|
||||
1,
|
||||
@ -405,10 +379,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of("host1"),
|
||||
5,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -451,24 +423,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
ImmutableSet.of(),
|
||||
0,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testSerdeHandlesInvalidDecommissioningPercentToMove()
|
||||
{
|
||||
final String errorMsg = "'decommissioningMaxPercentOfMaxSegmentsToMove' should be in range [0, 100]";
|
||||
assertThatDeserializationFailsWithMessage(
|
||||
"{\"decommissioningMaxPercentOfMaxSegmentsToMove\": -1}",
|
||||
errorMsg
|
||||
);
|
||||
assertThatDeserializationFailsWithMessage(
|
||||
"{\"decommissioningMaxPercentOfMaxSegmentsToMove\": 105}",
|
||||
errorMsg
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -486,7 +442,6 @@ public class CoordinatorDynamicConfigTest
|
||||
+ " \"killDataSourceWhitelist\": [\"test1\",\"test2\"],\n"
|
||||
+ " \"maxSegmentsInNodeLoadingQueue\": 1,\n"
|
||||
+ " \"decommissioningNodes\": [\"host1\", \"host2\"],\n"
|
||||
+ " \"decommissioningMaxPercentOfMaxSegmentsToMove\": 9,\n"
|
||||
+ " \"pauseCoordination\": false\n"
|
||||
+ "}\n";
|
||||
CoordinatorDynamicConfig actual = mapper.readValue(
|
||||
@ -512,10 +467,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
1,
|
||||
decommissioning,
|
||||
9,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -556,10 +509,8 @@ public class CoordinatorDynamicConfigTest
|
||||
true,
|
||||
1,
|
||||
ImmutableSet.of(),
|
||||
0,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
|
||||
// killAllDataSources is a config in versions 0.22.x and older and is no longer used.
|
||||
@ -611,10 +562,8 @@ public class CoordinatorDynamicConfigTest
|
||||
true,
|
||||
EXPECTED_DEFAULT_MAX_SEGMENTS_IN_NODE_LOADING_QUEUE,
|
||||
ImmutableSet.of(),
|
||||
0,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -638,10 +587,8 @@ public class CoordinatorDynamicConfigTest
|
||||
true,
|
||||
EXPECTED_DEFAULT_MAX_SEGMENTS_IN_NODE_LOADING_QUEUE,
|
||||
emptyList,
|
||||
70,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -668,10 +615,8 @@ public class CoordinatorDynamicConfigTest
|
||||
false,
|
||||
EXPECTED_DEFAULT_MAX_SEGMENTS_IN_NODE_LOADING_QUEUE,
|
||||
ImmutableSet.of(),
|
||||
70,
|
||||
false,
|
||||
false,
|
||||
Integer.MAX_VALUE
|
||||
false
|
||||
);
|
||||
}
|
||||
|
||||
@ -703,22 +648,11 @@ public class CoordinatorDynamicConfigTest
|
||||
null,
|
||||
null,
|
||||
null,
|
||||
null,
|
||||
null,
|
||||
null
|
||||
).build(current)
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testSerdeHandleInvalidMaxNonPrimaryReplicantsToLoad()
|
||||
{
|
||||
assertThatDeserializationFailsWithMessage(
|
||||
"{\"maxNonPrimaryReplicantsToLoad\": -1}",
|
||||
"maxNonPrimaryReplicantsToLoad must be greater than or equal to 0."
|
||||
);
|
||||
}
|
||||
|
||||
private void assertThatDeserializationFailsWithMessage(String json, String message)
|
||||
{
|
||||
JsonMappingException e = Assert.assertThrows(
|
||||
@ -759,10 +693,8 @@ public class CoordinatorDynamicConfigTest
|
||||
boolean expectedKillUnusedSegmentsInAllDataSources,
|
||||
int expectedMaxSegmentsInNodeLoadingQueue,
|
||||
Set<String> decommissioningNodes,
|
||||
int decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
boolean pauseCoordination,
|
||||
boolean replicateAfterLoadTimeout,
|
||||
int maxNonPrimaryReplicantsToLoad
|
||||
boolean replicateAfterLoadTimeout
|
||||
)
|
||||
{
|
||||
Assert.assertEquals(
|
||||
@ -784,13 +716,8 @@ public class CoordinatorDynamicConfigTest
|
||||
Assert.assertEquals((int) expectedMaxKillTaskSlots, config.getMaxKillTaskSlots());
|
||||
Assert.assertEquals(expectedMaxSegmentsInNodeLoadingQueue, config.getMaxSegmentsInNodeLoadingQueue());
|
||||
Assert.assertEquals(decommissioningNodes, config.getDecommissioningNodes());
|
||||
Assert.assertEquals(
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove,
|
||||
config.getDecommissioningMaxPercentOfMaxSegmentsToMove()
|
||||
);
|
||||
Assert.assertEquals(pauseCoordination, config.getPauseCoordination());
|
||||
Assert.assertEquals(replicateAfterLoadTimeout, config.getReplicateAfterLoadTimeout());
|
||||
Assert.assertEquals(maxNonPrimaryReplicantsToLoad, config.getMaxNonPrimaryReplicantsToLoad());
|
||||
}
|
||||
|
||||
private static int getDefaultNumBalancerThreads()
|
||||
|
@ -32,10 +32,8 @@ export const DEFAULT_COORDINATOR_DYNAMIC_CONFIG: CoordinatorDynamicConfig = {
|
||||
killPendingSegmentsSkipList: [],
|
||||
maxSegmentsInNodeLoadingQueue: 500,
|
||||
decommissioningNodes: [],
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove: 70,
|
||||
pauseCoordination: false,
|
||||
replicateAfterLoadTimeout: false,
|
||||
maxNonPrimaryReplicantsToLoad: 2147483647,
|
||||
useRoundRobinSegmentAssignment: true,
|
||||
smartSegmentLoading: true,
|
||||
debugDimensions: null,
|
||||
|
@ -38,9 +38,7 @@ export interface CoordinatorDynamicConfig {
|
||||
replicantLifetime?: number;
|
||||
replicationThrottleLimit?: number;
|
||||
decommissioningNodes?: string[];
|
||||
decommissioningMaxPercentOfMaxSegmentsToMove?: number;
|
||||
pauseCoordination?: boolean;
|
||||
maxNonPrimaryReplicantsToLoad?: number;
|
||||
replicateAfterLoadTimeout?: boolean;
|
||||
useRoundRobinSegmentAssignment?: boolean;
|
||||
smartSegmentLoading?: boolean;
|
||||
@ -146,50 +144,6 @@ export const COORDINATOR_DYNAMIC_CONFIG_FIELDS: Field<CoordinatorDynamicConfig>[
|
||||
</>
|
||||
),
|
||||
},
|
||||
{
|
||||
name: 'maxNonPrimaryReplicantsToLoad',
|
||||
type: 'number',
|
||||
defaultValue: 2147483647,
|
||||
defined: cdc => (cdc.smartSegmentLoading === false ? true : undefined),
|
||||
info: (
|
||||
<>
|
||||
The maximum number of non-primary replicants to load in a single Coordinator cycle. Once
|
||||
this limit is hit, only primary replicants will be loaded for the remainder of the cycle.
|
||||
Tuning this value lower can help reduce the delay in loading primary segments when the
|
||||
cluster has a very large number of non-primary replicants to load (such as when a single
|
||||
historical drops out of the cluster leaving many under-replicated segments).
|
||||
</>
|
||||
),
|
||||
},
|
||||
{
|
||||
name: 'decommissioningMaxPercentOfMaxSegmentsToMove',
|
||||
type: 'number',
|
||||
defaultValue: 70,
|
||||
defined: cdc => (cdc.smartSegmentLoading === false ? true : undefined),
|
||||
info: (
|
||||
<>
|
||||
<p>
|
||||
Upper limit of segments the Coordinator can move from decommissioning servers to active
|
||||
non-decommissioning servers during a single run. This value is relative to the total
|
||||
maximum number of segments that can be moved at any given time based upon the value of
|
||||
<Code>maxSegmentsToMove</Code>.
|
||||
</p>
|
||||
<p>
|
||||
If <Code>decommissioningMaxPercentOfMaxSegmentsToMove</Code> is 0, the Coordinator does
|
||||
not move segments to decommissioning servers, effectively putting them in a type of
|
||||
"maintenance" mode. In this case, decommissioning servers do not participate in
|
||||
balancing or assignment by load rules. The Coordinator still considers segments on
|
||||
decommissioning servers as candidates to replicate on active servers.
|
||||
</p>
|
||||
<p>
|
||||
Decommissioning can stall if there are no available active servers to move the segments
|
||||
to. You can use the maximum percent of decommissioning segment movements to prioritize
|
||||
balancing or to decrease commissioning time to prevent active servers from being
|
||||
overloaded. The value must be between 0 and 100.
|
||||
</p>
|
||||
</>
|
||||
),
|
||||
},
|
||||
|
||||
// End "smart" segment loading section
|
||||
|
||||
@ -202,7 +156,7 @@ export const COORDINATOR_DYNAMIC_CONFIG_FIELDS: Field<CoordinatorDynamicConfig>[
|
||||
List of historical services to 'decommission'. Coordinator will not assign new
|
||||
segments to 'decommissioning' services, and segments will be moved away from them
|
||||
to be placed on non-decommissioning services at the maximum rate specified by{' '}
|
||||
<Code>decommissioningMaxPercentOfMaxSegmentsToMove</Code>.
|
||||
<Code>maxSegmentsToMove</Code>.
|
||||
</>
|
||||
),
|
||||
},
|
||||
|
Loading…
x
Reference in New Issue
Block a user