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