All Downloads are FREE. Search and download functionalities are using the official Maven repository.

org.apache.hadoop.hbase.rsgroup.RSGroupBasedLoadBalancer Maven / Gradle / Ivy

There is a newer version: 2.6.0-hadoop3
Show newest version
/*
 * Licensed to the Apache Software Foundation (ASF) under one
 * or more contributor license agreements.  See the NOTICE file
 * distributed with this work for additional information
 * regarding copyright ownership.  The ASF licenses this file
 * to you under the Apache License, Version 2.0 (the
 * "License"); you may not use this file except in compliance
 * with the License.  You may obtain a copy of the License at
 *
 *     http://www.apache.org/licenses/LICENSE-2.0
 *
 * Unless required by applicable law or agreed to in writing, software
 * distributed under the License is distributed on an "AS IS" BASIS,
 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
 * See the License for the specific language governing permissions and
 * limitations under the License.
 */
package org.apache.hadoop.hbase.rsgroup;

import com.google.errorprone.annotations.RestrictedApi;
import edu.umd.cs.findbugs.annotations.NonNull;
import java.io.IOException;
import java.util.ArrayList;
import java.util.Collections;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.Optional;
import java.util.Set;
import java.util.TreeMap;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.ClusterMetrics;
import org.apache.hadoop.hbase.HBaseIOException;
import org.apache.hadoop.hbase.ServerName;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.client.RegionInfo;
import org.apache.hadoop.hbase.constraint.ConstraintException;
import org.apache.hadoop.hbase.master.LoadBalancer;
import org.apache.hadoop.hbase.master.MasterServices;
import org.apache.hadoop.hbase.master.RegionPlan;
import org.apache.hadoop.hbase.master.balancer.LoadBalancerFactory;
import org.apache.hadoop.hbase.master.balancer.StochasticLoadBalancer;
import org.apache.hadoop.hbase.net.Address;
import org.apache.hadoop.hbase.util.Pair;
import org.apache.hadoop.hbase.util.ReflectionUtils;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import org.apache.hbase.thirdparty.com.google.common.collect.ArrayListMultimap;
import org.apache.hbase.thirdparty.com.google.common.collect.ListMultimap;
import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
import org.apache.hbase.thirdparty.com.google.common.collect.Maps;

/**
 * GroupBasedLoadBalancer, used when Region Server Grouping is configured (HBase-6721) It does
 * region balance based on a table's group membership. Most assignment methods contain two exclusive
 * code paths: Online - when the group table is online and Offline - when it is unavailable. During
 * Offline, assignments are assigned based on cached information in zookeeper. If unavailable (ie
 * bootstrap) then regions are assigned randomly. Once the GROUP table has been assigned, the
 * balancer switches to Online and will then start providing appropriate assignments for user
 * tables.
 */
@InterfaceAudience.Private
public class RSGroupBasedLoadBalancer implements RSGroupableBalancer {
  private static final Logger LOG = LoggerFactory.getLogger(RSGroupBasedLoadBalancer.class);

  private MasterServices masterServices;
  private volatile RSGroupInfoManager rsGroupInfoManager;
  private volatile LoadBalancer internalBalancer;

  /**
   * Set this key to {@code true} to allow region fallback. Fallback to the default rsgroup first,
   * then fallback to any group if no online servers in default rsgroup. Please keep balancer switch
   * on at the same time, which is relied on to correct misplaced regions
   */
  public static final String FALLBACK_GROUP_ENABLE_KEY = "hbase.rsgroup.fallback.enable";

  private volatile boolean fallbackEnabled = false;

  /**
   * Used by reflection in {@link org.apache.hadoop.hbase.master.balancer.LoadBalancerFactory}.
   */
  @InterfaceAudience.Private
  public RSGroupBasedLoadBalancer() {
  }

  // must be called after calling initialize
  @Override
  public synchronized void updateClusterMetrics(ClusterMetrics sm) {
    assert internalBalancer != null;
    internalBalancer.updateClusterMetrics(sm);
  }

