@InterfaceAudience.LimitedPrivate(value="Configuration") public class StochasticLoadBalancer extends BaseLoadBalancer
This is a best effort load balancer. Given a Cost function F(C) => x It will randomly try and mutate the cluster to Cprime. If F(Cprime) < F(C) then the new cluster state becomes the plan. It includes costs functions to compute the cost of:
Every cost function returns a number between 0 and 1 inclusive; where 0 is the lowest cost best solution, and 1 is the highest possible cost and the worst solution. The computed costs are scaled by their respective multipliers:
You can also add custom Cost function by setting the the following configuration value:
All custom Cost Functions needs to extends CostFunction
In addition to the above configurations, the balancer can be tuned by the following configuration values:
This balancer is best used with hbase.master.loadbalance.bytable set to false so that the balancer gets the full picture of all loads on the cluster.
Modifier and Type | Class and Description |
---|---|
static class |
StochasticLoadBalancer.GeneratorType |
BALANCER_DECISION_BUFFER_ENABLED, BALANCER_REJECTION_BUFFER_ENABLED, clusterStatus, DEFAULT_BALANCER_DECISION_BUFFER_ENABLED, DEFAULT_BALANCER_REJECTION_BUFFER_ENABLED, DEFAULT_HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable, masterServerName, metricsBalancer, MIN_SERVER_BALANCE, onlySystemTablesOnMaster, rackManager, regionFinder, services, slop, useRegionFinder
BOGUS_SERVER_NAME, SYSTEM_TABLES_ON_MASTER, TABLES_ON_MASTER
Constructor and Description |
---|
StochasticLoadBalancer()
The constructor that pass a MetricsStochasticBalancer to BaseLoadBalancer to replace its
default MetricsBalancer
|
StochasticLoadBalancer(MetricsStochasticBalancer metricsStochasticBalancer) |
Modifier and Type | Method and Description |
---|---|
private void |
addCostFunction(CostFunction costFunction) |
private boolean |
areSomeRegionReplicasColocated(BalancerClusterState c) |
protected List<RegionPlan> |
balanceTable(TableName tableName,
Map<ServerName,List<RegionInfo>> loadOfOneTable)
Given the cluster state this will try and approach an optimal balance.
|
private long |
calculateMaxSteps(BalancerClusterState cluster) |
(package private) static String |
composeAttributeName(String tableName,
String costFunctionName)
A helper function to compose the attribute name from tablename and costfunction name
|
(package private) double |
computeCost(BalancerClusterState cluster,
double previousCost)
This is the main cost function.
|
protected List<CandidateGenerator> |
createCandidateGenerators() |
private static CostFunction |
createCostFunction(Class<? extends CostFunction> clazz,
org.apache.hadoop.conf.Configuration conf) |
private List<RegionPlan> |
createRegionPlans(BalancerClusterState cluster)
Create all of the RegionPlan's needed to move from the initial cluster state to the desired
state.
|
private String |
functionCost() |
(package private) List<CandidateGenerator> |
getCandidateGenerators() |
(package private) String[] |
getCostFunctionNames()
Get the names of the cost functions
|
protected CandidateGenerator |
getRandomGenerator()
Select the candidate generator to use based on the cost of cost functions.
|
(package private) void |
initCosts(BalancerClusterState cluster) |
protected void |
loadConf(org.apache.hadoop.conf.Configuration conf) |
private void |
loadCustomCostFunctions(org.apache.hadoop.conf.Configuration conf) |
(package private) boolean |
needsBalance(TableName tableName,
BalancerClusterState cluster) |
(package private) BalanceAction |
nextAction(BalancerClusterState cluster) |
private void |
sendRegionPlansToRingBuffer(List<RegionPlan> plans,
double currentCost,
double initCost,
String initFunctionTotalCosts,
long step) |
private void |
sendRejectionReasonToRingBuffer(String reason,
List<CostFunction> costFunctions) |
(package private) void |
setRackManager(RackManager rackManager) |
private String |
totalCostsPerFunc() |
void |
updateBalancerLoadInfo(Map<TableName,Map<ServerName,List<RegionInfo>>> loadOfAllTable)
In some scenarios, Balancer needs to update internal status or information according to the
current tables load
|
private void |
updateBalancerTableLoadInfo(TableName tableName,
Map<ServerName,List<RegionInfo>> loadOfOneTable) |
void |
updateClusterMetrics(ClusterMetrics st)
Set the current cluster status.
|
(package private) void |
updateCostsAndWeightsWithAction(BalancerClusterState cluster,
BalanceAction action) |
(package private) void |
updateMetricsSize(int size)
Update the number of metrics that are reported to JMX
|
private void |
updateRegionLoad()
Store the current region loads.
|
private void |
updateStochasticCosts(TableName tableName,
double overall,
double[] subCosts)
update costs to JMX
|
assignMasterSystemRegions, balanceCluster, balanceMasterRegions, getConf, getDefaultSlop, idleRegionServerExist, initialize, isStopped, onConfigurationChange, postMasterStartupInitialize, preBalanceCluster, randomAssignment, regionOffline, regionOnline, retainAssignment, roundRobinAssignment, setMasterServices, shouldBeOnMaster, sloppyRegionServerExist, stop, toEnsumbleTableLoad, updateBalancerStatus
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
isMasterCanHostUserRegions, isSystemTablesOnlyOnMaster, isTablesOnMaster
private static final org.slf4j.Logger LOG
protected static final String STEPS_PER_REGION_KEY
protected static final int DEFAULT_STEPS_PER_REGION
protected static final String MAX_STEPS_KEY
protected static final int DEFAULT_MAX_STEPS
protected static final String RUN_MAX_STEPS_KEY
protected static final boolean DEFAULT_RUN_MAX_STEPS
protected static final String MAX_RUNNING_TIME_KEY
protected static final long DEFAULT_MAX_RUNNING_TIME
protected static final String KEEP_REGION_LOADS
protected static final int DEFAULT_KEEP_REGION_LOADS
private static final String TABLE_FUNCTION_SEP
protected static final String MIN_COST_NEED_BALANCE_KEY
protected static final float DEFAULT_MIN_COST_NEED_BALANCE
protected static final String COST_FUNCTIONS_COST_FUNCTIONS_KEY
public static final String OVERALL_COST_FUNCTION_NAME
Map<String,Deque<BalancerRegionLoad>> loads
private int maxSteps
private boolean runMaxSteps
private int stepsPerRegion
private long maxRunningTime
private int numRegionLoadsToRemember
private float minCostNeedBalance
private boolean isBalancerDecisionRecording
private boolean isBalancerRejectionRecording
protected List<CandidateGenerator> candidateGenerators
private double[] weightsOfGenerators
private List<CostFunction> costFunctions
private float sumMultiplier
private double curOverallCost
private double[] tempFunctionCosts
private double[] curFunctionCosts
private LocalityBasedCandidateGenerator localityCandidateGenerator
private ServerLocalityCostFunction localityCost
private RackLocalityCostFunction rackLocalityCost
private RegionReplicaHostCostFunction regionReplicaHostCostFunction
private RegionReplicaRackCostFunction regionReplicaRackCostFunction
NamedQueueRecorder namedQueueRecorder
public StochasticLoadBalancer()
public StochasticLoadBalancer(MetricsStochasticBalancer metricsStochasticBalancer)
private static CostFunction createCostFunction(Class<? extends CostFunction> clazz, org.apache.hadoop.conf.Configuration conf)
private void loadCustomCostFunctions(org.apache.hadoop.conf.Configuration conf)
List<CandidateGenerator> getCandidateGenerators()
protected List<CandidateGenerator> createCandidateGenerators()
protected void loadConf(org.apache.hadoop.conf.Configuration conf)
loadConf
in class BaseLoadBalancer
public void updateClusterMetrics(ClusterMetrics st)
LoadBalancer
updateClusterMetrics
in interface LoadBalancer
updateClusterMetrics
in class BaseLoadBalancer
private void updateBalancerTableLoadInfo(TableName tableName, Map<ServerName,List<RegionInfo>> loadOfOneTable)
public void updateBalancerLoadInfo(Map<TableName,Map<ServerName,List<RegionInfo>>> loadOfAllTable)
LoadBalancer
loadOfAllTable
- region load of servers for all tablevoid updateMetricsSize(int size)
private boolean areSomeRegionReplicasColocated(BalancerClusterState c)
boolean needsBalance(TableName tableName, BalancerClusterState cluster)
BalanceAction nextAction(BalancerClusterState cluster)
protected CandidateGenerator getRandomGenerator()
void setRackManager(RackManager rackManager)
private long calculateMaxSteps(BalancerClusterState cluster)
protected List<RegionPlan> balanceTable(TableName tableName, Map<ServerName,List<RegionInfo>> loadOfOneTable)
balanceTable
in class BaseLoadBalancer
tableName
- the table to be balancedloadOfOneTable
- region load of servers for the specific one tableprivate void sendRejectionReasonToRingBuffer(String reason, List<CostFunction> costFunctions)
private void sendRegionPlansToRingBuffer(List<RegionPlan> plans, double currentCost, double initCost, String initFunctionTotalCosts, long step)
private void updateStochasticCosts(TableName tableName, double overall, double[] subCosts)
private void addCostFunction(CostFunction costFunction)
private String functionCost()
private String totalCostsPerFunc()
private List<RegionPlan> createRegionPlans(BalancerClusterState cluster)
cluster
- The state of the clusterprivate void updateRegionLoad()
void initCosts(BalancerClusterState cluster)
void updateCostsAndWeightsWithAction(BalancerClusterState cluster, BalanceAction action)
String[] getCostFunctionNames()
double computeCost(BalancerClusterState cluster, double previousCost)
cluster
- The state of the clusterpreviousCost
- the previous cost. This is used as an early out.static String composeAttributeName(String tableName, String costFunctionName)
Copyright © 2007–2020 The Apache Software Foundation. All rights reserved.