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.regionserver.throttle;
019
020import org.apache.hadoop.conf.Configuration;
021import org.apache.hadoop.hbase.HBaseInterfaceAudience;
022import org.apache.hadoop.hbase.ScheduledChore;
023import org.apache.yetus.audience.InterfaceAudience;
024import org.slf4j.Logger;
025import org.slf4j.LoggerFactory;
026import org.apache.hadoop.hbase.regionserver.RegionServerServices;
027import org.apache.hadoop.hbase.regionserver.compactions.OffPeakHours;
028
029/**
030 * A throughput controller which uses the follow schema to limit throughput
031 * <ul>
032 * <li>If flush pressure is greater than or equal to 1.0, no limitation.</li>
033 * <li>In normal case, the max throughput is tuned between
034 * {@value #HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_LOWER_BOUND} and
035 * {@value #HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_UPPER_BOUND}, using the formula &quot;lower +
036 * (upper - lower) * flushPressure&quot;, where flushPressure is in range [0.0, 1.0)</li>
037 * </ul>
038 * @see org.apache.hadoop.hbase.regionserver.HRegionServer#getFlushPressure()
039 */
040@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.CONFIG)
041public class PressureAwareFlushThroughputController extends PressureAwareThroughputController {
042
043  private static final Logger LOG =
044      LoggerFactory.getLogger(PressureAwareFlushThroughputController.class);
045
046  public static final String HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_UPPER_BOUND =
047      "hbase.hstore.flush.throughput.upper.bound";
048
049  private static final long DEFAULT_HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_UPPER_BOUND =
050      200L * 1024 * 1024;
051
052  public static final String HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_LOWER_BOUND =
053      "hbase.hstore.flush.throughput.lower.bound";
054
055  private static final long DEFAULT_HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_LOWER_BOUND =
056      100L * 1024 * 1024;
057
058  public static final String HBASE_HSTORE_FLUSH_THROUGHPUT_TUNE_PERIOD =
059      "hbase.hstore.flush.throughput.tune.period";
060
061  private static final int DEFAULT_HSTORE_FLUSH_THROUGHPUT_TUNE_PERIOD = 20 * 1000;
062
063  // check flush throughput every this size
064  public static final String HBASE_HSTORE_FLUSH_THROUGHPUT_CONTROL_CHECK_INTERVAL =
065      "hbase.hstore.flush.throughput.control.check.interval";
066
067  private static final long DEFAULT_HBASE_HSTORE_FLUSH_THROUGHPUT_CONTROL_CHECK_INTERVAL =
068      10L * 1024 * 1024;// 10MB
069
070  @Override
071  public void setup(final RegionServerServices server) {
072    server.getChoreService().scheduleChore(
073      new ScheduledChore("FlushThroughputTuner", this, tuningPeriod, this.tuningPeriod) {
074
075        @Override
076        protected void chore() {
077          tune(server.getFlushPressure());
078        }
079      });
080  }
081
082  private void tune(double flushPressure) {
083    double maxThroughputToSet;
084    if (flushPressure >= 1.0) {
085      // set to unlimited if global memstore size already exceeds lower limit
086      maxThroughputToSet = Double.MAX_VALUE;
087    } else {
088      // flushPressure is between 0.0 and 1.0, we use a simple linear formula to
089      // calculate the throughput limitation.
090      maxThroughputToSet =
091          maxThroughputLowerBound + (maxThroughputUpperBound - maxThroughputLowerBound)
092              * flushPressure;
093    }
094    if (LOG.isDebugEnabled()) {
095      LOG.debug("flushPressure is " + flushPressure + ", tune flush throughput to "
096          + throughputDesc(maxThroughputToSet));
097    }
098    this.setMaxThroughput(maxThroughputToSet);
099  }
100
101  @Override
102  public void setConf(Configuration conf) {
103    super.setConf(conf);
104    if (conf == null) {
105      return;
106    }
107    this.maxThroughputUpperBound =
108        conf.getLong(HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_UPPER_BOUND,
109          DEFAULT_HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_UPPER_BOUND);
110    this.maxThroughputLowerBound =
111        conf.getLong(HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_LOWER_BOUND,
112          DEFAULT_HBASE_HSTORE_FLUSH_MAX_THROUGHPUT_LOWER_BOUND);
113    this.offPeakHours = OffPeakHours.getInstance(conf);
114    this.controlPerSize =
115        conf.getLong(HBASE_HSTORE_FLUSH_THROUGHPUT_CONTROL_CHECK_INTERVAL,
116          DEFAULT_HBASE_HSTORE_FLUSH_THROUGHPUT_CONTROL_CHECK_INTERVAL);
117    this.setMaxThroughput(this.maxThroughputLowerBound);
118    this.tuningPeriod =
119        getConf().getInt(HBASE_HSTORE_FLUSH_THROUGHPUT_TUNE_PERIOD,
120          DEFAULT_HSTORE_FLUSH_THROUGHPUT_TUNE_PERIOD);
121    LOG.info("Flush throughput configurations, upper bound: "
122        + throughputDesc(maxThroughputUpperBound) + ", lower bound "
123        + throughputDesc(maxThroughputLowerBound) + ", tuning period: " + tuningPeriod + " ms");
124  }
125
126  @Override
127  public String toString() {
128    return "DefaultFlushController [maxThroughput=" + throughputDesc(getMaxThroughput())
129        + ", activeFlushNumber=" + activeOperations.size() + "]";
130  }
131
132  @Override
133  protected boolean skipControl(long deltaSize, long controlSize) {
134    // for flush, we control the flow no matter whether the flush size is small
135    return false;
136  }
137}