001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.master.balancer;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertFalse;
022import static org.junit.Assert.assertNotNull;
023import static org.junit.Assert.assertNull;
024import static org.junit.Assert.assertTrue;
025import static org.mockito.Mockito.mock;
026import static org.mockito.Mockito.when;
027
028import java.util.ArrayList;
029import java.util.Arrays;
030import java.util.List;
031import java.util.Map;
032import java.util.Queue;
033import java.util.TimeZone;
034import java.util.TreeMap;
035import org.apache.hadoop.conf.Configuration;
036import org.apache.hadoop.hbase.ClusterMetrics;
037import org.apache.hadoop.hbase.HBaseClassTestRule;
038import org.apache.hadoop.hbase.HBaseConfiguration;
039import org.apache.hadoop.hbase.HConstants;
040import org.apache.hadoop.hbase.RegionMetrics;
041import org.apache.hadoop.hbase.ServerMetrics;
042import org.apache.hadoop.hbase.ServerName;
043import org.apache.hadoop.hbase.Size;
044import org.apache.hadoop.hbase.TableName;
045import org.apache.hadoop.hbase.client.RegionInfo;
046import org.apache.hadoop.hbase.master.RegionPlan;
047import org.apache.hadoop.hbase.testclassification.MasterTests;
048import org.apache.hadoop.hbase.testclassification.MediumTests;
049import org.apache.hadoop.hbase.util.Bytes;
050import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
051import org.junit.Before;
052import org.junit.BeforeClass;
053import org.junit.ClassRule;
054import org.junit.Test;
055import org.junit.experimental.categories.Category;
056
057import org.apache.hbase.thirdparty.org.apache.commons.collections4.CollectionUtils;
058
059@Category({ MasterTests.class, MediumTests.class })
060public class TestStochasticLoadBalancer extends BalancerTestBase {
061
062  @ClassRule
063  public static final HBaseClassTestRule CLASS_RULE =
064    HBaseClassTestRule.forClass(TestStochasticLoadBalancer.class);
065
066  private static final String REGION_KEY = "testRegion";
067
068  // Mapping of locality test -> expected locality
069  private float[] expectedLocalities = { 1.0f, 0.0f, 0.50f, 0.25f, 1.0f };
070  private static Configuration storedConfiguration;
071
072  @BeforeClass
073  public static void saveInitialConfiguration() {
074    storedConfiguration = new Configuration(conf);
075  }
076
077  @Before
078  public void beforeEachTest() {
079    conf = new Configuration(storedConfiguration);
080    loadBalancer.onConfigurationChange(conf);
081  }
082
083  /**
084   * Data set for testLocalityCost: [test][0][0] = mapping of server to number of regions it hosts
085   * [test][region + 1][0] = server that region is hosted on [test][region + 1][server + 1] =
086   * locality for region on server
087   */
088  private int[][][] clusterRegionLocationMocks = new int[][][] {
089    // Test 1: each region is entirely on server that hosts it
090    new int[][] { new int[] { 2, 1, 1 }, new int[] { 2, 0, 0, 100 }, // region 0 is hosted and
091                                                                     // entirely local on server 2
092      new int[] { 0, 100, 0, 0 }, // region 1 is hosted and entirely on server 0
093      new int[] { 0, 100, 0, 0 }, // region 2 is hosted and entirely on server 0
094      new int[] { 1, 0, 100, 0 }, // region 3 is hosted and entirely on server 1
095    },
096
097    // Test 2: each region is 0% local on the server that hosts it
098    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 0, 0, 100 }, // region 0 is hosted and
099                                                                     // entirely local on server 2
100      new int[] { 1, 100, 0, 0 }, // region 1 is hosted and entirely on server 0
101      new int[] { 1, 100, 0, 0 }, // region 2 is hosted and entirely on server 0
102      new int[] { 2, 0, 100, 0 }, // region 3 is hosted and entirely on server 1
103    },
104
105    // Test 3: each region is 25% local on the server that hosts it (and 50% locality is possible)
106    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 25, 0, 50 }, // region 0 is hosted and
107                                                                     // entirely local on server 2
108      new int[] { 1, 50, 25, 0 }, // region 1 is hosted and entirely on server 0
109      new int[] { 1, 50, 25, 0 }, // region 2 is hosted and entirely on server 0
110      new int[] { 2, 0, 50, 25 }, // region 3 is hosted and entirely on server 1
111    },
112
113    // Test 4: each region is 25% local on the server that hosts it (and 100% locality is possible)
114    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 25, 0, 100 }, // region 0 is hosted and
115                                                                      // entirely local on server 2
116      new int[] { 1, 100, 25, 0 }, // region 1 is hosted and entirely on server 0
117      new int[] { 1, 100, 25, 0 }, // region 2 is hosted and entirely on server 0
118      new int[] { 2, 0, 100, 25 }, // region 3 is hosted and entirely on server 1
119    },
120
121    // Test 5: each region is 75% local on the server that hosts it (and 75% locality is possible
122    // everywhere)
123    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 75, 75, 75 }, // region 0 is hosted and
124                                                                      // entirely local on server 2
125      new int[] { 1, 75, 75, 75 }, // region 1 is hosted and entirely on server 0
126      new int[] { 1, 75, 75, 75 }, // region 2 is hosted and entirely on server 0
127      new int[] { 2, 75, 75, 75 }, // region 3 is hosted and entirely on server 1
128    }, };
129
130  @Test
131  public void testKeepRegionLoad() throws Exception {
132    ServerName sn = ServerName.valueOf("test:8080", 100);
133    int numClusterStatusToAdd = 20000;
134    for (int i = 0; i < numClusterStatusToAdd; i++) {
135      ServerMetrics sl = mock(ServerMetrics.class);
136
137      RegionMetrics rl = mock(RegionMetrics.class);
138      when(rl.getReadRequestCount()).thenReturn(0L);
139      when(rl.getWriteRequestCount()).thenReturn(0L);
140      when(rl.getMemStoreSize()).thenReturn(Size.ZERO);
141      when(rl.getStoreFileSize()).thenReturn(new Size(i, Size.Unit.MEGABYTE));
142
143      Map<byte[], RegionMetrics> regionLoadMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
144      regionLoadMap.put(Bytes.toBytes(REGION_KEY), rl);
145      when(sl.getRegionMetrics()).thenReturn(regionLoadMap);
146
147      ClusterMetrics clusterStatus = mock(ClusterMetrics.class);
148      Map<ServerName, ServerMetrics> serverMetricsMap = new TreeMap<>();
149      serverMetricsMap.put(sn, sl);
150      when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
151      // when(clusterStatus.getLoad(sn)).thenReturn(sl);
152
153      loadBalancer.updateClusterMetrics(clusterStatus);
154    }
155
156    String regionNameAsString = RegionInfo.getRegionNameAsString(Bytes.toBytes(REGION_KEY));
157    assertTrue(loadBalancer.loads.get(regionNameAsString) != null);
158    assertTrue(loadBalancer.loads.get(regionNameAsString).size() == 15);
159    assertNull(loadBalancer.namedQueueRecorder);
160
161    Queue<BalancerRegionLoad> loads = loadBalancer.loads.get(regionNameAsString);
162    int i = 0;
163    while (loads.size() > 0) {
164      BalancerRegionLoad rl = loads.remove();
165      assertEquals(i + (numClusterStatusToAdd - 15), rl.getStorefileSizeMB());
166      i++;
167    }
168  }
169
170  @Test
171  public void testUpdateBalancerLoadInfo() {
172    int[] cluster = new int[] { 10, 0 };
173    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
174    BalancerClusterState clusterState = mockCluster(cluster);
175    Map<TableName, Map<ServerName, List<RegionInfo>>> LoadOfAllTable =
176      (Map) mockClusterServersWithTables(servers);
177    boolean[] perTableBalancerConfigs = { true, false };
178    for (boolean isByTable : perTableBalancerConfigs) {
179      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
180      loadBalancer.onConfigurationChange(conf);
181      dummyMetricsStochasticBalancer.clearDummyMetrics();
182      loadBalancer.updateBalancerLoadInfo(LoadOfAllTable);
183      assertTrue("Metrics should be recorded!",
184        dummyMetricsStochasticBalancer.getDummyCostsMap() != null
185          && !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
186
187      String metricRecordKey;
188      if (isByTable) {
189        metricRecordKey = "table1#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
190      } else {
191        metricRecordKey =
192          HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
193      }
194      double curOverallCost = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
195      double curOverallCostInMetrics =
196        dummyMetricsStochasticBalancer.getDummyCostsMap().get(metricRecordKey);
197      assertEquals(curOverallCost, curOverallCostInMetrics, 0.001);
198    }
199  }
200
201  @Test
202  public void testUpdateStochasticCosts() {
203    int[] cluster = new int[] { 10, 0 };
204    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
205    BalancerClusterState clusterState = mockCluster(cluster);
206    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
207    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
208    loadBalancer.onConfigurationChange(conf);
209    dummyMetricsStochasticBalancer.clearDummyMetrics();
210    List<RegionPlan> plans =
211      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
212
213    assertTrue("Balance plan should not be empty!", plans != null && !plans.isEmpty());
214    assertTrue("There should be metrics record in MetricsStochasticBalancer",
215      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
216
217    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
218    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
219      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
220    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
221  }
222
223  @Test
224  public void testUpdateStochasticCostsIfBalanceNotRan() {
225    int[] cluster = new int[] { 10, 10 };
226    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
227    BalancerClusterState clusterState = mockCluster(cluster);
228    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", Float.MAX_VALUE);
229    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
230    loadBalancer.onConfigurationChange(conf);
231    dummyMetricsStochasticBalancer.clearDummyMetrics();
232    List<RegionPlan> plans =
233      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
234
235    assertTrue("Balance plan should be empty!", plans == null || plans.isEmpty());
236    assertTrue("There should be metrics record in MetricsStochasticBalancer!",
237      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
238
239    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
240    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
241      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
242    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
243  }
244
245  @Test
246  public void testNeedBalance() {
247    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
248    conf.setFloat(HConstants.LOAD_BALANCER_SLOP_KEY, -1f);
249    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
250    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
251    // Test with/without per table balancer.
252    boolean[] perTableBalancerConfigs = { true, false };
253    for (boolean isByTable : perTableBalancerConfigs) {
254      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
255      loadBalancer.onConfigurationChange(conf);
256      for (int[] mockCluster : clusterStateMocks) {
257        assertTrue(hasEmptyBalancerPlans(mockCluster) || needsBalanceIdleRegion(mockCluster));
258      }
259    }
260  }
261
262  @Test
263  public void testBalanceOfSloppyServers() {
264    // We are testing slop checks, so don't "accidentally" balance due to a minCost calculation.
265    // During development, imbalance of a 100 server cluster, with one node having 10 regions
266    // and the rest having 5, is 0.0048. With minCostNeedBalance default of 0.025, test should
267    // validate slop checks without this override. We override just to ensure we will always
268    // validate slop check here, and for small clusters as well.
269    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
270    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
271    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
272    loadBalancer.onConfigurationChange(conf);
273    for (int[] mockCluster : clusterStateMocksWithNoSlop) {
274      assertTrue(hasEmptyBalancerPlans(mockCluster));
275    }
276    for (int[] mockCluster : clusterStateMocksWithSlop) {
277      assertFalse(hasEmptyBalancerPlans(mockCluster));
278    }
279  }
280
281  @Test
282  public void testSloppyTablesLoadBalanceByTable() {
283    int[][] regionsPerServerPerTable = new int[][] {
284      new int[] { 8, 2, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5,
285        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 },
286      new int[] { 2, 8, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5,
287        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 }, };
288    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
289    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
290    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
291    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, true);
292    loadBalancer.onConfigurationChange(conf);
293    assertFalse(hasEmptyBalancerPlans(regionsPerServerPerTable));
294  }
295
296  private boolean hasEmptyBalancerPlans(int[] mockCluster) {
297    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
298    return hasEmptyBalancerPlans(servers);
299  }
300
301  private boolean hasEmptyBalancerPlans(int[][] mockCluster) {
302    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
303    return hasEmptyBalancerPlans(servers);
304  }
305
306  private boolean hasEmptyBalancerPlans(Map<ServerName, List<RegionInfo>> servers) {
307    Map<TableName, Map<ServerName, List<RegionInfo>>> loadOfAllTable =
308      (Map) mockClusterServersWithTables(servers);
309    List<RegionPlan> plans = loadBalancer.balanceCluster(loadOfAllTable);
310    return plans == null || plans.isEmpty();
311  }
312
313  @Test
314  public void testLocalityCost() throws Exception {
315    Configuration conf = HBaseConfiguration.create();
316    CostFunction costFunction = new ServerLocalityCostFunction(conf);
317
318    for (int test = 0; test < clusterRegionLocationMocks.length; test++) {
319      int[][] clusterRegionLocations = clusterRegionLocationMocks[test];
320      MockCluster cluster = new MockCluster(clusterRegionLocations);
321      costFunction.prepare(cluster);
322      double cost = costFunction.cost();
323      double expected = 1 - expectedLocalities[test];
324      assertEquals(expected, cost, 0.001);
325    }
326    assertNull(loadBalancer.namedQueueRecorder);
327  }
328
329  @Test
330  public void testMoveCostMultiplier() throws Exception {
331    Configuration conf = HBaseConfiguration.create();
332    CostFunction costFunction = new MoveCostFunction(conf);
333    BalancerClusterState cluster = mockCluster(clusterStateMocks[0]);
334    costFunction.prepare(cluster);
335    costFunction.cost();
336    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST, costFunction.getMultiplier(), 0.01);
337
338    // In offpeak hours, the multiplier of move cost should be lower
339    conf.setInt("hbase.offpeak.start.hour", 0);
340    conf.setInt("hbase.offpeak.end.hour", 23);
341    // Set a fixed time which hour is 15, so it will always in offpeak
342    // See HBASE-24898 for more info of the calculation here
343    long deltaFor15 = TimeZone.getDefault().getRawOffset() - 28800000;
344    long timeFor15 = 1597907081000L - deltaFor15;
345    EnvironmentEdgeManager.injectEdge(() -> timeFor15);
346    costFunction = new MoveCostFunction(conf);
347    costFunction.prepare(cluster);
348    costFunction.cost();
349    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST_OFFPEAK, costFunction.getMultiplier(), 0.01);
350  }
351
352  @Test
353  public void testMoveCost() throws Exception {
354    Configuration conf = HBaseConfiguration.create();
355    CostFunction costFunction = new MoveCostFunction(conf);
356    for (int[] mockCluster : clusterStateMocks) {
357      BalancerClusterState cluster = mockCluster(mockCluster);
358      costFunction.prepare(cluster);
359      double cost = costFunction.cost();
360      assertEquals(0.0f, cost, 0.001);
361
362      // cluster region number is smaller than maxMoves=600
363      cluster.setNumRegions(200);
364      cluster.setNumMovedRegions(10);
365      cost = costFunction.cost();
366      assertEquals(0.05f, cost, 0.001);
367      cluster.setNumMovedRegions(100);
368      cost = costFunction.cost();
369      assertEquals(0.5f, cost, 0.001);
370      cluster.setNumMovedRegions(200);
371      cost = costFunction.cost();
372      assertEquals(1.0f, cost, 0.001);
373
374      // cluster region number is bigger than maxMoves=2500
375      cluster.setNumRegions(10000);
376      cluster.setNumMovedRegions(250);
377      cost = costFunction.cost();
378      assertEquals(0.025f, cost, 0.001);
379      cluster.setNumMovedRegions(1250);
380      cost = costFunction.cost();
381      assertEquals(0.125f, cost, 0.001);
382      cluster.setNumMovedRegions(2500);
383      cost = costFunction.cost();
384      assertEquals(0.25f, cost, 0.01);
385    }
386  }
387
388  @Test
389  public void testSkewCost() {
390    Configuration conf = HBaseConfiguration.create();
391    CostFunction costFunction = new RegionCountSkewCostFunction(conf);
392    for (int[] mockCluster : clusterStateMocks) {
393      costFunction.prepare(mockCluster(mockCluster));
394      double cost = costFunction.cost();
395      assertTrue(cost >= 0);
396      assertTrue(cost <= 1.01);
397    }
398
399    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 0, 1 }));
400    assertEquals(0, costFunction.cost(), 0.01);
401    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 1, 1 }));
402    assertEquals(0, costFunction.cost(), 0.01);
403    costFunction.prepare(mockCluster(new int[] { 0, 0, 1, 1, 1 }));
404    assertEquals(0, costFunction.cost(), 0.01);
405    costFunction.prepare(mockCluster(new int[] { 0, 1, 1, 1, 1 }));
406    assertEquals(0, costFunction.cost(), 0.01);
407    costFunction.prepare(mockCluster(new int[] { 1, 1, 1, 1, 1 }));
408    assertEquals(0, costFunction.cost(), 0.01);
409    costFunction.prepare(mockCluster(new int[] { 10000, 0, 0, 0, 0 }));
410    assertEquals(1, costFunction.cost(), 0.01);
411  }
412
413  @Test
414  public void testCostAfterUndoAction() {
415    final int runs = 10;
416    for (int[] mockCluster : clusterStateMocks) {
417      BalancerClusterState cluster = mockCluster(mockCluster);
418      loadBalancer.initCosts(cluster);
419      for (int i = 0; i != runs; ++i) {
420        final double expectedCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
421        BalanceAction action = loadBalancer.nextAction(cluster);
422        cluster.doAction(action);
423        loadBalancer.updateCostsAndWeightsWithAction(cluster, action);
424        BalanceAction undoAction = action.undoAction();
425        cluster.doAction(undoAction);
426        loadBalancer.updateCostsAndWeightsWithAction(cluster, undoAction);
427        final double actualCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
428        assertEquals(expectedCost, actualCost, 0);
429      }
430    }
431  }
432
433  @Test
434  public void testTableSkewCost() {
435    Configuration conf = HBaseConfiguration.create();
436    CostFunction costFunction = new TableSkewCostFunction(conf);
437    for (int[] mockCluster : clusterStateMocks) {
438      BalancerClusterState cluster = mockCluster(mockCluster);
439      costFunction.prepare(cluster);
440      double cost = costFunction.cost();
441      assertTrue(cost >= 0);
442      assertTrue(cost <= 1.01);
443    }
444  }
445
446  @Test
447  public void testRegionLoadCost() {
448    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
449    for (int i = 1; i < 5; i++) {
450      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
451      when(regionLoad.getReadRequestsCount()).thenReturn(new Long(i));
452      when(regionLoad.getStorefileSizeMB()).thenReturn(i);
453      regionLoads.add(regionLoad);
454    }
455
456    Configuration conf = HBaseConfiguration.create();
457    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
458    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
459    // read requests are treated as a rate so the average rate here is simply 1
460    assertEquals(1, rateResult, 0.01);
461
462    StoreFileCostFunction storeFileCostFunction = new StoreFileCostFunction(conf);
463    double result = storeFileCostFunction.getRegionLoadCost(regionLoads);
464    // storefile size cost is simply an average of it's value over time
465    assertEquals(2.5, result, 0.01);
466  }
467
468  @Test
469  public void testRegionLoadCostWhenDecrease() {
470    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
471    // test region loads of [1,2,1,4]
472    for (int i = 1; i < 5; i++) {
473      int load = i == 3 ? 1 : i;
474      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
475      when(regionLoad.getReadRequestsCount()).thenReturn((long) load);
476      regionLoads.add(regionLoad);
477    }
478
479    Configuration conf = HBaseConfiguration.create();
480    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
481    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
482    // read requests are treated as a rate so the average rate here is simply 1
483    assertEquals(1.67, rateResult, 0.01);
484  }
485
486  @Test
487  public void testLosingRs() throws Exception {
488    int numNodes = 3;
489    int numRegions = 20;
490    int numRegionsPerServer = 3; // all servers except one
491    int replication = 1;
492    int numTables = 2;
493
494    Map<ServerName, List<RegionInfo>> serverMap =
495      createServerMap(numNodes, numRegions, numRegionsPerServer, replication, numTables);
496    List<ServerAndLoad> list = convertToList(serverMap);
497
498    List<RegionPlan> plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
499    assertNotNull(plans);
500
501    // Apply the plan to the mock cluster.
502    List<ServerAndLoad> balancedCluster = reconcile(list, plans, serverMap);
503
504    assertClusterAsBalanced(balancedCluster);
505
506    ServerName sn = serverMap.keySet().toArray(new ServerName[serverMap.size()])[0];
507
508    ServerName deadSn = ServerName.valueOf(sn.getHostname(), sn.getPort(), sn.getStartcode() - 100);
509
510    serverMap.put(deadSn, new ArrayList<>(0));
511
512    plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
513    assertNull(plans);
514  }
515
516  @Test
517  public void testAdditionalCostFunction() {
518    conf.set(StochasticLoadBalancer.COST_FUNCTIONS_COST_FUNCTIONS_KEY,
519      DummyCostFunction.class.getName());
520
521    loadBalancer.onConfigurationChange(conf);
522    assertTrue(Arrays.asList(loadBalancer.getCostFunctionNames())
523      .contains(DummyCostFunction.class.getSimpleName()));
524  }
525
526  @Test
527  public void testDefaultCostFunctionList() {
528    List<String> expected = Arrays.asList(RegionCountSkewCostFunction.class.getSimpleName(),
529      PrimaryRegionCountSkewCostFunction.class.getSimpleName(),
530      MoveCostFunction.class.getSimpleName(), RackLocalityCostFunction.class.getSimpleName(),
531      TableSkewCostFunction.class.getSimpleName(),
532      RegionReplicaHostCostFunction.class.getSimpleName(),
533      RegionReplicaRackCostFunction.class.getSimpleName(),
534      ReadRequestCostFunction.class.getSimpleName(), WriteRequestCostFunction.class.getSimpleName(),
535      MemStoreSizeCostFunction.class.getSimpleName(), StoreFileCostFunction.class.getSimpleName());
536
537    List<String> actual = Arrays.asList(loadBalancer.getCostFunctionNames());
538    assertTrue("ExpectedCostFunctions: " + expected + " ActualCostFunctions: " + actual,
539      CollectionUtils.isEqualCollection(expected, actual));
540  }
541
542  private boolean needsBalanceIdleRegion(int[] cluster) {
543    return Arrays.stream(cluster).anyMatch(x -> x > 1)
544      && Arrays.stream(cluster).anyMatch(x -> x < 1);
545  }
546
547  // This mock allows us to test the LocalityCostFunction
548  private class MockCluster extends BalancerClusterState {
549
550    private int[][] localities = null; // [region][server] = percent of blocks
551
552    public MockCluster(int[][] regions) {
553
554      // regions[0] is an array where index = serverIndex an value = number of regions
555      super(mockClusterServers(regions[0], 1), null, null, null);
556
557      localities = new int[regions.length - 1][];
558      for (int i = 1; i < regions.length; i++) {
559        int regionIndex = i - 1;
560        localities[regionIndex] = new int[regions[i].length - 1];
561        regionIndexToServerIndex[regionIndex] = regions[i][0];
562        for (int j = 1; j < regions[i].length; j++) {
563          int serverIndex = j - 1;
564          localities[regionIndex][serverIndex] =
565            regions[i][j] > 100 ? regions[i][j] % 100 : regions[i][j];
566        }
567      }
568    }
569
570    @Override
571    float getLocalityOfRegion(int region, int server) {
572      // convert the locality percentage to a fraction
573      return localities[region][server] / 100.0f;
574    }
575
576    @Override
577    public int getRegionSizeMB(int region) {
578      return 1;
579    }
580  }
581}