001/*
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.regionserver;
020
021import java.lang.management.MemoryType;
022import java.util.concurrent.ConcurrentHashMap;
023import java.util.concurrent.ConcurrentMap;
024import java.util.concurrent.atomic.LongAdder;
025
026import org.apache.hadoop.conf.Configuration;
027import org.apache.yetus.audience.InterfaceAudience;
028import org.apache.hadoop.hbase.io.util.MemorySizeUtil;
029import org.apache.hadoop.hbase.util.Pair;
030
031/**
032 * RegionServerAccounting keeps record of some basic real time information about
033 * the Region Server. Currently, it keeps record the global memstore size and global memstore
034 * on-heap and off-heap overhead. It also tracks the replay edits per region.
035 */
036@InterfaceAudience.Private
037public class RegionServerAccounting {
038  // memstore data size
039  private final LongAdder globalMemStoreDataSize = new LongAdder();
040  // memstore heap size.
041  private final LongAdder globalMemStoreHeapSize = new LongAdder();
042  // memstore off-heap size.
043  private final LongAdder globalMemStoreOffHeapSize = new LongAdder();
044
045  private long globalMemStoreLimit;
046  private final float globalMemStoreLimitLowMarkPercent;
047  private long globalMemStoreLimitLowMark;
048  private final MemoryType memType;
049  private long globalOnHeapMemstoreLimit;
050  private long globalOnHeapMemstoreLimitLowMark;
051
052  // encoded region name -> Pair -> read count as first, write count as second.
053  // when region close and target rs is the current server, we will put an entry,
054  // and will remove it when reigon open after recover them.
055  private ConcurrentMap<String, Pair<Long, Long>> retainedRegionRWRequestsCnt;
056
057  public RegionServerAccounting(Configuration conf) {
058    Pair<Long, MemoryType> globalMemstoreSizePair = MemorySizeUtil.getGlobalMemStoreSize(conf);
059    this.globalMemStoreLimit = globalMemstoreSizePair.getFirst();
060    this.memType = globalMemstoreSizePair.getSecond();
061    this.globalMemStoreLimitLowMarkPercent =
062        MemorySizeUtil.getGlobalMemStoreHeapLowerMark(conf, this.memType == MemoryType.HEAP);
063    // When off heap memstore in use we configure the global off heap space for memstore as bytes
064    // not as % of max memory size. In such case, the lower water mark should be specified using the
065    // key "hbase.regionserver.global.memstore.size.lower.limit" which says % of the global upper
066    // bound and defaults to 95%. In on heap case also specifying this way is ideal. But in the past
067    // we used to take lower bound also as the % of xmx (38% as default). For backward compatibility
068    // for this deprecated config,we will fall back to read that config when new one is missing.
069    // Only for on heap case, do this fallback mechanism. For off heap it makes no sense.
070    // TODO When to get rid of the deprecated config? ie
071    // "hbase.regionserver.global.memstore.lowerLimit". Can get rid of this boolean passing then.
072    this.globalMemStoreLimitLowMark =
073        (long) (this.globalMemStoreLimit * this.globalMemStoreLimitLowMarkPercent);
074    this.globalOnHeapMemstoreLimit = MemorySizeUtil.getOnheapGlobalMemStoreSize(conf);
075    this.globalOnHeapMemstoreLimitLowMark =
076        (long) (this.globalOnHeapMemstoreLimit * this.globalMemStoreLimitLowMarkPercent);
077    this.retainedRegionRWRequestsCnt = new ConcurrentHashMap<>();
078  }
079
080  long getGlobalMemStoreLimit() {
081    return this.globalMemStoreLimit;
082  }
083
084  long getGlobalOnHeapMemStoreLimit() {
085    return this.globalOnHeapMemstoreLimit;
086  }
087
088  // Called by the tuners.
089  void setGlobalMemStoreLimits(long newGlobalMemstoreLimit) {
090    if (this.memType == MemoryType.HEAP) {
091      this.globalMemStoreLimit = newGlobalMemstoreLimit;
092      this.globalMemStoreLimitLowMark =
093          (long) (this.globalMemStoreLimit * this.globalMemStoreLimitLowMarkPercent);
094    } else {
095      this.globalOnHeapMemstoreLimit = newGlobalMemstoreLimit;
096      this.globalOnHeapMemstoreLimitLowMark =
097          (long) (this.globalOnHeapMemstoreLimit * this.globalMemStoreLimitLowMarkPercent);
098    }
099  }
100
101  boolean isOffheap() {
102    return this.memType == MemoryType.NON_HEAP;
103  }
104
105  long getGlobalMemStoreLimitLowMark() {
106    return this.globalMemStoreLimitLowMark;
107  }
108
109  float getGlobalMemStoreLimitLowMarkPercent() {
110    return this.globalMemStoreLimitLowMarkPercent;
111  }
112
113  /**
114   * @return the global Memstore data size in the RegionServer
115   */
116  public long getGlobalMemStoreDataSize() {
117    return globalMemStoreDataSize.sum();
118  }
119
120  /**
121   * @return the global memstore heap size in the RegionServer
122   */
123  public long getGlobalMemStoreHeapSize() {
124    return this.globalMemStoreHeapSize.sum();
125  }
126
127  /**
128   * @return the global memstore heap size in the RegionServer
129   */
130  public long getGlobalMemStoreOffHeapSize() {
131    return this.globalMemStoreOffHeapSize.sum();
132  }
133
134  /**
135   * @return the retained metrics of region's read and write requests count
136   */
137  protected ConcurrentMap<String, Pair<Long, Long>> getRetainedRegionRWRequestsCnt() {
138    return this.retainedRegionRWRequestsCnt;
139  }
140
141  void incGlobalMemStoreSize(MemStoreSize mss) {
142    incGlobalMemStoreSize(mss.getDataSize(), mss.getHeapSize(), mss.getOffHeapSize());
143  }
144
145  public void incGlobalMemStoreSize(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta) {
146    globalMemStoreDataSize.add(dataSizeDelta);
147    globalMemStoreHeapSize.add(heapSizeDelta);
148    globalMemStoreOffHeapSize.add(offHeapSizeDelta);
149  }
150
151  public void decGlobalMemStoreSize(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta) {
152    globalMemStoreDataSize.add(-dataSizeDelta);
153    globalMemStoreHeapSize.add(-heapSizeDelta);
154    globalMemStoreOffHeapSize.add(-offHeapSizeDelta);
155  }
156
157  /**
158   * Return true if we are above the memstore high water mark
159   * @return the flushtype
160   */
161  public FlushType isAboveHighWaterMark() {
162    // for onheap memstore we check if the global memstore size and the
163    // global heap overhead is greater than the global memstore limit
164    if (memType == MemoryType.HEAP) {
165      if (getGlobalMemStoreHeapSize() >= globalMemStoreLimit) {
166        return FlushType.ABOVE_ONHEAP_HIGHER_MARK;
167      }
168    } else {
169      // If the configured memstore is offheap, check for two things
170      // 1) If the global memstore off-heap size is greater than the configured
171      // 'hbase.regionserver.offheap.global.memstore.size'
172      // 2) If the global memstore heap size is greater than the configured onheap
173      // global memstore limit 'hbase.regionserver.global.memstore.size'.
174      // We do this to avoid OOME incase of scenarios where the heap is occupied with
175      // lot of onheap references to the cells in memstore
176      if (getGlobalMemStoreOffHeapSize() >= globalMemStoreLimit) {
177        // Indicates that global memstore size is above the configured
178        // 'hbase.regionserver.offheap.global.memstore.size'
179        return FlushType.ABOVE_OFFHEAP_HIGHER_MARK;
180      } else if (getGlobalMemStoreHeapSize() >= this.globalOnHeapMemstoreLimit) {
181        // Indicates that the offheap memstore's heap overhead is greater than the
182        // configured 'hbase.regionserver.global.memstore.size'.
183        return FlushType.ABOVE_ONHEAP_HIGHER_MARK;
184      }
185    }
186    return FlushType.NORMAL;
187  }
188
189  /**
190   * Return true if we're above the low watermark
191   */
192  public FlushType isAboveLowWaterMark() {
193    // for onheap memstore we check if the global memstore size and the
194    // global heap overhead is greater than the global memstore lower mark limit
195    if (memType == MemoryType.HEAP) {
196      if (getGlobalMemStoreHeapSize() >= globalMemStoreLimitLowMark) {
197        return FlushType.ABOVE_ONHEAP_LOWER_MARK;
198      }
199    } else {
200      if (getGlobalMemStoreOffHeapSize() >= globalMemStoreLimitLowMark) {
201        // Indicates that the offheap memstore's size is greater than the global memstore
202        // lower limit
203        return FlushType.ABOVE_OFFHEAP_LOWER_MARK;
204      } else if (getGlobalMemStoreHeapSize() >= globalOnHeapMemstoreLimitLowMark) {
205        // Indicates that the offheap memstore's heap overhead is greater than the global memstore
206        // onheap lower limit
207        return FlushType.ABOVE_ONHEAP_LOWER_MARK;
208      }
209    }
210    return FlushType.NORMAL;
211  }
212
213  /**
214   * @return the flush pressure of all stores on this regionserver. The value should be greater than
215   *         or equal to 0.0, and any value greater than 1.0 means we enter the emergency state that
216   *         global memstore size already exceeds lower limit.
217   */
218  public double getFlushPressure() {
219    if (memType == MemoryType.HEAP) {
220      return (getGlobalMemStoreHeapSize()) * 1.0 / globalMemStoreLimitLowMark;
221    } else {
222      return Math.max(getGlobalMemStoreOffHeapSize() * 1.0 / globalMemStoreLimitLowMark,
223          getGlobalMemStoreHeapSize() * 1.0 / globalOnHeapMemstoreLimitLowMark);
224    }
225  }
226}