001/*
002 * Licensed to the Apache Software Foundation (ASF) under one or more
003 * contributor license agreements.  See the NOTICE file distributed with
004 * this work for additional information regarding copyright ownership.
005 * The ASF licenses this file to you under the Apache License, Version 2.0
006 * (the "License"); you may not use this file except in compliance with
007 * the License.  You may obtain a copy of the License at
008 *
009 * http://www.apache.org/licenses/LICENSE-2.0
010 *
011 * Unless required by applicable law or agreed to in writing, software
012 * distributed under the License is distributed on an "AS IS" BASIS,
013 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
014 * See the License for the specific language governing permissions and
015 * limitations under the License.
016 */
017package org.apache.hadoop.hbase.quotas;
018
019import java.util.HashSet;
020import java.util.Iterator;
021import java.util.List;
022import java.util.Map.Entry;
023import java.util.Set;
024import java.util.concurrent.TimeUnit;
025
026import org.apache.hadoop.conf.Configuration;
027import org.apache.hadoop.hbase.ScheduledChore;
028import org.apache.hadoop.hbase.client.RegionInfo;
029import org.apache.hadoop.hbase.regionserver.MetricsRegionServer;
030import org.apache.hadoop.hbase.regionserver.Region;
031import org.apache.hadoop.hbase.regionserver.RegionServerServices;
032import org.apache.yetus.audience.InterfaceAudience;
033import org.slf4j.Logger;
034import org.slf4j.LoggerFactory;
035
036/**
037 * A Chore which sends the region size reports on this RegionServer to the Master.
038 */
039@InterfaceAudience.Private
040public class RegionSizeReportingChore extends ScheduledChore {
041  private static final Logger LOG = LoggerFactory.getLogger(RegionSizeReportingChore.class);
042
043  static final String REGION_SIZE_REPORTING_CHORE_PERIOD_KEY =
044      "hbase.regionserver.quotas.region.size.reporting.chore.period";
045  static final int REGION_SIZE_REPORTING_CHORE_PERIOD_DEFAULT = 1000 * 60;
046
047  static final String REGION_SIZE_REPORTING_CHORE_DELAY_KEY =
048      "hbase.regionserver.quotas.region.size.reporting.chore.delay";
049  static final long REGION_SIZE_REPORTING_CHORE_DELAY_DEFAULT = 1000 * 30;
050
051  static final String REGION_SIZE_REPORTING_CHORE_TIMEUNIT_KEY =
052      "hbase.regionserver.quotas.region.size.reporting.chore.timeunit";
053  static final String REGION_SIZE_REPORTING_CHORE_TIMEUNIT_DEFAULT = TimeUnit.MILLISECONDS.name();
054
055  private final RegionServerServices rsServices;
056  private final MetricsRegionServer metrics;
057
058  public RegionSizeReportingChore(RegionServerServices rsServices) {
059    super(
060        RegionSizeReportingChore.class.getSimpleName(), rsServices,
061        getPeriod(rsServices.getConfiguration()), getInitialDelay(rsServices.getConfiguration()),
062        getTimeUnit(rsServices.getConfiguration()));
063    this.rsServices = rsServices;
064    this.metrics = rsServices.getMetrics();
065  }
066
067  @Override
068  protected void chore() {
069    final long start = System.nanoTime();
070    try {
071      _chore();
072    } finally {
073      if (metrics != null) {
074        metrics.incrementRegionSizeReportingChoreTime(
075            TimeUnit.MILLISECONDS.convert(System.nanoTime() - start, TimeUnit.NANOSECONDS));
076      }
077    }
078  }
079
080  void _chore() {
081    final RegionServerSpaceQuotaManager quotaManager =
082        rsServices.getRegionServerSpaceQuotaManager();
083    // Get the HRegionInfo for each online region
084    HashSet<RegionInfo> onlineRegionInfos = getOnlineRegionInfos(rsServices.getRegions());
085    RegionSizeStore store = quotaManager.getRegionSizeStore();
086    // Remove all sizes for non-online regions
087    removeNonOnlineRegions(store, onlineRegionInfos);
088    rsServices.reportRegionSizesForQuotas(store);
089  }
090
091  HashSet<RegionInfo> getOnlineRegionInfos(List<? extends Region> onlineRegions) {
092    HashSet<RegionInfo> regionInfos = new HashSet<>();
093    onlineRegions.forEach((region) -> regionInfos.add(region.getRegionInfo()));
094    return regionInfos;
095  }
096
097  void removeNonOnlineRegions(RegionSizeStore store, Set<RegionInfo> onlineRegions) {
098    // We have to remove regions which are no longer online from the store, otherwise they will
099    // continue to be sent to the Master which will prevent size report expiration.
100    if (onlineRegions.isEmpty()) {
101      // Easy-case, no online regions means no size reports
102      store.clear();
103      return;
104    }
105
106    Iterator<Entry<RegionInfo,RegionSize>> iter = store.iterator();
107    int numEntriesRemoved = 0;
108    while (iter.hasNext()) {
109      Entry<RegionInfo,RegionSize> entry = iter.next();
110      RegionInfo regionInfo = entry.getKey();
111      if (!onlineRegions.contains(regionInfo)) {
112        numEntriesRemoved++;
113        iter.remove();
114      }
115    }
116    if (LOG.isTraceEnabled()) {
117      LOG.trace("Removed " + numEntriesRemoved + " region sizes before reporting to Master "
118          + "because they are for non-online regions.");
119    }
120  }
121
122  /**
123   * Extracts the period for the chore from the configuration.
124   *
125   * @param conf The configuration object.
126   * @return The configured chore period or the default value.
127   */
128  static int getPeriod(Configuration conf) {
129    return conf.getInt(
130        REGION_SIZE_REPORTING_CHORE_PERIOD_KEY, REGION_SIZE_REPORTING_CHORE_PERIOD_DEFAULT);
131  }
132
133  /**
134   * Extracts the initial delay for the chore from the configuration.
135   *
136   * @param conf The configuration object.
137   * @return The configured chore initial delay or the default value.
138   */
139  static long getInitialDelay(Configuration conf) {
140    return conf.getLong(
141        REGION_SIZE_REPORTING_CHORE_DELAY_KEY, REGION_SIZE_REPORTING_CHORE_DELAY_DEFAULT);
142  }
143
144  /**
145   * Extracts the time unit for the chore period and initial delay from the configuration. The
146   * configuration value for {@link #REGION_SIZE_REPORTING_CHORE_TIMEUNIT_KEY} must correspond to a
147   * {@link TimeUnit} value.
148   *
149   * @param conf The configuration object.
150   * @return The configured time unit for the chore period and initial delay or the default value.
151   */
152  static TimeUnit getTimeUnit(Configuration conf) {
153    return TimeUnit.valueOf(conf.get(REGION_SIZE_REPORTING_CHORE_TIMEUNIT_KEY,
154        REGION_SIZE_REPORTING_CHORE_TIMEUNIT_DEFAULT));
155  }
156}