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      when(rl.getRegionSizeMB()).thenReturn(Size.ZERO);
143      when(rl.getCurrentRegionCachedRatio()).thenReturn(0.0f);
144
145      Map<byte[], RegionMetrics> regionLoadMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
146      regionLoadMap.put(Bytes.toBytes(REGION_KEY), rl);
147      when(sl.getRegionMetrics()).thenReturn(regionLoadMap);
148
149      ClusterMetrics clusterStatus = mock(ClusterMetrics.class);
150      Map<ServerName, ServerMetrics> serverMetricsMap = new TreeMap<>();
151      serverMetricsMap.put(sn, sl);
152      when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
153      // when(clusterStatus.getLoad(sn)).thenReturn(sl);
154
155      loadBalancer.updateClusterMetrics(clusterStatus);
156    }
157
158    String regionNameAsString = RegionInfo.getRegionNameAsString(Bytes.toBytes(REGION_KEY));
159    assertTrue(loadBalancer.loads.get(regionNameAsString) != null);
160    assertTrue(loadBalancer.loads.get(regionNameAsString).size() == 15);
161    assertNull(loadBalancer.namedQueueRecorder);
162
163    Queue<BalancerRegionLoad> loads = loadBalancer.loads.get(regionNameAsString);
164    int i = 0;
165    while (loads.size() > 0) {
166      BalancerRegionLoad rl = loads.remove();
167      assertEquals(i + (numClusterStatusToAdd - 15), rl.getStorefileSizeMB());
168      i++;
169    }
170  }
171
172  @Test
173  public void testUpdateBalancerLoadInfo() {
174    int[] cluster = new int[] { 10, 0 };
175    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
176    BalancerClusterState clusterState = mockCluster(cluster);
177    Map<TableName, Map<ServerName, List<RegionInfo>>> LoadOfAllTable =
178      (Map) mockClusterServersWithTables(servers);
179    boolean[] perTableBalancerConfigs = { true, false };
180    for (boolean isByTable : perTableBalancerConfigs) {
181      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
182      loadBalancer.onConfigurationChange(conf);
183      dummyMetricsStochasticBalancer.clearDummyMetrics();
184      loadBalancer.updateBalancerLoadInfo(LoadOfAllTable);
185      assertTrue("Metrics should be recorded!",
186        dummyMetricsStochasticBalancer.getDummyCostsMap() != null
187          && !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
188
189      String metricRecordKey;
190      if (isByTable) {
191        metricRecordKey = "table1#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
192      } else {
193        metricRecordKey =
194          HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
195      }
196      double curOverallCost = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
197      double curOverallCostInMetrics =
198        dummyMetricsStochasticBalancer.getDummyCostsMap().get(metricRecordKey);
199      assertEquals(curOverallCost, curOverallCostInMetrics, 0.001);
200    }
201  }
202
203  @Test
204  public void testUpdateStochasticCosts() {
205    int[] cluster = new int[] { 10, 0 };
206    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
207    BalancerClusterState clusterState = mockCluster(cluster);
208    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
209    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
210    loadBalancer.onConfigurationChange(conf);
211    dummyMetricsStochasticBalancer.clearDummyMetrics();
212    List<RegionPlan> plans =
213      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
214
215    assertTrue("Balance plan should not be empty!", plans != null && !plans.isEmpty());
216    assertTrue("There should be metrics record in MetricsStochasticBalancer",
217      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
218
219    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
220    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
221      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
222    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
223  }
224
225  @Test
226  public void testUpdateStochasticCostsIfBalanceNotRan() {
227    int[] cluster = new int[] { 10, 10 };
228    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
229    BalancerClusterState clusterState = mockCluster(cluster);
230    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", Float.MAX_VALUE);
231    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
232    loadBalancer.onConfigurationChange(conf);
233    dummyMetricsStochasticBalancer.clearDummyMetrics();
234    List<RegionPlan> plans =
235      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
236
237    assertTrue("Balance plan should be empty!", plans == null || plans.isEmpty());
238    assertTrue("There should be metrics record in MetricsStochasticBalancer!",
239      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
240
241    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
242    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
243      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
244    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
245  }
246
247  @Test
248  public void testNeedBalance() {
249    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
250    conf.setFloat(HConstants.LOAD_BALANCER_SLOP_KEY, -1f);
251    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
252    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
253    // Test with/without per table balancer.
254    boolean[] perTableBalancerConfigs = { true, false };
255    for (boolean isByTable : perTableBalancerConfigs) {
256      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
257      loadBalancer.onConfigurationChange(conf);
258      for (int[] mockCluster : clusterStateMocks) {
259        assertTrue(hasEmptyBalancerPlans(mockCluster) || needsBalanceIdleRegion(mockCluster));
260      }
261    }
262  }
263
264  @Test
265  public void testBalanceOfSloppyServers() {
266    // We are testing slop checks, so don't "accidentally" balance due to a minCost calculation.
267    // During development, imbalance of a 100 server cluster, with one node having 10 regions
268    // and the rest having 5, is 0.0048. With minCostNeedBalance default of 0.025, test should
269    // validate slop checks without this override. We override just to ensure we will always
270    // validate slop check here, and for small clusters as well.
271    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
272    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
273    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
274    loadBalancer.onConfigurationChange(conf);
275    for (int[] mockCluster : clusterStateMocksWithNoSlop) {
276      assertTrue(hasEmptyBalancerPlans(mockCluster));
277    }
278    for (int[] mockCluster : clusterStateMocksWithSlop) {
279      assertFalse(hasEmptyBalancerPlans(mockCluster));
280    }
281  }
282
283  @Test
284  public void testSloppyTablesLoadBalanceByTable() {
285    int[][] regionsPerServerPerTable = new int[][] {
286      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,
287        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 },
288      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,
289        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 }, };
290    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
291    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
292    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
293    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, true);
294    loadBalancer.onConfigurationChange(conf);
295    assertFalse(hasEmptyBalancerPlans(regionsPerServerPerTable));
296  }
297
298  private boolean hasEmptyBalancerPlans(int[] mockCluster) {
299    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
300    return hasEmptyBalancerPlans(servers);
301  }
302
303  private boolean hasEmptyBalancerPlans(int[][] mockCluster) {
304    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
305    return hasEmptyBalancerPlans(servers);
306  }
307
308  private boolean hasEmptyBalancerPlans(Map<ServerName, List<RegionInfo>> servers) {
309    Map<TableName, Map<ServerName, List<RegionInfo>>> loadOfAllTable =
310      (Map) mockClusterServersWithTables(servers);
311    List<RegionPlan> plans = loadBalancer.balanceCluster(loadOfAllTable);
312    return plans == null || plans.isEmpty();
313  }
314
315  @Test
316  public void testLocalityCost() throws Exception {
317    Configuration conf = HBaseConfiguration.create();
318    CostFunction costFunction = new ServerLocalityCostFunction(conf);
319
320    for (int test = 0; test < clusterRegionLocationMocks.length; test++) {
321      int[][] clusterRegionLocations = clusterRegionLocationMocks[test];
322      MockCluster cluster = new MockCluster(clusterRegionLocations);
323      costFunction.prepare(cluster);
324      double cost = costFunction.cost();
325      double expected = 1 - expectedLocalities[test];
326      assertEquals(expected, cost, 0.001);
327    }
328    assertNull(loadBalancer.namedQueueRecorder);
329  }
330
331  @Test
332  public void testMoveCostMultiplier() throws Exception {
333    Configuration conf = HBaseConfiguration.create();
334    CostFunction costFunction = new MoveCostFunction(conf);
335    BalancerClusterState cluster = mockCluster(clusterStateMocks[0]);
336    costFunction.prepare(cluster);
337    costFunction.cost();
338    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST, costFunction.getMultiplier(), 0.01);
339
340    // In offpeak hours, the multiplier of move cost should be lower
341    conf.setInt("hbase.offpeak.start.hour", 0);
342    conf.setInt("hbase.offpeak.end.hour", 23);
343    // Set a fixed time which hour is 15, so it will always in offpeak
344    // See HBASE-24898 for more info of the calculation here
345    long deltaFor15 = TimeZone.getDefault().getRawOffset() - 28800000;
346    long timeFor15 = 1597907081000L - deltaFor15;
347    EnvironmentEdgeManager.injectEdge(() -> timeFor15);
348    costFunction = new MoveCostFunction(conf);
349    costFunction.prepare(cluster);
350    costFunction.cost();
351    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST_OFFPEAK, costFunction.getMultiplier(), 0.01);
352  }
353
354  @Test
355  public void testMoveCost() throws Exception {
356    Configuration conf = HBaseConfiguration.create();
357    CostFunction costFunction = new MoveCostFunction(conf);
358    for (int[] mockCluster : clusterStateMocks) {
359      BalancerClusterState cluster = mockCluster(mockCluster);
360      costFunction.prepare(cluster);
361      double cost = costFunction.cost();
362      assertEquals(0.0f, cost, 0.001);
363
364      // cluster region number is smaller than maxMoves=600
365      cluster.setNumRegions(200);
366      cluster.setNumMovedRegions(10);
367      cost = costFunction.cost();
368      assertEquals(0.05f, cost, 0.001);
369      cluster.setNumMovedRegions(100);
370      cost = costFunction.cost();
371      assertEquals(0.5f, cost, 0.001);
372      cluster.setNumMovedRegions(200);
373      cost = costFunction.cost();
374      assertEquals(1.0f, cost, 0.001);
375
376      // cluster region number is bigger than maxMoves=2500
377      cluster.setNumRegions(10000);
378      cluster.setNumMovedRegions(250);
379      cost = costFunction.cost();
380      assertEquals(0.025f, cost, 0.001);
381      cluster.setNumMovedRegions(1250);
382      cost = costFunction.cost();
383      assertEquals(0.125f, cost, 0.001);
384      cluster.setNumMovedRegions(2500);
385      cost = costFunction.cost();
386      assertEquals(0.25f, cost, 0.01);
387    }
388  }
389
390  @Test
391  public void testSkewCost() {
392    Configuration conf = HBaseConfiguration.create();
393    CostFunction costFunction = new RegionCountSkewCostFunction(conf);
394    for (int[] mockCluster : clusterStateMocks) {
395      costFunction.prepare(mockCluster(mockCluster));
396      double cost = costFunction.cost();
397      assertTrue(cost >= 0);
398      assertTrue(cost <= 1.01);
399    }
400
401    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 0, 1 }));
402    assertEquals(0, costFunction.cost(), 0.01);
403    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 1, 1 }));
404    assertEquals(0, costFunction.cost(), 0.01);
405    costFunction.prepare(mockCluster(new int[] { 0, 0, 1, 1, 1 }));
406    assertEquals(0, costFunction.cost(), 0.01);
407    costFunction.prepare(mockCluster(new int[] { 0, 1, 1, 1, 1 }));
408    assertEquals(0, costFunction.cost(), 0.01);
409    costFunction.prepare(mockCluster(new int[] { 1, 1, 1, 1, 1 }));
410    assertEquals(0, costFunction.cost(), 0.01);
411    costFunction.prepare(mockCluster(new int[] { 10000, 0, 0, 0, 0 }));
412    assertEquals(1, costFunction.cost(), 0.01);
413  }
414
415  @Test
416  public void testCostAfterUndoAction() {
417    final int runs = 10;
418    for (int[] mockCluster : clusterStateMocks) {
419      BalancerClusterState cluster = mockCluster(mockCluster);
420      loadBalancer.initCosts(cluster);
421      for (int i = 0; i != runs; ++i) {
422        final double expectedCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
423        BalanceAction action = loadBalancer.nextAction(cluster);
424        cluster.doAction(action);
425        loadBalancer.updateCostsAndWeightsWithAction(cluster, action);
426        BalanceAction undoAction = action.undoAction();
427        cluster.doAction(undoAction);
428        loadBalancer.updateCostsAndWeightsWithAction(cluster, undoAction);
429        final double actualCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
430        assertEquals(expectedCost, actualCost, 0);
431      }
432    }
433  }
434
435  @Test
436  public void testTableSkewCost() {
437    Configuration conf = HBaseConfiguration.create();
438    CostFunction costFunction = new TableSkewCostFunction(conf);
439    for (int[] mockCluster : clusterStateMocks) {
440      BalancerClusterState cluster = mockCluster(mockCluster);
441      costFunction.prepare(cluster);
442      double cost = costFunction.cost();
443      assertTrue(cost >= 0);
444      assertTrue(cost <= 1.01);
445    }
446  }
447
448  @Test
449  public void testRegionLoadCost() {
450    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
451    for (int i = 1; i < 5; i++) {
452      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
453      when(regionLoad.getReadRequestsCount()).thenReturn(new Long(i));
454      when(regionLoad.getStorefileSizeMB()).thenReturn(i);
455      regionLoads.add(regionLoad);
456    }
457
458    Configuration conf = HBaseConfiguration.create();
459    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
460    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
461    // read requests are treated as a rate so the average rate here is simply 1
462    assertEquals(1, rateResult, 0.01);
463
464    StoreFileCostFunction storeFileCostFunction = new StoreFileCostFunction(conf);
465    double result = storeFileCostFunction.getRegionLoadCost(regionLoads);
466    // storefile size cost is simply an average of it's value over time
467    assertEquals(2.5, result, 0.01);
468  }
469
470  @Test
471  public void testRegionLoadCostWhenDecrease() {
472    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
473    // test region loads of [1,2,1,4]
474    for (int i = 1; i < 5; i++) {
475      int load = i == 3 ? 1 : i;
476      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
477      when(regionLoad.getReadRequestsCount()).thenReturn((long) load);
478      regionLoads.add(regionLoad);
479    }
480
481    Configuration conf = HBaseConfiguration.create();
482    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
483    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
484    // read requests are treated as a rate so the average rate here is simply 1
485    assertEquals(1.67, rateResult, 0.01);
486  }
487
488  @Test
489  public void testLosingRs() throws Exception {
490    int numNodes = 3;
491    int numRegions = 20;
492    int numRegionsPerServer = 3; // all servers except one
493    int replication = 1;
494    int numTables = 2;
495
496    Map<ServerName, List<RegionInfo>> serverMap =
497      createServerMap(numNodes, numRegions, numRegionsPerServer, replication, numTables);
498    List<ServerAndLoad> list = convertToList(serverMap);
499
500    List<RegionPlan> plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
501    assertNotNull(plans);
502
503    // Apply the plan to the mock cluster.
504    List<ServerAndLoad> balancedCluster = reconcile(list, plans, serverMap);
505
506    assertClusterAsBalanced(balancedCluster);
507
508    ServerName sn = serverMap.keySet().toArray(new ServerName[serverMap.size()])[0];
509
510    ServerName deadSn = ServerName.valueOf(sn.getHostname(), sn.getPort(), sn.getStartcode() - 100);
511
512    serverMap.put(deadSn, new ArrayList<>(0));
513
514    plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
515    assertNull(plans);
516  }
517
518  @Test
519  public void testAdditionalCostFunction() {
520    conf.set(StochasticLoadBalancer.COST_FUNCTIONS_COST_FUNCTIONS_KEY,
521      DummyCostFunction.class.getName());
522
523    loadBalancer.onConfigurationChange(conf);
524    assertTrue(Arrays.asList(loadBalancer.getCostFunctionNames())
525      .contains(DummyCostFunction.class.getSimpleName()));
526  }
527
528  @Test
529  public void testDefaultCostFunctionList() {
530    List<String> expected = Arrays.asList(RegionCountSkewCostFunction.class.getSimpleName(),
531      PrimaryRegionCountSkewCostFunction.class.getSimpleName(),
532      MoveCostFunction.class.getSimpleName(), RackLocalityCostFunction.class.getSimpleName(),
533      TableSkewCostFunction.class.getSimpleName(),
534      RegionReplicaHostCostFunction.class.getSimpleName(),
535      RegionReplicaRackCostFunction.class.getSimpleName(),
536      ReadRequestCostFunction.class.getSimpleName(), WriteRequestCostFunction.class.getSimpleName(),
537      MemStoreSizeCostFunction.class.getSimpleName(), StoreFileCostFunction.class.getSimpleName());
538
539    List<String> actual = Arrays.asList(loadBalancer.getCostFunctionNames());
540    assertTrue("ExpectedCostFunctions: " + expected + " ActualCostFunctions: " + actual,
541      CollectionUtils.isEqualCollection(expected, actual));
542  }
543
544  private boolean needsBalanceIdleRegion(int[] cluster) {
545    return Arrays.stream(cluster).anyMatch(x -> x > 1)
546      && Arrays.stream(cluster).anyMatch(x -> x < 1);
547  }
548
549  // This mock allows us to test the LocalityCostFunction
550  private class MockCluster extends BalancerClusterState {
551
552    private int[][] localities = null; // [region][server] = percent of blocks
553
554    public MockCluster(int[][] regions) {
555
556      // regions[0] is an array where index = serverIndex an value = number of regions
557      super(mockClusterServers(regions[0], 1), null, null, null);
558
559      localities = new int[regions.length - 1][];
560      for (int i = 1; i < regions.length; i++) {
561        int regionIndex = i - 1;
562        localities[regionIndex] = new int[regions[i].length - 1];
563        regionIndexToServerIndex[regionIndex] = regions[i][0];
564        for (int j = 1; j < regions[i].length; j++) {
565          int serverIndex = j - 1;
566          localities[regionIndex][serverIndex] =
567            regions[i][j] > 100 ? regions[i][j] % 100 : regions[i][j];
568        }
569      }
570    }
571
572    @Override
573    float getLocalityOfRegion(int region, int server) {
574      // convert the locality percentage to a fraction
575      return localities[region][server] / 100.0f;
576    }
577
578    @Override
579    public int getRegionSizeMB(int region) {
580      return 1;
581    }
582  }
583}