  @Override
  public synchronized void
    updateBalancerLoadInfo(Map>> loadOfAllTable) {
    internalBalancer.updateBalancerLoadInfo(loadOfAllTable);
  }

  public void setMasterServices(MasterServices masterServices) {
    this.masterServices = masterServices;
  }

  @RestrictedApi(explanation = "Should only be called in tests", link = "",
      allowedOnPath = ".*/src/test/.*")
  public void setRsGroupInfoManager(RSGroupInfoManager rsGroupInfoManager) {
    this.rsGroupInfoManager = rsGroupInfoManager;
  }

  /**
   * Balance by RSGroup.
   */
  @Override
  public synchronized List balanceCluster(
    Map>> loadOfAllTable) throws IOException {
    if (!isOnline()) {
      throw new ConstraintException(
        RSGroupInfoManager.RSGROUP_TABLE_NAME + " is not online, unable to perform balance");
    }
    // Calculate correct assignments and a list of RegionPlan for mis-placed regions
    Pair>>,
      List> correctedStateAndRegionPlans = correctAssignments(loadOfAllTable);
    Map>> correctedLoadOfAllTable =
      correctedStateAndRegionPlans.getFirst();
    List regionPlans = correctedStateAndRegionPlans.getSecond();
    // Add RegionPlan for the regions which have been placed according to the region server group
    // assignment into the movement list
    try {
      // For each rsgroup
      for (RSGroupInfo rsgroup : rsGroupInfoManager.listRSGroups()) {
        Map>> loadOfTablesInGroup = new HashMap<>();
        for (Map.Entry>> entry : correctedLoadOfAllTable
          .entrySet()) {
          TableName tableName = entry.getKey();
          String targetRSGroupName = rsGroupInfoManager.getRSGroupOfTable(tableName);
          if (targetRSGroupName == null) {
            targetRSGroupName = RSGroupInfo.DEFAULT_GROUP;
          }
          if (targetRSGroupName.equals(rsgroup.getName())) {
            loadOfTablesInGroup.put(tableName, entry.getValue());
          }
        }
        List groupPlans = null;
        if (!loadOfTablesInGroup.isEmpty()) {
          LOG.info("Start Generate Balance plan for group: " + rsgroup.getName());
          groupPlans = this.internalBalancer.balanceCluster(loadOfTablesInGroup);
        }
        if (groupPlans != null) {
          regionPlans.addAll(groupPlans);
        }
      }
    } catch (IOException exp) {
      LOG.warn("Exception while balancing cluster.", exp);
      regionPlans.clear();
    }
    return regionPlans;
  }

  @Override
  @NonNull
  public Map> roundRobinAssignment(List regions,
    List servers) throws HBaseIOException {
    Map> assignments = Maps.newHashMap();
    List, List>> pairs =
      generateGroupAssignments(regions, servers);
    for (Pair, List> pair : pairs) {
      Map> result =
        this.internalBalancer.roundRobinAssignment(pair.getFirst(), pair.getSecond());
      result.forEach((server, regionInfos) -> assignments
        .computeIfAbsent(server, s -> Lists.newArrayList()).addAll(regionInfos));
    }
    return assignments;
  }

  @Override
  @NonNull
  public Map> retainAssignment(Map regions,
    List servers) throws HBaseIOException {
    try {
      Map> assignments = new TreeMap<>();
      List, List>> pairs =
        generateGroupAssignments(Lists.newArrayList(regions.keySet()), servers);
      for (Pair, List> pair : pairs) {
        List regionList = pair.getFirst();
        Map currentAssignmentMap = Maps.newTreeMap();
        regionList.forEach(r -> currentAssignmentMap.put(r, regions.get(r)));
        Map> pairResult =
          this.internalBalancer.retainAssignment(currentAssignmentMap, pair.getSecond());
        pairResult.forEach((server, rs) -> assignments
          .computeIfAbsent(server, s -> Lists.newArrayList()).addAll(rs));
      }
      return assignments;
    } catch (IOException e) {
      throw new HBaseIOException("Failed to do online retain assignment", e);
    }
  }

