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.io.util;
019
020import java.lang.management.ManagementFactory;
021import java.lang.management.MemoryType;
022import java.lang.management.MemoryUsage;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.conf.StorageUnit;
025import org.apache.hadoop.hbase.HConstants;
026import org.apache.hadoop.hbase.regionserver.MemStoreLAB;
027import org.apache.hadoop.hbase.util.Pair;
028import org.apache.yetus.audience.InterfaceAudience;
029import org.slf4j.Logger;
030import org.slf4j.LoggerFactory;
031
032/**
033 * Util class to calculate memory size for memstore(on heap, off heap), block cache(L1, L2) of RS.
034 */
035@InterfaceAudience.Private
036public class MemorySizeUtil {
037
038  public static final String MEMSTORE_SIZE_KEY = "hbase.regionserver.global.memstore.size";
039  public static final String MEMSTORE_SIZE_OLD_KEY =
040    "hbase.regionserver.global.memstore.upperLimit";
041  public static final String MEMSTORE_SIZE_LOWER_LIMIT_KEY =
042    "hbase.regionserver.global.memstore.size.lower.limit";
043  public static final String MEMSTORE_SIZE_LOWER_LIMIT_OLD_KEY =
044    "hbase.regionserver.global.memstore.lowerLimit";
045  // Max global off heap memory that can be used for all memstores
046  // This should be an absolute value in MBs and not percent.
047  public static final String OFFHEAP_MEMSTORE_SIZE_KEY =
048    "hbase.regionserver.offheap.global.memstore.size";
049
050  public static final float DEFAULT_MEMSTORE_SIZE = 0.4f;
051  // Default lower water mark limit is 95% size of memstore size.
052  public static final float DEFAULT_MEMSTORE_SIZE_LOWER_LIMIT = 0.95f;
053
054  private static final Logger LOG = LoggerFactory.getLogger(MemorySizeUtil.class);
055  // a constant to convert a fraction to a percentage
056  private static final int CONVERT_TO_PERCENTAGE = 100;
057
058  private static final String JVM_HEAP_EXCEPTION = "Got an exception while attempting to read "
059    + "information about the JVM heap. Please submit this log information in a bug report and "
060    + "include your JVM settings, specifically the GC in use and any -XX options. Consider "
061    + "restarting the service.";
062
063  /**
064   * Return JVM memory statistics while properly handling runtime exceptions from the JVM.
065   * @return a memory usage object, null if there was a runtime exception. (n.b. you could also get
066   *         -1 values back from the JVM)
067   * @see MemoryUsage
068   */
069  public static MemoryUsage safeGetHeapMemoryUsage() {
070    MemoryUsage usage = null;
071    try {
072      usage = ManagementFactory.getMemoryMXBean().getHeapMemoryUsage();
073    } catch (RuntimeException exception) {
074      LOG.warn(JVM_HEAP_EXCEPTION, exception);
075    }
076    return usage;
077  }
078
079  /**
080   * Checks whether we have enough heap memory left out after portion for Memstore and Block cache.
081   * We need atleast 20% of heap left out for other RS functions.
082   */
083  public static void checkForClusterFreeHeapMemoryLimit(Configuration conf) {
084    if (conf.get(MEMSTORE_SIZE_OLD_KEY) != null) {
085      LOG.warn(MEMSTORE_SIZE_OLD_KEY + " is deprecated by " + MEMSTORE_SIZE_KEY);
086    }
087    float globalMemstoreSize = getGlobalMemStoreHeapPercent(conf, false);
088    int gml = (int) (globalMemstoreSize * CONVERT_TO_PERCENTAGE);
089    float blockCacheUpperLimit = getBlockCacheHeapPercent(conf);
090    int bcul = (int) (blockCacheUpperLimit * CONVERT_TO_PERCENTAGE);
091    if (
092      CONVERT_TO_PERCENTAGE - (gml + bcul)
093          < (int) (CONVERT_TO_PERCENTAGE * HConstants.HBASE_CLUSTER_MINIMUM_MEMORY_THRESHOLD)
094    ) {
095      throw new RuntimeException("Current heap configuration for MemStore and BlockCache exceeds "
096        + "the threshold required for successful cluster operation. "
097        + "The combined value cannot exceed 0.8. Please check " + "the settings for "
098        + MEMSTORE_SIZE_KEY + " and either " + HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY + " or "
099        + HConstants.HFILE_BLOCK_CACHE_SIZE_KEY + " in your configuration. " + MEMSTORE_SIZE_KEY
100        + "=" + globalMemstoreSize + ", " + HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY + "="
101        + conf.get(HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY) + ", "
102        + HConstants.HFILE_BLOCK_CACHE_SIZE_KEY + "="
103        + conf.get(HConstants.HFILE_BLOCK_CACHE_SIZE_KEY) + ". (Note: If both "
104        + HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY + " and "
105        + HConstants.HFILE_BLOCK_CACHE_SIZE_KEY + " are set, " + "the system will use "
106        + HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY + ")");
107    }
108  }
109
110  /**
111   * Retrieve global memstore configured size as percentage of total heap.
112   */
113  public static float getGlobalMemStoreHeapPercent(final Configuration c,
114    final boolean logInvalid) {
115    float limit =
116      c.getFloat(MEMSTORE_SIZE_KEY, c.getFloat(MEMSTORE_SIZE_OLD_KEY, DEFAULT_MEMSTORE_SIZE));
117    if (limit > 0.8f || limit <= 0.0f) {
118      if (logInvalid) {
119        LOG.warn("Setting global memstore limit to default of " + DEFAULT_MEMSTORE_SIZE
120          + " because supplied value outside allowed range of (0 -> 0.8]");
121      }
122      limit = DEFAULT_MEMSTORE_SIZE;
123    }
124    return limit;
125  }
126
127  /**
128   * Retrieve configured size for global memstore lower water mark as fraction of global memstore
129   * size.
130   */
131  public static float getGlobalMemStoreHeapLowerMark(final Configuration conf,
132    boolean honorOldConfig) {
133    String lowMarkPercentStr = conf.get(MEMSTORE_SIZE_LOWER_LIMIT_KEY);
134    if (lowMarkPercentStr != null) {
135      float lowMarkPercent = Float.parseFloat(lowMarkPercentStr);
136      if (lowMarkPercent > 1.0f) {
137        LOG.error("Bad configuration value for " + MEMSTORE_SIZE_LOWER_LIMIT_KEY + ": "
138          + lowMarkPercent + ". Using 1.0f instead.");
139        lowMarkPercent = 1.0f;
140      }
141      return lowMarkPercent;
142    }
143    if (!honorOldConfig) return DEFAULT_MEMSTORE_SIZE_LOWER_LIMIT;
144    String lowerWaterMarkOldValStr = conf.get(MEMSTORE_SIZE_LOWER_LIMIT_OLD_KEY);
145    if (lowerWaterMarkOldValStr != null) {
146      LOG.warn(MEMSTORE_SIZE_LOWER_LIMIT_OLD_KEY + " is deprecated. Instead use "
147        + MEMSTORE_SIZE_LOWER_LIMIT_KEY);
148      float lowerWaterMarkOldVal = Float.parseFloat(lowerWaterMarkOldValStr);
149      float upperMarkPercent = getGlobalMemStoreHeapPercent(conf, false);
150      if (lowerWaterMarkOldVal > upperMarkPercent) {
151        lowerWaterMarkOldVal = upperMarkPercent;
152        LOG.error("Value of " + MEMSTORE_SIZE_LOWER_LIMIT_OLD_KEY + " (" + lowerWaterMarkOldVal
153          + ") is greater than global memstore limit (" + upperMarkPercent + ") set by "
154          + MEMSTORE_SIZE_KEY + "/" + MEMSTORE_SIZE_OLD_KEY + ". Setting memstore lower limit "
155          + "to " + upperMarkPercent);
156      }
157      return lowerWaterMarkOldVal / upperMarkPercent;
158    }
159    return DEFAULT_MEMSTORE_SIZE_LOWER_LIMIT;
160  }
161
162  /** Returns Pair of global memstore size and memory type(ie. on heap or off heap). */
163  public static Pair<Long, MemoryType> getGlobalMemStoreSize(Configuration conf) {
164    long offheapMSGlobal = conf.getLong(OFFHEAP_MEMSTORE_SIZE_KEY, 0);// Size in MBs
165    if (offheapMSGlobal > 0) {
166      // Off heap memstore size has not relevance when MSLAB is turned OFF. We will go with making
167      // this entire size split into Chunks and pooling them in MemstoreLABPoool. We dont want to
168      // create so many on demand off heap chunks. In fact when this off heap size is configured, we
169      // will go with 100% of this size as the pool size
170      if (MemStoreLAB.isEnabled(conf)) {
171        // We are in offheap Memstore use
172        long globalMemStoreLimit = (long) (offheapMSGlobal * 1024 * 1024); // Size in bytes
173        return new Pair<>(globalMemStoreLimit, MemoryType.NON_HEAP);
174      } else {
175        // Off heap max memstore size is configured with turning off MSLAB. It makes no sense. Do a
176        // warn log and go with on heap memstore percentage. By default it will be 40% of Xmx
177        LOG.warn("There is no relevance of configuring '" + OFFHEAP_MEMSTORE_SIZE_KEY + "' when '"
178          + MemStoreLAB.USEMSLAB_KEY + "' is turned off."
179          + " Going with on heap global memstore size ('" + MEMSTORE_SIZE_KEY + "')");
180      }
181    }
182    return new Pair<>(getOnheapGlobalMemStoreSize(conf), MemoryType.HEAP);
183  }
184
185  /**
186   * Returns the onheap global memstore limit based on the config
187   * 'hbase.regionserver.global.memstore.size'.
188   * @return the onheap global memstore limt
189   */
190  public static long getOnheapGlobalMemStoreSize(Configuration conf) {
191    long max = -1L;
192    final MemoryUsage usage = safeGetHeapMemoryUsage();
193    if (usage != null) {
194      max = usage.getMax();
195    }
196    float globalMemStorePercent = getGlobalMemStoreHeapPercent(conf, true);
197    return ((long) (max * globalMemStorePercent));
198  }
199
200  /**
201   * Retrieve configured size for on heap block cache as percentage of total heap.
202   */
203  public static float getBlockCacheHeapPercent(final Configuration conf) {
204    // Check if an explicit block cache size is configured.
205    long l1CacheSizeInBytes = getBlockCacheSizeInBytes(conf);
206    if (l1CacheSizeInBytes > 0) {
207      final MemoryUsage usage = safeGetHeapMemoryUsage();
208      return usage == null ? 0 : (float) l1CacheSizeInBytes / usage.getMax();
209    }
210
211    return conf.getFloat(HConstants.HFILE_BLOCK_CACHE_SIZE_KEY,
212      HConstants.HFILE_BLOCK_CACHE_SIZE_DEFAULT);
213  }
214
215  /**
216   * Retrieve an explicit block cache size in bytes in the configuration.
217   * @param conf used to read cache configs
218   * @return the number of bytes to use for LRU, negative if disabled.
219   * @throws IllegalArgumentException if HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY format is invalid
220   */
221  public static long getBlockCacheSizeInBytes(Configuration conf) {
222    final String key = HConstants.HFILE_BLOCK_CACHE_MEMORY_SIZE_KEY;
223    try {
224      return Long.parseLong(conf.get(key));
225    } catch (NumberFormatException e) {
226      return (long) conf.getStorageSize(key, -1, StorageUnit.BYTES);
227    }
228  }
229
230  /**
231   * @param conf used to read cache configs
232   * @return the number of bytes to use for LRU, negative if disabled.
233   * @throws IllegalArgumentException if HFILE_BLOCK_CACHE_SIZE_KEY is > 1.0
234   */
235  public static long getOnHeapCacheSize(final Configuration conf) {
236    final float cachePercentage = getBlockCacheHeapPercent(conf);
237    if (cachePercentage <= 0.0001f) {
238      return -1;
239    }
240    if (cachePercentage > 1.0) {
241      throw new IllegalArgumentException(
242        HConstants.HFILE_BLOCK_CACHE_SIZE_KEY + " must be between 0.0 and 1.0, and not > 1.0");
243    }
244
245    final MemoryUsage usage = safeGetHeapMemoryUsage();
246    if (usage == null) {
247      return -1;
248    }
249    final long heapMax = usage.getMax();
250    float onHeapCacheFixedSize =
251      (float) conf.getLong(HConstants.HFILE_ONHEAP_BLOCK_CACHE_FIXED_SIZE_KEY,
252        HConstants.HFILE_ONHEAP_BLOCK_CACHE_FIXED_SIZE_DEFAULT) / heapMax;
253    // Calculate the amount of heap to give the heap.
254    if (onHeapCacheFixedSize > 0 && onHeapCacheFixedSize < cachePercentage) {
255      return (long) (heapMax * onHeapCacheFixedSize);
256    } else {
257      final long cacheSizeInBytes = getBlockCacheSizeInBytes(conf);
258      return cacheSizeInBytes > 0 ? cacheSizeInBytes : (long) (heapMax * cachePercentage);
259    }
260  }
261
262  /**
263   * @param conf used to read config for bucket cache size.
264   * @return the number of bytes to use for bucket cache, negative if disabled.
265   */
266  public static long getBucketCacheSize(final Configuration conf) {
267    // Size configured in MBs
268    float bucketCacheSize = conf.getFloat(HConstants.BUCKET_CACHE_SIZE_KEY, 0F);
269    if (bucketCacheSize < 1) {
270      throw new IllegalArgumentException("Bucket Cache should be minimum 1 MB in size."
271        + "Configure 'hbase.bucketcache.size' with > 1 value");
272    }
273    return (long) (bucketCacheSize * 1024 * 1024);
274  }
275}