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.quotas;
019
020import java.util.HashSet;
021import java.util.Iterator;
022import java.util.List;
023import java.util.Map.Entry;
024import java.util.Set;
025import java.util.concurrent.TimeUnit;
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(RegionSizeReportingChore.class.getSimpleName(), rsServices,
060      getPeriod(rsServices.getConfiguration()), getInitialDelay(rsServices.getConfiguration()),
061      getTimeUnit(rsServices.getConfiguration()));
062    this.rsServices = rsServices;
063    this.metrics = rsServices.getMetrics();
064  }
065
066  @Override
067  protected void chore() {
068    final long start = System.nanoTime();
069    try {
070      _chore();
071    } finally {
072      if (metrics != null) {
073        metrics.incrementRegionSizeReportingChoreTime(
074          TimeUnit.MILLISECONDS.convert(System.nanoTime() - start, TimeUnit.NANOSECONDS));
075      }
076    }
077  }
078
079  void _chore() {
080    final RegionServerSpaceQuotaManager quotaManager =
081      rsServices.getRegionServerSpaceQuotaManager();
082    // Get the HRegionInfo for each online region
083    HashSet<RegionInfo> onlineRegionInfos = getOnlineRegionInfos(rsServices.getRegions());
084    RegionSizeStore store = quotaManager.getRegionSizeStore();
085    // Remove all sizes for non-online regions
086    removeNonOnlineRegions(store, onlineRegionInfos);
087    rsServices.reportRegionSizesForQuotas(store);
088  }
089
090  HashSet<RegionInfo> getOnlineRegionInfos(List<? extends Region> onlineRegions) {
091    HashSet<RegionInfo> regionInfos = new HashSet<>();
092    onlineRegions.forEach((region) -> regionInfos.add(region.getRegionInfo()));
093    return regionInfos;
094  }
095
096  void removeNonOnlineRegions(RegionSizeStore store, Set<RegionInfo> onlineRegions) {
097    // We have to remove regions which are no longer online from the store, otherwise they will
098    // continue to be sent to the Master which will prevent size report expiration.
099    if (onlineRegions.isEmpty()) {
100      // Easy-case, no online regions means no size reports
101      store.clear();
102      return;
103    }
104
105    Iterator<Entry<RegionInfo, RegionSize>> iter = store.iterator();
106    int numEntriesRemoved = 0;
107    while (iter.hasNext()) {
108      Entry<RegionInfo, RegionSize> entry = iter.next();
109      RegionInfo regionInfo = entry.getKey();
110      if (!onlineRegions.contains(regionInfo)) {
111        numEntriesRemoved++;
112        iter.remove();
113      }
114    }
115    if (LOG.isTraceEnabled()) {
116      LOG.trace("Removed " + numEntriesRemoved + " region sizes before reporting to Master "
117        + "because they are for non-online regions.");
118    }
119  }
120
121  /**
122   * Extracts the period for the chore from the configuration.
123   * @param conf The configuration object.
124   * @return The configured chore period or the default value.
125   */
126  static int getPeriod(Configuration conf) {
127    return conf.getInt(REGION_SIZE_REPORTING_CHORE_PERIOD_KEY,
128      REGION_SIZE_REPORTING_CHORE_PERIOD_DEFAULT);
129  }
130
131  /**
132   * Extracts the initial delay for the chore from the configuration.
133   * @param conf The configuration object.
134   * @return The configured chore initial delay or the default value.
135   */
136  static long getInitialDelay(Configuration conf) {
137    return conf.getLong(REGION_SIZE_REPORTING_CHORE_DELAY_KEY,
138      REGION_SIZE_REPORTING_CHORE_DELAY_DEFAULT);
139  }
140
141  /**
142   * Extracts the time unit for the chore period and initial delay from the configuration. The
143   * configuration value for {@link #REGION_SIZE_REPORTING_CHORE_TIMEUNIT_KEY} must correspond to a
144   * {@link TimeUnit} value.
145   * @param conf The configuration object.
146   * @return The configured time unit for the chore period and initial delay or the default value.
147   */
148  static TimeUnit getTimeUnit(Configuration conf) {
149    return TimeUnit.valueOf(conf.get(REGION_SIZE_REPORTING_CHORE_TIMEUNIT_KEY,
150      REGION_SIZE_REPORTING_CHORE_TIMEUNIT_DEFAULT));
151  }
152}