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.assertNotNull;
022import static org.junit.Assert.assertNull;
023import static org.junit.Assert.assertTrue;
024import static org.mockito.Mockito.mock;
025import static org.mockito.Mockito.when;
026
027import java.util.ArrayList;
028import java.util.Arrays;
029import java.util.List;
030import java.util.Map;
031import java.util.Queue;
032import java.util.TimeZone;
033import java.util.TreeMap;
034import org.apache.hadoop.conf.Configuration;
035import org.apache.hadoop.hbase.ClusterMetrics;
036import org.apache.hadoop.hbase.HBaseClassTestRule;
037import org.apache.hadoop.hbase.HBaseConfiguration;
038import org.apache.hadoop.hbase.HConstants;
039import org.apache.hadoop.hbase.RegionMetrics;
040import org.apache.hadoop.hbase.ServerMetrics;
041import org.apache.hadoop.hbase.ServerName;
042import org.apache.hadoop.hbase.Size;
043import org.apache.hadoop.hbase.TableName;
044import org.apache.hadoop.hbase.client.RegionInfo;
045import org.apache.hadoop.hbase.master.MockNoopMasterServices;
046import org.apache.hadoop.hbase.master.RegionPlan;
047import org.apache.hadoop.hbase.master.balancer.BaseLoadBalancer.Cluster;
048import org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer.ServerLocalityCostFunction;
049import org.apache.hadoop.hbase.testclassification.MasterTests;
050import org.apache.hadoop.hbase.testclassification.MediumTests;
051import org.apache.hadoop.hbase.util.Bytes;
052import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
053import org.junit.ClassRule;
054import org.junit.Test;
055import org.junit.experimental.categories.Category;
056
057@Category({ MasterTests.class, MediumTests.class })
058public class TestStochasticLoadBalancer extends BalancerTestBase {
059
060  @ClassRule
061  public static final HBaseClassTestRule CLASS_RULE =
062      HBaseClassTestRule.forClass(TestStochasticLoadBalancer.class);
063
064  private static final String REGION_KEY = "testRegion";
065
066  // Mapping of locality test -> expected locality
067  private float[] expectedLocalities = {1.0f, 0.0f, 0.50f, 0.25f, 1.0f};
068
069  /**
070   * Data set for testLocalityCost:
071   * [test][0][0] = mapping of server to number of regions it hosts
072   * [test][region + 1][0] = server that region is hosted on
073   * [test][region + 1][server + 1] = locality for region on server
074   */
075
076  private int[][][] clusterRegionLocationMocks = new int[][][]{
077
078      // Test 1: each region is entirely on server that hosts it
079      new int[][]{
080          new int[]{2, 1, 1},
081          new int[]{2, 0, 0, 100},   // region 0 is hosted and entirely local on server 2
082          new int[]{0, 100, 0, 0},   // region 1 is hosted and entirely on server 0
083          new int[]{0, 100, 0, 0},   // region 2 is hosted and entirely on server 0
084          new int[]{1, 0, 100, 0},   // region 1 is hosted and entirely on server 1
085      },
086
087      // Test 2: each region is 0% local on the server that hosts it
088      new int[][]{
089          new int[]{1, 2, 1},
090          new int[]{0, 0, 0, 100},   // region 0 is hosted and entirely local on server 2
091          new int[]{1, 100, 0, 0},   // region 1 is hosted and entirely on server 0
092          new int[]{1, 100, 0, 0},   // region 2 is hosted and entirely on server 0
093          new int[]{2, 0, 100, 0},   // region 1 is hosted and entirely on server 1
094      },
095
096      // Test 3: each region is 25% local on the server that hosts it (and 50% locality is possible)
097      new int[][]{
098          new int[]{1, 2, 1},
099          new int[]{0, 25, 0, 50},   // region 0 is hosted and entirely local on server 2
100          new int[]{1, 50, 25, 0},   // region 1 is hosted and entirely on server 0
101          new int[]{1, 50, 25, 0},   // region 2 is hosted and entirely on server 0
102          new int[]{2, 0, 50, 25},   // region 1 is hosted and entirely on server 1
103      },
104
105      // Test 4: each region is 25% local on the server that hosts it (and 100% locality is possible)
106      new int[][]{
107          new int[]{1, 2, 1},
108          new int[]{0, 25, 0, 100},   // region 0 is hosted and entirely local on server 2
109          new int[]{1, 100, 25, 0},   // region 1 is hosted and entirely on server 0
110          new int[]{1, 100, 25, 0},   // region 2 is hosted and entirely on server 0
111          new int[]{2, 0, 100, 25},   // region 1 is hosted and entirely on server 1
112      },
113
114      // Test 5: each region is 75% local on the server that hosts it (and 75% locality is possible everywhere)
115      new int[][]{
116          new int[]{1, 2, 1},
117          new int[]{0, 75, 75, 75},   // region 0 is hosted and entirely local on server 2
118          new int[]{1, 75, 75, 75},   // region 1 is hosted and entirely on server 0
119          new int[]{1, 75, 75, 75},   // region 2 is hosted and entirely on server 0
120          new int[]{2, 75, 75, 75},   // region 1 is hosted and entirely on server 1
121      },
122  };
123
124  @Test
125  public void testKeepRegionLoad() throws Exception {
126
127    ServerName sn = ServerName.valueOf("test:8080", 100);
128    int numClusterStatusToAdd = 20000;
129    for (int i = 0; i < numClusterStatusToAdd; i++) {
130      ServerMetrics sl = mock(ServerMetrics.class);
131
132      RegionMetrics rl = mock(RegionMetrics.class);
133      when(rl.getReadRequestCount()).thenReturn(0L);
134      when(rl.getWriteRequestCount()).thenReturn(0L);
135      when(rl.getMemStoreSize()).thenReturn(Size.ZERO);
136      when(rl.getStoreFileSize()).thenReturn(new Size(i, Size.Unit.MEGABYTE));
137
138      Map<byte[], RegionMetrics> regionLoadMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
139      regionLoadMap.put(Bytes.toBytes(REGION_KEY), rl);
140      when(sl.getRegionMetrics()).thenReturn(regionLoadMap);
141
142      ClusterMetrics clusterStatus = mock(ClusterMetrics.class);
143      Map<ServerName, ServerMetrics> serverMetricsMap = new TreeMap<>();
144      serverMetricsMap.put(sn, sl);
145      when(clusterStatus.getLiveServerMetrics()).thenReturn(serverMetricsMap);
146//      when(clusterStatus.getLoad(sn)).thenReturn(sl);
147
148      loadBalancer.setClusterMetrics(clusterStatus);
149    }
150
151    String regionNameAsString = RegionInfo.getRegionNameAsString(Bytes.toBytes(REGION_KEY));
152    assertTrue(loadBalancer.loads.get(regionNameAsString) != null);
153    assertTrue(loadBalancer.loads.get(regionNameAsString).size() == 15);
154    assertNull(loadBalancer.namedQueueRecorder);
155
156    Queue<BalancerRegionLoad> loads = loadBalancer.loads.get(regionNameAsString);
157    int i = 0;
158    while(loads.size() > 0) {
159      BalancerRegionLoad rl = loads.remove();
160      assertEquals(i + (numClusterStatusToAdd - 15), rl.getStorefileSizeMB());
161      i ++;
162    }
163  }
164
165  @Test
166  public void testNeedBalance() {
167    float minCost = conf.getFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 0.05f);
168    conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", 1.0f);
169    try {
170      // Test with/without per table balancer.
171      boolean[] perTableBalancerConfigs = {true, false};
172      for (boolean isByTable : perTableBalancerConfigs) {
173        conf.setBoolean(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE, isByTable);
174        loadBalancer.setConf(conf);
175        for (int[] mockCluster : clusterStateMocks) {
176          Map<ServerName, List<RegionInfo>> servers = mockClusterServers(mockCluster);
177          Map<TableName, Map<ServerName, List<RegionInfo>>> LoadOfAllTable =
178              (Map) mockClusterServersWithTables(servers);
179          List<RegionPlan> plans = loadBalancer.balanceCluster(LoadOfAllTable);
180          boolean emptyPlans = plans == null || plans.isEmpty();
181          assertTrue(emptyPlans || needsBalanceIdleRegion(mockCluster));
182        }
183      }
184    } finally {
185      // reset config
186      conf.unset(HConstants.HBASE_MASTER_LOADBALANCE_BYTABLE);
187      conf.setFloat("hbase.master.balancer.stochastic.minCostNeedBalance", minCost);
188      loadBalancer.setConf(conf);
189    }
190  }
191
192  @Test
193  public void testLocalityCost() throws Exception {
194    Configuration conf = HBaseConfiguration.create();
195    MockNoopMasterServices master = new MockNoopMasterServices();
196    StochasticLoadBalancer.CostFunction
197        costFunction = new ServerLocalityCostFunction(conf, master);
198
199    for (int test = 0; test < clusterRegionLocationMocks.length; test++) {
200      int[][] clusterRegionLocations = clusterRegionLocationMocks[test];
201      MockCluster cluster = new MockCluster(clusterRegionLocations);
202      costFunction.init(cluster);
203      double cost = costFunction.cost();
204      double expected = 1 - expectedLocalities[test];
205      assertEquals(expected, cost, 0.001);
206    }
207    assertNull(loadBalancer.namedQueueRecorder);
208  }
209
210  @Test
211  public void testMoveCostMultiplier() throws Exception {
212    Configuration conf = HBaseConfiguration.create();
213    StochasticLoadBalancer.CostFunction
214      costFunction = new StochasticLoadBalancer.MoveCostFunction(conf);
215    BaseLoadBalancer.Cluster cluster = mockCluster(clusterStateMocks[0]);
216    costFunction.init(cluster);
217    costFunction.cost();
218    assertEquals(StochasticLoadBalancer.MoveCostFunction.DEFAULT_MOVE_COST,
219      costFunction.getMultiplier(), 0.01);
220
221    // In offpeak hours, the multiplier of move cost should be lower
222    conf.setInt("hbase.offpeak.start.hour",0);
223    conf.setInt("hbase.offpeak.end.hour",23);
224    // Set a fixed time which hour is 15, so it will always in offpeak
225    // See HBASE-24898 for more info of the calculation here
226    long deltaFor15 = TimeZone.getDefault().getRawOffset() - 28800000;
227    long timeFor15 = 1597907081000L - deltaFor15;
228    EnvironmentEdgeManager.injectEdge(() -> timeFor15);
229    costFunction = new StochasticLoadBalancer.MoveCostFunction(conf);
230    costFunction.init(cluster);
231    costFunction.cost();
232    assertEquals(StochasticLoadBalancer.MoveCostFunction.DEFAULT_MOVE_COST_OFFPEAK
233      , costFunction.getMultiplier(), 0.01);
234  }
235
236  @Test
237  public void testMoveCost() throws Exception {
238    Configuration conf = HBaseConfiguration.create();
239    StochasticLoadBalancer.CostFunction
240        costFunction = new StochasticLoadBalancer.MoveCostFunction(conf);
241    for (int[] mockCluster : clusterStateMocks) {
242      BaseLoadBalancer.Cluster cluster = mockCluster(mockCluster);
243      costFunction.init(cluster);
244      double cost = costFunction.cost();
245      assertEquals(0.0f, cost, 0.001);
246
247      // cluster region number is smaller than maxMoves=600
248      cluster.setNumRegions(200);
249      cluster.setNumMovedRegions(10);
250      cost = costFunction.cost();
251      assertEquals(0.05f, cost, 0.001);
252      cluster.setNumMovedRegions(100);
253      cost = costFunction.cost();
254      assertEquals(0.5f, cost, 0.001);
255      cluster.setNumMovedRegions(200);
256      cost = costFunction.cost();
257      assertEquals(1.0f, cost, 0.001);
258
259
260      // cluster region number is bigger than maxMoves=2500
261      cluster.setNumRegions(10000);
262      cluster.setNumMovedRegions(250);
263      cost = costFunction.cost();
264      assertEquals(0.1f, cost, 0.001);
265      cluster.setNumMovedRegions(1250);
266      cost = costFunction.cost();
267      assertEquals(0.5f, cost, 0.001);
268      cluster.setNumMovedRegions(2500);
269      cost = costFunction.cost();
270      assertEquals(1.0f, cost, 0.01);
271    }
272  }
273
274  @Test
275  public void testSkewCost() {
276    Configuration conf = HBaseConfiguration.create();
277    StochasticLoadBalancer.CostFunction
278        costFunction = new StochasticLoadBalancer.RegionCountSkewCostFunction(conf);
279    for (int[] mockCluster : clusterStateMocks) {
280      costFunction.init(mockCluster(mockCluster));
281      double cost = costFunction.cost();
282      assertTrue(cost >= 0);
283      assertTrue(cost <= 1.01);
284    }
285
286    costFunction.init(mockCluster(new int[]{0, 0, 0, 0, 1}));
287    assertEquals(0,costFunction.cost(), 0.01);
288    costFunction.init(mockCluster(new int[]{0, 0, 0, 1, 1}));
289    assertEquals(0, costFunction.cost(), 0.01);
290    costFunction.init(mockCluster(new int[]{0, 0, 1, 1, 1}));
291    assertEquals(0, costFunction.cost(), 0.01);
292    costFunction.init(mockCluster(new int[]{0, 1, 1, 1, 1}));
293    assertEquals(0, costFunction.cost(), 0.01);
294    costFunction.init(mockCluster(new int[]{1, 1, 1, 1, 1}));
295    assertEquals(0, costFunction.cost(), 0.01);
296    costFunction.init(mockCluster(new int[]{10000, 0, 0, 0, 0}));
297    assertEquals(1, costFunction.cost(), 0.01);
298  }
299
300  @Test
301  public void testCostAfterUndoAction() {
302    final int runs = 10;
303    loadBalancer.setConf(conf);
304    for (int[] mockCluster : clusterStateMocks) {
305      BaseLoadBalancer.Cluster cluster = mockCluster(mockCluster);
306      loadBalancer.initCosts(cluster);
307      for (int i = 0; i != runs; ++i) {
308        final double expectedCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
309        Cluster.Action action = loadBalancer.nextAction(cluster);
310        cluster.doAction(action);
311        loadBalancer.updateCostsWithAction(cluster, action);
312        Cluster.Action undoAction = action.undoAction();
313        cluster.doAction(undoAction);
314        loadBalancer.updateCostsWithAction(cluster, undoAction);
315        final double actualCost = loadBalancer.computeCost(cluster, Double.MAX_VALUE);
316        assertEquals(expectedCost, actualCost, 0);
317      }
318    }
319  }
320
321  @Test
322  public void testTableSkewCost() {
323    Configuration conf = HBaseConfiguration.create();
324    StochasticLoadBalancer.CostFunction
325        costFunction = new StochasticLoadBalancer.TableSkewCostFunction(conf);
326    for (int[] mockCluster : clusterStateMocks) {
327      BaseLoadBalancer.Cluster cluster = mockCluster(mockCluster);
328      costFunction.init(cluster);
329      double cost = costFunction.cost();
330      assertTrue(cost >= 0);
331      assertTrue(cost <= 1.01);
332    }
333  }
334
335  @Test
336  public void testRegionLoadCost() {
337    List<BalancerRegionLoad> regionLoads = new ArrayList<>();
338    for (int i = 1; i < 5; i++) {
339      BalancerRegionLoad regionLoad = mock(BalancerRegionLoad.class);
340      when(regionLoad.getReadRequestsCount()).thenReturn(new Long(i));
341      when(regionLoad.getStorefileSizeMB()).thenReturn(i);
342      regionLoads.add(regionLoad);
343    }
344
345    Configuration conf = HBaseConfiguration.create();
346    StochasticLoadBalancer.ReadRequestCostFunction readCostFunction =
347        new StochasticLoadBalancer.ReadRequestCostFunction(conf);
348    double rateResult = readCostFunction.getRegionLoadCost(regionLoads);
349    // read requests are treated as a rate so the average rate here is simply 1
350    assertEquals(1, rateResult, 0.01);
351
352    StochasticLoadBalancer.StoreFileCostFunction storeFileCostFunction =
353        new StochasticLoadBalancer.StoreFileCostFunction(conf);
354    double result = storeFileCostFunction.getRegionLoadCost(regionLoads);
355    // storefile size cost is simply an average of it's value over time
356    assertEquals(2.5, result, 0.01);
357  }
358
359  @Test
360  public void testCostFromArray() {
361    Configuration conf = HBaseConfiguration.create();
362    StochasticLoadBalancer.CostFromRegionLoadFunction
363        costFunction = new StochasticLoadBalancer.MemStoreSizeCostFunction(conf);
364    costFunction.init(mockCluster(new int[]{0, 0, 0, 0, 1}));
365
366    double[] statOne = new double[100];
367    for (int i =0; i < 100; i++) {
368      statOne[i] = 10;
369    }
370    assertEquals(0, costFunction.costFromArray(statOne), 0.01);
371
372    double[] statTwo= new double[101];
373    for (int i =0; i < 100; i++) {
374      statTwo[i] = 0;
375    }
376    statTwo[100] = 100;
377    assertEquals(1, costFunction.costFromArray(statTwo), 0.01);
378
379    double[] statThree = new double[200];
380    for (int i =0; i < 100; i++) {
381      statThree[i] = (0);
382      statThree[i+100] = 100;
383    }
384    assertEquals(0.5, costFunction.costFromArray(statThree), 0.01);
385  }
386
387  @Test
388  public void testLosingRs() throws Exception {
389    int numNodes = 3;
390    int numRegions = 20;
391    int numRegionsPerServer = 3; //all servers except one
392    int replication = 1;
393    int numTables = 2;
394
395    Map<ServerName, List<RegionInfo>> serverMap =
396        createServerMap(numNodes, numRegions, numRegionsPerServer, replication, numTables);
397    List<ServerAndLoad> list = convertToList(serverMap);
398
399
400    List<RegionPlan> plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
401    assertNotNull(plans);
402
403    // Apply the plan to the mock cluster.
404    List<ServerAndLoad> balancedCluster = reconcile(list, plans, serverMap);
405
406    assertClusterAsBalanced(balancedCluster);
407
408    ServerName sn = serverMap.keySet().toArray(new ServerName[serverMap.size()])[0];
409
410    ServerName deadSn = ServerName.valueOf(sn.getHostname(), sn.getPort(), sn.getStartcode() - 100);
411
412    serverMap.put(deadSn, new ArrayList<>(0));
413
414    plans = loadBalancer.balanceTable(HConstants.ENSEMBLE_TABLE_NAME, serverMap);
415    assertNull(plans);
416  }
417
418  @Test
419  public void testAdditionalCostFunction() {
420    conf.set(StochasticLoadBalancer.COST_FUNCTIONS_COST_FUNCTIONS_KEY,
421            DummyCostFunction.class.getName());
422
423    loadBalancer.setConf(conf);
424    assertTrue(Arrays.
425            asList(loadBalancer.getCostFunctionNames()).
426            contains(DummyCostFunction.class.getSimpleName()));
427  }
428
429  private boolean needsBalanceIdleRegion(int[] cluster){
430    return (Arrays.stream(cluster).anyMatch(x -> x>1)) && (Arrays.stream(cluster).anyMatch(x -> x<1));
431  }
432
433  // This mock allows us to test the LocalityCostFunction
434  private class MockCluster extends BaseLoadBalancer.Cluster {
435
436    private int[][] localities = null;   // [region][server] = percent of blocks
437
438    public MockCluster(int[][] regions) {
439
440      // regions[0] is an array where index = serverIndex an value = number of regions
441      super(mockClusterServers(regions[0], 1), null, null, null);
442
443      localities = new int[regions.length - 1][];
444      for (int i = 1; i < regions.length; i++) {
445        int regionIndex = i - 1;
446        localities[regionIndex] = new int[regions[i].length - 1];
447        regionIndexToServerIndex[regionIndex] = regions[i][0];
448        for (int j = 1; j < regions[i].length; j++) {
449          int serverIndex = j - 1;
450          localities[regionIndex][serverIndex] = regions[i][j] > 100 ? regions[i][j] % 100 : regions[i][j];
451        }
452      }
453    }
454
455    @Override
456    float getLocalityOfRegion(int region, int server) {
457      // convert the locality percentage to a fraction
458      return localities[region][server] / 100.0f;
459    }
460
461    @Override
462    public int getRegionSizeMB(int region) {
463      return 1;
464    }
465  }
466}