  @Override
  public ServerName randomAssignment(RegionInfo region, List servers)
    throws HBaseIOException {
    List, List>> pairs =
      generateGroupAssignments(Lists.newArrayList(region), servers);
    List filteredServers = pairs.iterator().next().getSecond();
    return this.internalBalancer.randomAssignment(region, filteredServers);
  }

  private List, List>> generateGroupAssignments(
    List regions, List servers) throws HBaseIOException {
    try {
      ListMultimap regionMap = ArrayListMultimap.create();
      ListMultimap serverMap = ArrayListMultimap.create();
      RSGroupInfo defaultInfo = rsGroupInfoManager.getRSGroup(RSGroupInfo.DEFAULT_GROUP);
      for (RegionInfo region : regions) {
        String groupName =
          Optional.ofNullable(rsGroupInfoManager.getRSGroupOfTable(region.getTable()))
            .orElse(defaultInfo.getName());
        regionMap.put(groupName, region);
      }
      for (String groupKey : regionMap.keySet()) {
        RSGroupInfo info = rsGroupInfoManager.getRSGroup(groupKey);
        serverMap.putAll(groupKey, filterOfflineServers(info, servers));
      }

      List, List>> result = Lists.newArrayList();
      List fallbackRegions = Lists.newArrayList();
      for (String groupKey : regionMap.keySet()) {
        if (serverMap.get(groupKey).isEmpty()) {
          fallbackRegions.addAll(regionMap.get(groupKey));
        } else {
          result.add(Pair.newPair(regionMap.get(groupKey), serverMap.get(groupKey)));
        }
      }
      if (!fallbackRegions.isEmpty()) {
        List candidates = null;
        if (isFallbackEnabled()) {
          candidates = getFallBackCandidates(servers);
        }
        candidates = (candidates == null || candidates.isEmpty())
          ? Lists.newArrayList(BOGUS_SERVER_NAME)
          : candidates;
        result.add(Pair.newPair(fallbackRegions, candidates));
      }
      return result;
    } catch (IOException e) {
      throw new HBaseIOException("Failed to generate group assignments", e);
    }
  }

  private List filterOfflineServers(RSGroupInfo RSGroupInfo,
    List onlineServers) {
    if (RSGroupInfo != null) {
      return filterServers(RSGroupInfo.getServers(), onlineServers);
    } else {
      LOG.warn("RSGroup Information found to be null. Some regions might be unassigned.");
      return Collections.emptyList();
    }
  }

