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