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.HashMap;
031import java.util.HashSet;
032import java.util.List;
033import java.util.Map;
034import java.util.Queue;
035import java.util.Set;
036import java.util.TreeMap;
037import org.apache.hadoop.conf.Configuration;
038import org.apache.hadoop.hbase.ClusterMetrics;
039import org.apache.hadoop.hbase.HBaseClassTestRule;
040import org.apache.hadoop.hbase.HBaseConfiguration;
041import org.apache.hadoop.hbase.HConstants;
042import org.apache.hadoop.hbase.RegionMetrics;
043import org.apache.hadoop.hbase.ServerMetrics;
044import org.apache.hadoop.hbase.ServerName;
045import org.apache.hadoop.hbase.Size;
046import org.apache.hadoop.hbase.TableName;
047import org.apache.hadoop.hbase.client.RegionInfo;
048import org.apache.hadoop.hbase.master.RegionPlan;
049import org.apache.hadoop.hbase.testclassification.MasterTests;
050import org.apache.hadoop.hbase.testclassification.MediumTests;
051import org.apache.hadoop.hbase.util.Bytes;
052import org.junit.Before;
053import org.junit.BeforeClass;
054import org.junit.ClassRule;
055import org.junit.Test;
056import org.junit.experimental.categories.Category;
057
058import org.apache.hbase.thirdparty.org.apache.commons.collections4.CollectionUtils;
059
060@Category({ MasterTests.class, MediumTests.class })
061public class TestStochasticLoadBalancer extends StochasticBalancerTestBase {
062
063  @ClassRule
064  public static final HBaseClassTestRule CLASS_RULE =
065    HBaseClassTestRule.forClass(TestStochasticLoadBalancer.class);
066
067  private static final String REGION_KEY = "testRegion";
068
069  // Mapping of locality test -> expected locality
070  private float[] expectedLocalities = { 1.0f, 0.0f, 0.50f, 0.25f, 1.0f };
071  private static Configuration storedConfiguration;
072
073  @BeforeClass
074  public static void saveInitialConfiguration() {
075    storedConfiguration = new Configuration(conf);
076  }
077
078  @Before
079  public void beforeEachTest() {
080    conf = new Configuration(storedConfiguration);
081    loadBalancer.onConfigurationChange(conf);
082  }
083
084  /**
085   * Data set for testLocalityCost: [test][0][0] = mapping of server to number of regions it hosts
086   * [test][region + 1][0] = server that region is hosted on [test][region + 1][server + 1] =
087   * locality for region on server
088   */
089  private int[][][] clusterRegionLocationMocks = new int[][][] {
090    // Test 1: each region is entirely on server that hosts it
091    new int[][] { new int[] { 2, 1, 1 }, new int[] { 2, 0, 0, 100 }, // region 0 is hosted and
092                                                                     // entirely local on server 2
093      new int[] { 0, 100, 0, 0 }, // region 1 is hosted and entirely on server 0
094      new int[] { 0, 100, 0, 0 }, // region 2 is hosted and entirely on server 0
095      new int[] { 1, 0, 100, 0 }, // region 3 is hosted and entirely on server 1
096    },
097
098    // Test 2: each region is 0% local on the server that hosts it
099    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 0, 0, 100 }, // region 0 is hosted and
100                                                                     // entirely local on server 2
101      new int[] { 1, 100, 0, 0 }, // region 1 is hosted and entirely on server 0
102      new int[] { 1, 100, 0, 0 }, // region 2 is hosted and entirely on server 0
103      new int[] { 2, 0, 100, 0 }, // region 3 is hosted and entirely on server 1
104    },
105
106    // Test 3: each region is 25% local on the server that hosts it (and 50% locality is possible)
107    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 25, 0, 50 }, // region 0 is hosted and
108                                                                     // entirely local on server 2
109      new int[] { 1, 50, 25, 0 }, // region 1 is hosted and entirely on server 0
110      new int[] { 1, 50, 25, 0 }, // region 2 is hosted and entirely on server 0
111      new int[] { 2, 0, 50, 25 }, // region 3 is hosted and entirely on server 1
112    },
113
114    // Test 4: each region is 25% local on the server that hosts it (and 100% locality is possible)
115    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 25, 0, 100 }, // region 0 is hosted and
116                                                                      // entirely local on server 2
117      new int[] { 1, 100, 25, 0 }, // region 1 is hosted and entirely on server 0
118      new int[] { 1, 100, 25, 0 }, // region 2 is hosted and entirely on server 0
119      new int[] { 2, 0, 100, 25 }, // region 3 is hosted and entirely on server 1
120    },
121
122    // Test 5: each region is 75% local on the server that hosts it (and 75% locality is possible
123    // everywhere)
124    new int[][] { new int[] { 1, 2, 1 }, new int[] { 0, 75, 75, 75 }, // region 0 is hosted and
125                                                                      // entirely local on server 2
126      new int[] { 1, 75, 75, 75 }, // region 1 is hosted and entirely on server 0
127      new int[] { 1, 75, 75, 75 }, // region 2 is hosted and entirely on server 0
128      new int[] { 2, 75, 75, 75 }, // region 3 is hosted and entirely on server 1
129    }, };
130
131  private ServerMetrics mockServerMetricsWithCpRequests(List<RegionInfo> regionsOnServer,
132    long cpRequestCount) {
133    ServerMetrics serverMetrics = mock(ServerMetrics.class);
134    Map<byte[], RegionMetrics> regionLoadMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
135    for (RegionInfo info : regionsOnServer) {
136      RegionMetrics rl = mock(RegionMetrics.class);
137      when(rl.getReadRequestCount()).thenReturn(0L);
138      when(rl.getCpRequestCount()).thenReturn(cpRequestCount);
139      when(rl.getWriteRequestCount()).thenReturn(0L);
140      when(rl.getMemStoreSize()).thenReturn(Size.ZERO);
141      when(rl.getStoreFileSize()).thenReturn(Size.ZERO);
142      when(rl.getRegionSizeMB()).thenReturn(Size.ZERO);
143      when(rl.getCurrentRegionCachedRatio()).thenReturn(0.0f);
144      regionLoadMap.put(info.getRegionName(), rl);
145    }
146    when(serverMetrics.getRegionMetrics()).thenReturn(regionLoadMap);
147    return serverMetrics;
148  }
149
150  @Test
151  public void testCPRequestCost() {
152    // in order to pass needsBalance judgement
153    conf.setFloat("hbase.master.balancer.stochastic.cpRequestCost", 10000f);
154    loadBalancer.onConfigurationChange(conf);
155    // mock cluster State
156    Map<ServerName, List<RegionInfo>> clusterState = new HashMap<ServerName, List<RegionInfo>>();
157    ServerName serverA = randomServer(3).getServerName();
158    ServerName serverB = randomServer(3).getServerName();
159    ServerName serverC = randomServer(3).getServerName();
160    List<RegionInfo> regionsOnServerA = randomRegions(3);
161    List<RegionInfo> regionsOnServerB = randomRegions(3);
162    List<RegionInfo> regionsOnServerC = randomRegions(3);
163    clusterState.put(serverA, regionsOnServerA);
164    clusterState.put(serverB, regionsOnServerB);
165    clusterState.put(serverC, regionsOnServerC);
166    // mock ClusterMetrics
167    Map<ServerName, ServerMetrics> serverMetricsMap = new TreeMap<>();
168    serverMetricsMap.put(serverA, mockServerMetricsWithCpRequests(regionsOnServerA, 0));
169    serverMetricsMap.put(serverB, mockServerMetricsWithCpRequests(regionsOnServerB, 0));
170    serverMetricsMap.put(serverC, mockServerMetricsWithCpRequests(regionsOnServerC, 0));
171    ClusterMetrics clusterStatus = mock(ClusterMetrics.class);
172    when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
173    loadBalancer.updateClusterMetrics(clusterStatus);
174
175    // CPRequestCostFunction are Rate based, So doing setClusterMetrics again
176    // this time, regions on serverA with more cpRequestCount load
177    // serverA : 1000,1000,1000
178    // serverB : 0,0,0
179    // serverC : 0,0,0
180    // so should move two regions from serverA to serverB & serverC
181    serverMetricsMap = new TreeMap<>();
182    serverMetricsMap.put(serverA, mockServerMetricsWithCpRequests(regionsOnServerA, 1000));
183    serverMetricsMap.put(serverB, mockServerMetricsWithCpRequests(regionsOnServerB, 0));
184    serverMetricsMap.put(serverC, mockServerMetricsWithCpRequests(regionsOnServerC, 0));
185    clusterStatus = mock(ClusterMetrics.class);
186    when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
187    loadBalancer.updateClusterMetrics(clusterStatus);
188
189    List<RegionPlan> plans =
190      loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, clusterState);
191    Set<RegionInfo> regionsMoveFromServerA = new HashSet<>();
192    Set<ServerName> targetServers = new HashSet<>();
193    for (RegionPlan plan : plans) {
194      if (plan.getSource().equals(serverA)) {
195        regionsMoveFromServerA.add(plan.getRegionInfo());
196        targetServers.add(plan.getDestination());
197      }
198    }
199    // should move 2 regions from serverA, one moves to serverB, the other moves to serverC
200    assertEquals(2, regionsMoveFromServerA.size());
201    assertEquals(2, targetServers.size());
202    assertTrue(regionsOnServerA.containsAll(regionsMoveFromServerA));
203  }
204
205  @Test
206  public void testKeepRegionLoad() throws Exception {
207    ServerName sn = ServerName.valueOf("test:8080", 100);
208    int numClusterStatusToAdd = 20000;
209    for (int i = 0; i < numClusterStatusToAdd; i++) {
210      ServerMetrics sl = mock(ServerMetrics.class);
211
212      RegionMetrics rl = mock(RegionMetrics.class);
213      when(rl.getReadRequestCount()).thenReturn(0L);
214      when(rl.getCpRequestCount()).thenReturn(0L);
215      when(rl.getWriteRequestCount()).thenReturn(0L);
216      when(rl.getMemStoreSize()).thenReturn(Size.ZERO);
217      when(rl.getStoreFileSize()).thenReturn(new Size(i, Size.Unit.MEGABYTE));
218      when(rl.getRegionSizeMB()).thenReturn(Size.ZERO);
219      when(rl.getCurrentRegionCachedRatio()).thenReturn(0.0f);
220
221      Map<byte[], RegionMetrics> regionLoadMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
222      regionLoadMap.put(Bytes.toBytes(REGION_KEY), rl);
223      when(sl.getRegionMetrics()).thenReturn(regionLoadMap);
224
225      ClusterMetrics clusterStatus = mock(ClusterMetrics.class);
226      Map<ServerName, ServerMetrics> serverMetricsMap = new TreeMap<>();
227      serverMetricsMap.put(sn, sl);
228      when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
229
230      loadBalancer.updateClusterMetrics(clusterStatus);
231    }
232
233    String regionNameAsString = RegionInfo.getRegionNameAsString(Bytes.toBytes(REGION_KEY));
234    assertTrue(loadBalancer.loads.get(regionNameAsString) != null);
235    assertTrue(loadBalancer.loads.get(regionNameAsString).size() == 15);
236
237    Queue<BalancerRegionLoad> loads = loadBalancer.loads.get(regionNameAsString);
238    int i = 0;
239    while (loads.size() > 0) {
240      BalancerRegionLoad rl = loads.remove();
241      assertEquals(i + (numClusterStatusToAdd - 15), rl.getStorefileSizeMB());
242      i++;
243    }
244  }
245
246  @Test
247  public void testUpdateBalancerLoadInfo() {
248    int[] cluster = new int[] { 10, 0 };
249    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
250    BalancerClusterState clusterState = mockCluster(cluster);
251    Map<TableName, Map<ServerName, List<RegionInfo>>> LoadOfAllTable =
252      (Map) mockClusterServersWithTables(servers);
253    boolean[] perTableBalancerConfigs = { true, false };
254    for (boolean isByTable : perTableBalancerConfigs) {
255      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
256      loadBalancer.onConfigurationChange(conf);
257      dummyMetricsStochasticBalancer.clearDummyMetrics();
258      loadBalancer.updateBalancerLoadInfo(LoadOfAllTable);
259      assertTrue("Metrics should be recorded!",
260        dummyMetricsStochasticBalancer.getDummyCostsMap() != null
261          && !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
262
263      String metricRecordKey;
264      if (isByTable) {
265        metricRecordKey = "table1#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
266      } else {
267        metricRecordKey =
268          HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME;
269      }
270      double curOverallCost = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
271      double curOverallCostInMetrics =
272        dummyMetricsStochasticBalancer.getDummyCostsMap().get(metricRecordKey);
273      assertEquals(curOverallCost, curOverallCostInMetrics, 0.001);
274    }
275  }
276
277  @Test
278  public void testUpdateStochasticCosts() {
279    int[] cluster = new int[] { 10, 0 };
280    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
281    BalancerClusterState clusterState = mockCluster(cluster);
282    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
283    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
284    loadBalancer.onConfigurationChange(conf);
285    dummyMetricsStochasticBalancer.clearDummyMetrics();
286    List<RegionPlan> plans =
287      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
288
289    assertTrue("Balance plan should not be empty!", plans != null && !plans.isEmpty());
290    assertTrue("There should be metrics record in MetricsStochasticBalancer",
291      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
292
293    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
294    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
295      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
296    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
297  }
298
299  @Test
300  public void testUpdateStochasticCostsIfBalanceNotRan() {
301    int[] cluster = new int[] { 10, 10 };
302    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(cluster);
303    BalancerClusterState clusterState = mockCluster(cluster);
304    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", Float.MAX_VALUE);
305    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, false);
306    loadBalancer.onConfigurationChange(conf);
307    dummyMetricsStochasticBalancer.clearDummyMetrics();
308    List<RegionPlan> plans =
309      loadBalancer.balanceCluster((Map) mockClusterServersWithTables(servers));
310
311    assertTrue("Balance plan should be empty!", plans == null || plans.isEmpty());
312    assertTrue("There should be metrics record in MetricsStochasticBalancer!",
313      !dummyMetricsStochasticBalancer.getDummyCostsMap().isEmpty());
314
315    double overallCostOfCluster = loadBalancer.computeCost(clusterState, Double.MAX_VALUE);
316    double overallCostInMetrics = dummyMetricsStochasticBalancer.getDummyCostsMap().get(
317      HConstants.ENSEMBLE_TABLE_NAME + "#" + StochasticLoadBalancer.OVERALL_COST_FUNCTION_NAME);
318    assertEquals(overallCostOfCluster, overallCostInMetrics, 0.001);
319  }
320
321  @Test
322  public void testNeedBalance() {
323    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
324    conf.setFloat(HConstants.LOAD_BALANCER_SLOP_KEY, -1f);
325    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
326    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
327    // Test with/without per table balancer.
328    boolean[] perTableBalancerConfigs = { true, false };
329    for (boolean isByTable : perTableBalancerConfigs) {
330      conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
331      loadBalancer.onConfigurationChange(conf);
332      for (int[] mockCluster : clusterStateMocks) {
333        assertTrue(hasEmptyBalancerPlans(mockCluster) || needsBalanceIdleRegion(mockCluster));
334      }
335    }
336  }
337
338  @Test
339  public void testBalanceOfSloppyServers() {
340    // We are testing slop checks, so don't "accidentally" balance due to a minCost calculation.
341    // During development, imbalance of a 100 server cluster, with one node having 10 regions
342    // and the rest having 5, is 0.0048. With minCostNeedBalance default of 0.025, test should
343    // validate slop checks without this override. We override just to ensure we will always
344    // validate slop check here, and for small clusters as well.
345    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
346    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
347    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
348    loadBalancer.onConfigurationChange(conf);
349    for (int[] mockCluster : clusterStateMocksWithNoSlop) {
350      assertTrue(hasEmptyBalancerPlans(mockCluster));
351    }
352    for (int[] mockCluster : clusterStateMocksWithSlop) {
353      assertFalse(hasEmptyBalancerPlans(mockCluster));
354    }
355  }
356
357  @Test
358  public void testSloppyTablesLoadBalanceByTable() {
359    int[][] regionsPerServerPerTable = new int[][] {
360      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,
361        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 },
362      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,
363        5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5, 5 }, };
364    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
365    conf.setBoolean("hbase.master.balancer.stochastic.runMaxSteps", false);
366    conf.setLong("hbase.master.balancer.stochastic.maxSteps", 5000L);
367    conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, true);
368    loadBalancer.onConfigurationChange(conf);
369    assertFalse(hasEmptyBalancerPlans(regionsPerServerPerTable));
370  }
371
372  private boolean hasEmptyBalancerPlans(int[] mockCluster) {
373    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
374    return hasEmptyBalancerPlans(servers);
375  }
376
377  private boolean hasEmptyBalancerPlans(int[][] mockCluster) {
378    Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
379    return hasEmptyBalancerPlans(servers);
380  }
381
382  private boolean hasEmptyBalancerPlans(Map<ServerName, List<RegionInfo>> servers) {
383    Map<TableName, Map<ServerName, List<RegionInfo>>> loadOfAllTable =
384      (Map) mockClusterServersWithTables(servers);
385    List<RegionPlan> plans = loadBalancer.balanceCluster(loadOfAllTable);
386    return plans == null || plans.isEmpty();
387  }
388
389  @Test
390  public void testLocalityCost() throws Exception {
391    Configuration conf = HBaseConfiguration.create();
392    CostFunction costFunction = new ServerLocalityCostFunction(conf);
393
394    for (int test = 0; test < clusterRegionLocationMocks.length; test++) {
395      int[][] clusterRegionLocations = clusterRegionLocationMocks[test];
396      MockCluster cluster = new MockCluster(clusterRegionLocations);
397      costFunction.prepare(cluster);
398      double cost = costFunction.cost();
399      double expected = 1 - expectedLocalities[test];
400      assertEquals(expected, cost, 0.001);
401    }
402  }
403
404  @Test
405  public void testMoveCostMultiplier() throws Exception {
406    Configuration conf = HBaseConfiguration.create();
407    ClusterInfoProvider provider = mock(ClusterInfoProvider.class);
408    CostFunction costFunction = new MoveCostFunction(conf, provider);
409    when(provider.isOffPeakHour()).thenReturn(false);
410    BalancerClusterState cluster = mockCluster(clusterStateMocks[0]);
411    costFunction.prepare(cluster);
412    costFunction.cost();
413    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST, costFunction.getMultiplier(), 0.01);
414
415    // In offpeak hours, the multiplier of move cost should be lower
416    when(provider.isOffPeakHour()).thenReturn(true);
417    costFunction.prepare(cluster);
418    costFunction.cost();
419    assertEquals(MoveCostFunction.DEFAULT_MOVE_COST_OFFPEAK, costFunction.getMultiplier(), 0.01);
420  }
421
422  @Test
423  public void testMoveCost() throws Exception {
424    Configuration conf = HBaseConfiguration.create();
425    CostFunction costFunction = new MoveCostFunction(conf, new DummyClusterInfoProvider(conf));
426    for (int[] mockCluster : clusterStateMocks) {
427      BalancerClusterState cluster = mockCluster(mockCluster);
428      costFunction.prepare(cluster);
429      double cost = costFunction.cost();
430      assertEquals(0.0f, cost, 0.001);
431
432      // cluster region number is smaller than maxMoves=600
433      cluster.setNumRegions(200);
434      cluster.setNumMovedRegions(10);
435      cost = costFunction.cost();
436      assertEquals(0.05f, cost, 0.001);
437      cluster.setNumMovedRegions(100);
438      cost = costFunction.cost();
439      assertEquals(0.5f, cost, 0.001);
440      cluster.setNumMovedRegions(200);
441      cost = costFunction.cost();
442      assertEquals(1.0f, cost, 0.001);
443
444      // cluster region number is bigger than maxMoves=2500
445      cluster.setNumRegions(10000);
446      cluster.setNumMovedRegions(250);
447      cost = costFunction.cost();
448      assertEquals(0.025f, cost, 0.001);
449      cluster.setNumMovedRegions(1250);
450      cost = costFunction.cost();
451      assertEquals(0.125f, cost, 0.001);
452      cluster.setNumMovedRegions(2500);
453      cost = costFunction.cost();
454      assertEquals(0.25f, cost, 0.01);
455    }
456  }
457
458  @Test
459  public void testSkewCost() {
460    Configuration conf = HBaseConfiguration.create();
461    CostFunction costFunction = new RegionCountSkewCostFunction(conf);
462    for (int[] mockCluster : clusterStateMocks) {
463      costFunction.prepare(mockCluster(mockCluster));
464      double cost = costFunction.cost();
465      assertTrue(cost >= 0);
466      assertTrue(cost <= 1.01);
467    }
468
469    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 0, 1 }));
470    assertEquals(0, costFunction.cost(), 0.01);
471    costFunction.prepare(mockCluster(new int[] { 0, 0, 0, 1, 1 }));
472    assertEquals(0, costFunction.cost(), 0.01);
473    costFunction.prepare(mockCluster(new int[] { 0, 0, 1, 1, 1 }));
474    assertEquals(0, costFunction.cost(), 0.01);
475    costFunction.prepare(mockCluster(new int[] { 0, 1, 1, 1, 1 }));
476    assertEquals(0, costFunction.cost(), 0.01);
477    costFunction.prepare(mockCluster(new int[] { 1, 1, 1, 1, 1 }));
478    assertEquals(0, costFunction.cost(), 0.01);
479    costFunction.prepare(mockCluster(new int[] { 10000, 0, 0, 0, 0 }));
480    assertEquals(1, costFunction.cost(), 0.01);
481  }
482
483  @Test
484  public void testCostAfterUndoAction() {
485    final int runs = 10;
486    for (int[] mockCluster : clusterStateMocks) {
487      BalancerClusterState cluster = mockCluster(mockCluster);
488      loadBalancer.initCosts(cluster);
489      for (int i = 0; i != runs; ++i) {
490        final double expectedCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
491        BalanceAction action = loadBalancer.nextAction(cluster);
492        cluster.doAction(action);
493        loadBalancer.updateCostsAndWeightsWithAction(cluster, action);
494        BalanceAction undoAction = action.undoAction();
495        cluster.doAction(undoAction);
496        loadBalancer.updateCostsAndWeightsWithAction(cluster, undoAction);
497        final double actualCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
498        assertEquals(expectedCost, actualCost, 0);
499      }
500    }
501  }
502
503  @Test
504  public void testTableSkewCost() {
505    Configuration conf = HBaseConfiguration.create();
506    CostFunction costFunction = new TableSkewCostFunction(conf);
507    for (int[] mockCluster : clusterStateMocks) {
508      BalancerClusterState cluster = mockCluster(mockCluster);
509      costFunction.prepare(cluster);
510      double cost = costFunction.cost();
511      assertTrue(cost >= 0);
512      assertTrue(cost <= 1.01);
513    }
514  }
515
516  @Test
517  public void testRegionLoadCost() {
518    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
519    for (int i = 1; i < 5; i++) {
520      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
521      when(regionLoad.getReadRequestsCount()).thenReturn((long) i);
522      when(regionLoad.getCpRequestsCount()).thenReturn((long) i);
523      when(regionLoad.getStorefileSizeMB()).thenReturn(i);
524      regionLoads.add(regionLoad);
525    }
526
527    Configuration conf = HBaseConfiguration.create();
528    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
529    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
530    // read requests are treated as a rate so the average rate here is simply 1
531    assertEquals(1, rateResult, 0.01);
532
533    CPRequestCostFunction cpCostFunction = new CPRequestCostFunction(conf);
534    rateResult = cpCostFunction.getRegionLoadCost(regionLoads);
535    // coprocessor requests are treated as a rate so the average rate here is simply 1
536    assertEquals(1, rateResult, 0.01);
537
538    StoreFileCostFunction storeFileCostFunction = new StoreFileCostFunction(conf);
539    double result = storeFileCostFunction.getRegionLoadCost(regionLoads);
540    // storefile size cost is simply an average of it's value over time
541    assertEquals(2.5, result, 0.01);
542  }
543
544  @Test
545  public void testRegionLoadCostWhenDecrease() {
546    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
547    // test region loads of [1,2,1,4]
548    for (int i = 1; i < 5; i++) {
549      int load = i == 3 ? 1 : i;
550      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
551      when(regionLoad.getReadRequestsCount()).thenReturn((long) load);
552      when(regionLoad.getCpRequestsCount()).thenReturn((long) load);
553      regionLoads.add(regionLoad);
554    }
555
556    Configuration conf = HBaseConfiguration.create();
557    ReadRequestCostFunction readCostFunction = new ReadRequestCostFunction(conf);
558    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
559    // read requests are treated as a rate so the average rate here is simply 1
560    assertEquals(1.67, rateResult, 0.01);
561
562    CPRequestCostFunction cpCostFunction = new CPRequestCostFunction(conf);
563    rateResult = cpCostFunction.getRegionLoadCost(regionLoads);
564    // coprocessor requests are treated as a rate so the average rate here is simply 1
565    assertEquals(1.67, rateResult, 0.01);
566  }
567
568  @Test
569  public void testLosingRs() throws Exception {
570    int numNodes = 3;
571    int numRegions = 20;
572    int numRegionsPerServer = 3; // all servers except one
573    int replication = 1;
574    int numTables = 2;
575
576    Map<ServerName, List<RegionInfo>> serverMap =
577      createServerMap(numNodes, numRegions, numRegionsPerServer, replication, numTables);
578    List<ServerAndLoad> list = convertToList(serverMap);
579
580    List<RegionPlan> plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
581    assertNotNull(plans);
582
583    // Apply the plan to the mock cluster.
584    List<ServerAndLoad> balancedCluster = reconcile(list, plans, serverMap);
585
586    assertClusterAsBalanced(balancedCluster);
587
588    ServerName sn = serverMap.keySet().toArray(new ServerName[serverMap.size()])[0];
589
590    ServerName deadSn = ServerName.valueOf(sn.getHostname(), sn.getPort(), sn.getStartcode() - 100);
591
592    serverMap.put(deadSn, new ArrayList<>(0));
593
594    plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
595    assertNull(plans);
596  }
597
598  @Test
599  public void testAdditionalCostFunction() {
600    conf.set(StochasticLoadBalancer.COST_FUNCTIONS_COST_FUNCTIONS_KEY,
601      DummyCostFunction.class.getName());
602
603    loadBalancer.onConfigurationChange(conf);
604    assertTrue(Arrays.asList(loadBalancer.getCostFunctionNames())
605      .contains(DummyCostFunction.class.getSimpleName()));
606  }
607
608  @Test
609  public void testDefaultCostFunctionList() {
610    List<String> expected = Arrays.asList(RegionCountSkewCostFunction.class.getSimpleName(),
611      PrimaryRegionCountSkewCostFunction.class.getSimpleName(),
612      MoveCostFunction.class.getSimpleName(), RackLocalityCostFunction.class.getSimpleName(),
613      TableSkewCostFunction.class.getSimpleName(),
614      RegionReplicaHostCostFunction.class.getSimpleName(),
615      RegionReplicaRackCostFunction.class.getSimpleName(),
616      ReadRequestCostFunction.class.getSimpleName(), CPRequestCostFunction.class.getSimpleName(),
617      WriteRequestCostFunction.class.getSimpleName(),
618      MemStoreSizeCostFunction.class.getSimpleName(), StoreFileCostFunction.class.getSimpleName());
619
620    List<String> actual = Arrays.asList(loadBalancer.getCostFunctionNames());
621    assertTrue("ExpectedCostFunctions: " + expected + " ActualCostFunctions: " + actual,
622      CollectionUtils.isEqualCollection(expected, actual));
623  }
624
625  private boolean needsBalanceIdleRegion(int[] cluster) {
626    return Arrays.stream(cluster).anyMatch(x -> x > 1)
627      && Arrays.stream(cluster).anyMatch(x -> x < 1);
628  }
629
630  // This mock allows us to test the LocalityCostFunction
631  private class MockCluster extends BalancerClusterState {
632
633    private int[][] localities = null; // [region][server] = percent of blocks
634
635    public MockCluster(int[][] regions) {
636
637      // regions[0] is an array where index = serverIndex an value = number of regions
638      super(mockClusterServers(regions[0], 1), null, null, null);
639
640      localities = new int[regions.length - 1][];
641      for (int i = 1; i < regions.length; i++) {
642        int regionIndex = i - 1;
643        localities[regionIndex] = new int[regions[i].length - 1];
644        regionIndexToServerIndex[regionIndex] = regions[i][0];
645        for (int j = 1; j < regions[i].length; j++) {
646          int serverIndex = j - 1;
647          localities[regionIndex][serverIndex] =
648            regions[i][j] > 100 ? regions[i][j] % 100 : regions[i][j];
649        }
650      }
651    }
652
653    @Override
654    float getLocalityOfRegion(int region, int server) {
655      // convert the locality percentage to a fraction
656      return localities[region][server] / 100.0f;
657    }
658
659    @Override
660    public int getRegionSizeMB(int region) {
661      return 1;
662    }
663  }
664}