  /**
   * Filter servers based on the online servers.
   * 

* servers is actually a TreeSet (see {@link org.apache.hadoop.hbase.rsgroup.RSGroupInfo}), having * its contains()'s time complexity as O(logn), which is good enough. *

* TODO: consider using HashSet to pursue O(1) for contains() throughout the calling chain if * needed. * @param servers the servers * @param onlineServers List of servers which are online. * @return the list */ private List filterServers(Set

servers, List onlineServers) { ArrayList finalList = new ArrayList<>(); for (ServerName onlineServer : onlineServers) { if (servers.contains(onlineServer.getAddress())) { finalList.add(onlineServer); } } return finalList; } private Pair>>, List> correctAssignments(Map>> existingAssignments) throws IOException { // To return Map>> correctAssignments = new HashMap<>(); List regionPlansForMisplacedRegions = new ArrayList<>(); for (Map.Entry>> assignments : existingAssignments .entrySet()) { TableName tableName = assignments.getKey(); Map> clusterLoad = assignments.getValue(); Map> correctServerRegion = new TreeMap<>(); RSGroupInfo targetRSGInfo = null; try { String groupName = rsGroupInfoManager.getRSGroupOfTable(tableName); if (groupName == null) { LOG.debug("Group not found for table " + tableName + ", using default"); groupName = RSGroupInfo.DEFAULT_GROUP; } targetRSGInfo = rsGroupInfoManager.getRSGroup(groupName); } catch (IOException exp) { LOG.debug("RSGroup information null for region of table " + tableName, exp); } for (Map.Entry> serverRegionMap : clusterLoad.entrySet()) { ServerName currentHostServer = serverRegionMap.getKey(); List regionInfoList = serverRegionMap.getValue(); if ( targetRSGInfo == null || !targetRSGInfo.containsServer(currentHostServer.getAddress()) ) { regionInfoList.forEach(regionInfo -> { regionPlansForMisplacedRegions.add(new RegionPlan(regionInfo, currentHostServer, null)); }); } else { correctServerRegion.put(currentHostServer, regionInfoList); } } correctAssignments.put(tableName, correctServerRegion); } // Return correct assignments and region movement plan for mis-placed regions together return new Pair>>, List>( correctAssignments, regionPlansForMisplacedRegions); } @Override public void initialize() throws HBaseIOException { try { if (rsGroupInfoManager == null) { List cps = masterServices.getMasterCoprocessorHost().findCoprocessors(RSGroupAdminEndpoint.class); if (cps.size() != 1) { String msg = "Expected one implementation of GroupAdminEndpoint but found " + cps.size(); LOG.error(msg); throw new HBaseIOException(msg); } rsGroupInfoManager = cps.get(0).getGroupInfoManager(); if (rsGroupInfoManager == null) { String msg = "RSGroupInfoManager hasn't been initialized"; LOG.error(msg); throw new HBaseIOException(msg); } rsGroupInfoManager.start(); } } catch (IOException e) { throw new HBaseIOException("Failed to initialize GroupInfoManagerImpl", e); } Configuration conf = masterServices.getConfiguration(); // Create the balancer Class balancerClass = conf.getClass(HBASE_RSGROUP_LOADBALANCER_CLASS, StochasticLoadBalancer.class, LoadBalancer.class); if (this.getClass().isAssignableFrom(balancerClass)) { LOG.warn("The internal balancer of RSGroupBasedLoadBalancer cannot be itself, " + "falling back to the default LoadBalancer class"); balancerClass = LoadBalancerFactory.getDefaultLoadBalancerClass(); } internalBalancer = ReflectionUtils.newInstance(balancerClass); internalBalancer.setMasterServices(masterServices); internalBalancer.initialize(); // init fallback groups this.fallbackEnabled = conf.getBoolean(FALLBACK_GROUP_ENABLE_KEY, false); } public boolean isOnline() { if (this.rsGroupInfoManager == null) { return false; } return this.rsGroupInfoManager.isOnline(); } public boolean isFallbackEnabled() { return fallbackEnabled; } @Override public void regionOnline(RegionInfo regionInfo, ServerName sn) { } @Override public void regionOffline(RegionInfo regionInfo) { } @Override public synchronized void onConfigurationChange(Configuration conf) { boolean newFallbackEnabled = conf.getBoolean(FALLBACK_GROUP_ENABLE_KEY, false); if (fallbackEnabled != newFallbackEnabled) { LOG.info("Changing the value of {} from {} to {}", FALLBACK_GROUP_ENABLE_KEY, fallbackEnabled, newFallbackEnabled); fallbackEnabled = newFallbackEnabled; } internalBalancer.onConfigurationChange(conf); } @Override public void stop(String why) { internalBalancer.stop(why); } @Override public boolean isStopped() { return internalBalancer.isStopped(); } @Override public synchronized void postMasterStartupInitialize() { this.internalBalancer.postMasterStartupInitialize(); } public void updateBalancerStatus(boolean status) { internalBalancer.updateBalancerStatus(status); } private List getFallBackCandidates(List servers) { List serverNames = null; try { RSGroupInfo info = rsGroupInfoManager.getRSGroup(RSGroupInfo.DEFAULT_GROUP); serverNames = filterOfflineServers(info, servers); } catch (IOException e) { LOG.error("Failed to get default rsgroup info to fallback", e); } return serverNames == null || serverNames.isEmpty() ? servers : serverNames; } }




© 2015 - 2024 Weber Informatics LLC | Privacy Policy