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.hfile;
019
020import java.util.Optional;
021
022import org.apache.hadoop.conf.Configuration;
023import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
024import org.apache.hadoop.hbase.io.hfile.BlockType.BlockCategory;
025import org.apache.yetus.audience.InterfaceAudience;
026import org.slf4j.Logger;
027import org.slf4j.LoggerFactory;
028
029import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
030
031/**
032 * Stores all of the cache objects and configuration for a single HFile.
033 */
034@InterfaceAudience.Private
035public class CacheConfig {
036  private static final Logger LOG = LoggerFactory.getLogger(CacheConfig.class.getName());
037
038  /**
039   * Disabled cache configuration
040   */
041  public static final CacheConfig DISABLED = new CacheConfig();
042
043  /**
044   * Configuration key to cache data blocks on read. Bloom blocks and index blocks are always be
045   * cached if the block cache is enabled.
046   */
047  public static final String CACHE_DATA_ON_READ_KEY = "hbase.block.data.cacheonread";
048
049  /**
050   * Configuration key to cache data blocks on write. There are separate
051   * switches for bloom blocks and non-root index blocks.
052   */
053  public static final String CACHE_BLOCKS_ON_WRITE_KEY = "hbase.rs.cacheblocksonwrite";
054
055  /**
056   * Configuration key to cache leaf and intermediate-level index blocks on
057   * write.
058   */
059  public static final String CACHE_INDEX_BLOCKS_ON_WRITE_KEY = "hfile.block.index.cacheonwrite";
060
061  /**
062   * Configuration key to cache compound bloom filter blocks on write.
063   */
064  public static final String CACHE_BLOOM_BLOCKS_ON_WRITE_KEY = "hfile.block.bloom.cacheonwrite";
065
066  /**
067   * Configuration key to cache data blocks in compressed and/or encrypted format.
068   */
069  public static final String CACHE_DATA_BLOCKS_COMPRESSED_KEY = "hbase.block.data.cachecompressed";
070
071  /**
072   * Configuration key to evict all blocks of a given file from the block cache
073   * when the file is closed.
074   */
075  public static final String EVICT_BLOCKS_ON_CLOSE_KEY = "hbase.rs.evictblocksonclose";
076
077  /**
078   * Configuration key to prefetch all blocks of a given file into the block cache
079   * when the file is opened.
080   */
081  public static final String PREFETCH_BLOCKS_ON_OPEN_KEY = "hbase.rs.prefetchblocksonopen";
082
083  public static final String DROP_BEHIND_CACHE_COMPACTION_KEY =
084      "hbase.hfile.drop.behind.compaction";
085
086  // Defaults
087  public static final boolean DEFAULT_CACHE_DATA_ON_READ = true;
088  public static final boolean DEFAULT_CACHE_DATA_ON_WRITE = false;
089  public static final boolean DEFAULT_IN_MEMORY = false;
090  public static final boolean DEFAULT_CACHE_INDEXES_ON_WRITE = false;
091  public static final boolean DEFAULT_CACHE_BLOOMS_ON_WRITE = false;
092  public static final boolean DEFAULT_EVICT_ON_CLOSE = false;
093  public static final boolean DEFAULT_CACHE_DATA_COMPRESSED = false;
094  public static final boolean DEFAULT_PREFETCH_ON_OPEN = false;
095  public static final boolean DROP_BEHIND_CACHE_COMPACTION_DEFAULT = true;
096
097  /**
098   * Whether blocks should be cached on read (default is on if there is a
099   * cache but this can be turned off on a per-family or per-request basis).
100   * If off we will STILL cache meta blocks; i.e. INDEX and BLOOM types.
101   * This cannot be disabled.
102   */
103  private final boolean cacheDataOnRead;
104
105  /** Whether blocks should be flagged as in-memory when being cached */
106  private final boolean inMemory;
107
108  /** Whether data blocks should be cached when new files are written */
109  private boolean cacheDataOnWrite;
110
111  /** Whether index blocks should be cached when new files are written */
112  private final boolean cacheIndexesOnWrite;
113
114  /** Whether compound bloom filter blocks should be cached on write */
115  private final boolean cacheBloomsOnWrite;
116
117  /** Whether blocks of a file should be evicted when the file is closed */
118  private boolean evictOnClose;
119
120  /** Whether data blocks should be stored in compressed and/or encrypted form in the cache */
121  private final boolean cacheDataCompressed;
122
123  /** Whether data blocks should be prefetched into the cache */
124  private final boolean prefetchOnOpen;
125
126  private final boolean dropBehindCompaction;
127
128  // Local reference to the block cache
129  private final BlockCache blockCache;
130
131  /**
132   * Create a cache configuration using the specified configuration object and
133   * defaults for family level settings. Only use if no column family context.
134   * @param conf hbase configuration
135   */
136  public CacheConfig(Configuration conf) {
137    this(conf, null);
138  }
139
140  public CacheConfig(Configuration conf, BlockCache blockCache) {
141    this(conf, null, blockCache);
142  }
143
144  /**
145   * Create a cache configuration using the specified configuration object and
146   * family descriptor.
147   * @param conf hbase configuration
148   * @param family column family configuration
149   */
150  public CacheConfig(Configuration conf, ColumnFamilyDescriptor family, BlockCache blockCache) {
151    this.cacheDataOnRead = conf.getBoolean(CACHE_DATA_ON_READ_KEY, DEFAULT_CACHE_DATA_ON_READ) &&
152        (family == null ? true : family.isBlockCacheEnabled());
153    this.inMemory = family == null ? DEFAULT_IN_MEMORY : family.isInMemory();
154    this.cacheDataCompressed =
155        conf.getBoolean(CACHE_DATA_BLOCKS_COMPRESSED_KEY, DEFAULT_CACHE_DATA_COMPRESSED);
156    this.dropBehindCompaction =
157        conf.getBoolean(DROP_BEHIND_CACHE_COMPACTION_KEY, DROP_BEHIND_CACHE_COMPACTION_DEFAULT);
158    // For the following flags we enable them regardless of per-schema settings
159    // if they are enabled in the global configuration.
160    this.cacheDataOnWrite =
161        conf.getBoolean(CACHE_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_DATA_ON_WRITE) ||
162            (family == null ? false : family.isCacheDataOnWrite());
163    this.cacheIndexesOnWrite =
164        conf.getBoolean(CACHE_INDEX_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_INDEXES_ON_WRITE) ||
165            (family == null ? false : family.isCacheIndexesOnWrite());
166    this.cacheBloomsOnWrite =
167        conf.getBoolean(CACHE_BLOOM_BLOCKS_ON_WRITE_KEY, DEFAULT_CACHE_BLOOMS_ON_WRITE) ||
168            (family == null ? false : family.isCacheBloomsOnWrite());
169    this.evictOnClose = conf.getBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, DEFAULT_EVICT_ON_CLOSE) ||
170        (family == null ? false : family.isEvictBlocksOnClose());
171    this.prefetchOnOpen = conf.getBoolean(PREFETCH_BLOCKS_ON_OPEN_KEY, DEFAULT_PREFETCH_ON_OPEN) ||
172        (family == null ? false : family.isPrefetchBlocksOnOpen());
173    this.blockCache = blockCache;
174    LOG.info("Created cacheConfig: " + this + (family == null ? "" : " for family " + family) +
175        " with blockCache=" + blockCache);
176  }
177
178  /**
179   * Constructs a cache configuration copied from the specified configuration.
180   * @param cacheConf
181   */
182  public CacheConfig(CacheConfig cacheConf) {
183    this.cacheDataOnRead = cacheConf.cacheDataOnRead;
184    this.inMemory = cacheConf.inMemory;
185    this.cacheDataOnWrite = cacheConf.cacheDataOnWrite;
186    this.cacheIndexesOnWrite = cacheConf.cacheIndexesOnWrite;
187    this.cacheBloomsOnWrite = cacheConf.cacheBloomsOnWrite;
188    this.evictOnClose = cacheConf.evictOnClose;
189    this.cacheDataCompressed = cacheConf.cacheDataCompressed;
190    this.prefetchOnOpen = cacheConf.prefetchOnOpen;
191    this.dropBehindCompaction = cacheConf.dropBehindCompaction;
192    this.blockCache = cacheConf.blockCache;
193  }
194
195  private CacheConfig() {
196    this.cacheDataOnRead = false;
197    this.inMemory = false;
198    this.cacheDataOnWrite = false;
199    this.cacheIndexesOnWrite = false;
200    this.cacheBloomsOnWrite = false;
201    this.evictOnClose = false;
202    this.cacheDataCompressed = false;
203    this.prefetchOnOpen = false;
204    this.dropBehindCompaction = false;
205    this.blockCache = null;
206  }
207
208  /**
209   * Returns whether the DATA blocks of this HFile should be cached on read or not (we always
210   * cache the meta blocks, the INDEX and BLOOM blocks).
211   * @return true if blocks should be cached on read, false if not
212   */
213  public boolean shouldCacheDataOnRead() {
214    return cacheDataOnRead;
215  }
216
217  public boolean shouldDropBehindCompaction() {
218    return dropBehindCompaction;
219  }
220
221  /**
222   * Should we cache a block of a particular category? We always cache
223   * important blocks such as index blocks, as long as the block cache is
224   * available.
225   */
226  public boolean shouldCacheBlockOnRead(BlockCategory category) {
227    return cacheDataOnRead || category == BlockCategory.INDEX || category == BlockCategory.BLOOM ||
228        (prefetchOnOpen && (category != BlockCategory.META && category != BlockCategory.UNKNOWN));
229  }
230
231  /**
232   * @return true if blocks in this file should be flagged as in-memory
233   */
234  public boolean isInMemory() {
235    return this.inMemory;
236  }
237
238  /**
239   * @return true if data blocks should be written to the cache when an HFile is
240   *         written, false if not
241   */
242  public boolean shouldCacheDataOnWrite() {
243    return this.cacheDataOnWrite;
244  }
245
246  /**
247   * @param cacheDataOnWrite whether data blocks should be written to the cache
248   *                         when an HFile is written
249   */
250  @VisibleForTesting
251  public void setCacheDataOnWrite(boolean cacheDataOnWrite) {
252    this.cacheDataOnWrite = cacheDataOnWrite;
253  }
254
255  /**
256   * @return true if index blocks should be written to the cache when an HFile
257   *         is written, false if not
258   */
259  public boolean shouldCacheIndexesOnWrite() {
260    return this.cacheIndexesOnWrite;
261  }
262
263  /**
264   * @return true if bloom blocks should be written to the cache when an HFile
265   *         is written, false if not
266   */
267  public boolean shouldCacheBloomsOnWrite() {
268    return this.cacheBloomsOnWrite;
269  }
270
271  /**
272   * @return true if blocks should be evicted from the cache when an HFile
273   *         reader is closed, false if not
274   */
275  public boolean shouldEvictOnClose() {
276    return this.evictOnClose;
277  }
278
279  /**
280   * Only used for testing.
281   * @param evictOnClose whether blocks should be evicted from the cache when an
282   *                     HFile reader is closed
283   */
284  @VisibleForTesting
285  public void setEvictOnClose(boolean evictOnClose) {
286    this.evictOnClose = evictOnClose;
287  }
288
289  /**
290   * @return true if data blocks should be compressed in the cache, false if not
291   */
292  public boolean shouldCacheDataCompressed() {
293    return this.cacheDataOnRead && this.cacheDataCompressed;
294  }
295
296  /**
297   * @return true if this {@link BlockCategory} should be compressed in blockcache, false otherwise
298   */
299  public boolean shouldCacheCompressed(BlockCategory category) {
300    switch (category) {
301      case DATA:
302        return this.cacheDataOnRead && this.cacheDataCompressed;
303      default:
304        return false;
305    }
306  }
307
308  /**
309   * @return true if blocks should be prefetched into the cache on open, false if not
310   */
311  public boolean shouldPrefetchOnOpen() {
312    return this.prefetchOnOpen;
313  }
314
315  /**
316   * Return true if we may find this type of block in block cache.
317   * <p>
318   * TODO: today {@code family.isBlockCacheEnabled()} only means {@code cacheDataOnRead}, so here we
319   * consider lots of other configurations such as {@code cacheDataOnWrite}. We should fix this in
320   * the future, {@code cacheDataOnWrite} should honor the CF level {@code isBlockCacheEnabled}
321   * configuration.
322   */
323  public boolean shouldReadBlockFromCache(BlockType blockType) {
324    if (cacheDataOnRead) {
325      return true;
326    }
327    if (prefetchOnOpen) {
328      return true;
329    }
330    if (cacheDataOnWrite) {
331      return true;
332    }
333    if (blockType == null) {
334      return true;
335    }
336    if (blockType.getCategory() == BlockCategory.BLOOM ||
337        blockType.getCategory() == BlockCategory.INDEX) {
338      return true;
339    }
340    return false;
341  }
342
343  /**
344   * If we make sure the block could not be cached, we will not acquire the lock
345   * otherwise we will acquire lock
346   */
347  public boolean shouldLockOnCacheMiss(BlockType blockType) {
348    if (blockType == null) {
349      return true;
350    }
351    return shouldCacheBlockOnRead(blockType.getCategory());
352  }
353
354  /**
355   * Returns the block cache.
356   *
357   * @return the block cache, or null if caching is completely disabled
358   */
359  public Optional<BlockCache> getBlockCache() {
360    return Optional.ofNullable(this.blockCache);
361  }
362
363  @Override
364  public String toString() {
365    return "cacheDataOnRead=" + shouldCacheDataOnRead() + ", cacheDataOnWrite="
366        + shouldCacheDataOnWrite() + ", cacheIndexesOnWrite=" + shouldCacheIndexesOnWrite()
367        + ", cacheBloomsOnWrite=" + shouldCacheBloomsOnWrite() + ", cacheEvictOnClose="
368        + shouldEvictOnClose() + ", cacheDataCompressed=" + shouldCacheDataCompressed()
369        + ", prefetchOnOpen=" + shouldPrefetchOnOpen();
370  }
371}