Skip to content

Commit f1c7c8b

Browse files
committed
HBASE-28004 Persistent cache map can get corrupt if crash happens midway through the write (apache#5341)
Signed-off-by: Ankit Singhal <[email protected]> Reviewed-by: Rahul Agarkar <[email protected]> Change-Id: I577990e1460d6fdc137e1dfcd26e85fed373ed6e
1 parent 4e94a55 commit f1c7c8b

18 files changed

+758
-239
lines changed

hbase-protocol-shaded/src/main/protobuf/BucketCacheEntry.proto

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -32,6 +32,7 @@ message BucketCacheEntry {
3232
map<int32, string> deserializers = 4;
3333
required BackingMap backing_map = 5;
3434
optional bytes checksum = 6;
35+
map<string, bool> prefetched_files = 7;
3536
}
3637

3738
message BackingMap {
@@ -71,6 +72,8 @@ message BucketEntry {
7172
required int64 access_counter = 3;
7273
required int32 deserialiser_index = 4;
7374
required BlockPriority priority = 5;
75+
required int64 cachedTime = 6;
76+
optional int32 disk_size_with_header = 7;
7477
}
7578

7679
enum BlockPriority {

hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/CacheConfig.java

Lines changed: 0 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -93,8 +93,6 @@ public class CacheConfig {
9393
public static final String DROP_BEHIND_CACHE_COMPACTION_KEY =
9494
"hbase.hfile.drop.behind.compaction";
9595

96-
public static final String PREFETCH_PERSISTENCE_PATH_KEY = "hbase.prefetch.file.list.path";
97-
9896
/**
9997
* Configuration key to set interval for persisting bucket cache to disk.
10098
*/

hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/HFileBlock.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -233,7 +233,7 @@ static class Header {
233233
* (This EXTRA info came in with original commit of the bucketcache, HBASE-7404. It was formerly
234234
* known as EXTRA_SERIALIZATION_SPACE).
235235
*/
236-
static final int BLOCK_METADATA_SPACE = Bytes.SIZEOF_BYTE + Bytes.SIZEOF_LONG + Bytes.SIZEOF_INT;
236+
public static final int BLOCK_METADATA_SPACE = Bytes.SIZEOF_BYTE + Bytes.SIZEOF_LONG + Bytes.SIZEOF_INT;
237237

238238
/**
239239
* Each checksum value is an integer that can be stored in 4 bytes.

hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/HFilePreadReader.java

Lines changed: 39 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -18,9 +18,13 @@
1818
package org.apache.hadoop.hbase.io.hfile;
1919

2020
import java.io.IOException;
21+
import java.util.Optional;
22+
import org.apache.commons.lang3.mutable.MutableBoolean;
2123
import org.apache.hadoop.conf.Configuration;
2224
import org.apache.hadoop.fs.Path;
2325
import org.apache.hadoop.hbase.io.FSDataInputStreamWrapper;
26+
import org.apache.hadoop.hbase.io.hfile.bucket.BucketCache;
27+
import org.apache.hadoop.hbase.io.hfile.bucket.BucketEntry;
2428
import org.apache.yetus.audience.InterfaceAudience;
2529
import org.slf4j.Logger;
2630
import org.slf4j.LoggerFactory;
@@ -35,8 +39,14 @@ public class HFilePreadReader extends HFileReaderImpl {
3539
public HFilePreadReader(ReaderContext context, HFileInfo fileInfo, CacheConfig cacheConf,
3640
Configuration conf) throws IOException {
3741
super(context, fileInfo, cacheConf, conf);
42+
final MutableBoolean fileAlreadyCached = new MutableBoolean(false);
43+
BucketCache.getBuckedCacheFromCacheConfig(cacheConf).ifPresent(bc -> fileAlreadyCached
44+
.setValue(bc.getFullyCachedFiles().get(path.getName()) == null ? false : true));
3845
// Prefetch file blocks upon open if requested
39-
if (cacheConf.shouldPrefetchOnOpen() && cacheIfCompactionsOff()) {
46+
if (
47+
cacheConf.shouldPrefetchOnOpen() && cacheIfCompactionsOff()
48+
&& !fileAlreadyCached.booleanValue()
49+
) {
4050
PrefetchExecutor.request(path, new Runnable() {
4151
@Override
4252
public void run() {
@@ -55,12 +65,36 @@ public void run() {
5565
if (LOG.isTraceEnabled()) {
5666
LOG.trace("Prefetch start " + getPathOffsetEndStr(path, offset, end));
5767
}
68+
Optional<BucketCache> bucketCacheOptional =
69+
BucketCache.getBuckedCacheFromCacheConfig(cacheConf);
5870
// Don't use BlockIterator here, because it's designed to read load-on-open section.
5971
long onDiskSizeOfNextBlock = -1;
6072
while (offset < end) {
6173
if (Thread.interrupted()) {
6274
break;
6375
}
76+
// BucketCache can be persistent and resilient to restarts, so we check first if the
77+
// block exists on its in-memory index, if so, we just update the offset and move on
78+
// to the next block without actually going read all the way to the cache.
79+
if (bucketCacheOptional.isPresent()) {
80+
BucketCache cache = bucketCacheOptional.get();
81+
BlockCacheKey cacheKey = new BlockCacheKey(name, offset);
82+
BucketEntry entry = cache.getBackingMap().get(cacheKey);
83+
if (entry != null) {
84+
cacheKey = new BlockCacheKey(name, offset);
85+
entry = cache.getBackingMap().get(cacheKey);
86+
if (entry == null) {
87+
LOG.debug("No cache key {}, we'll read and cache it", cacheKey);
88+
} else {
89+
offset += entry.getOnDiskSizeWithHeader();
90+
LOG.debug("Found cache key {}. Skipping prefetch, the block is already cached.",
91+
cacheKey);
92+
continue;
93+
}
94+
} else {
95+
LOG.debug("No entry in the backing map for cache key {}", cacheKey);
96+
}
97+
}
6498
// Perhaps we got our block from cache? Unlikely as this may be, if it happens, then
6599
// the internal-to-hfileblock thread local which holds the overread that gets the
66100
// next header, will not have happened...so, pass in the onDiskSize gotten from the
@@ -77,12 +111,15 @@ public void run() {
77111
block.release();
78112
}
79113
}
114+
BucketCache.getBuckedCacheFromCacheConfig(cacheConf)
115+
.ifPresent(bc -> bc.fileCacheCompleted(path.getName()));
116+
80117
} catch (IOException e) {
81118
// IOExceptions are probably due to region closes (relocation, etc.)
82119
if (LOG.isTraceEnabled()) {
83120
LOG.trace("Prefetch " + getPathOffsetEndStr(path, offset, end), e);
84121
}
85-
} catch (Exception e) {
122+
} catch (Throwable e) {
86123
// Other exceptions are interesting
87124
LOG.warn("Prefetch " + getPathOffsetEndStr(path, offset, end), e);
88125
} finally {

hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/PrefetchExecutor.java

Lines changed: 4 additions & 90 deletions
Original file line numberDiff line numberDiff line change
@@ -17,11 +17,6 @@
1717
*/
1818
package org.apache.hadoop.hbase.io.hfile;
1919

20-
import java.io.File;
21-
import java.io.FileInputStream;
22-
import java.io.FileOutputStream;
23-
import java.io.IOException;
24-
import java.util.HashMap;
2520
import java.util.Map;
2621
import java.util.concurrent.ConcurrentSkipListMap;
2722
import java.util.concurrent.Future;
@@ -36,29 +31,24 @@
3631
import org.apache.hadoop.fs.Path;
3732
import org.apache.hadoop.hbase.HBaseConfiguration;
3833
import org.apache.hadoop.hbase.HConstants;
34+
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
3935
import org.apache.yetus.audience.InterfaceAudience;
4036
import org.slf4j.Logger;
4137
import org.slf4j.LoggerFactory;
4238

43-
import org.apache.hadoop.hbase.shaded.protobuf.generated.PersistentPrefetchProtos;
44-
4539
@InterfaceAudience.Private
4640
public final class PrefetchExecutor {
4741

4842
private static final Logger LOG = LoggerFactory.getLogger(PrefetchExecutor.class);
4943

5044
/** Futures for tracking block prefetch activity */
5145
private static final Map<Path, Future<?>> prefetchFutures = new ConcurrentSkipListMap<>();
52-
/** Set of files for which prefetch is completed */
53-
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value = "MS_SHOULD_BE_FINAL")
54-
private static HashMap<String, Boolean> prefetchCompleted = new HashMap<>();
5546
/** Executor pool shared among all HFiles for block prefetch */
5647
private static final ScheduledExecutorService prefetchExecutorPool;
5748
/** Delay before beginning prefetch */
5849
private static final int prefetchDelayMillis;
5950
/** Variation in prefetch delay times, to mitigate stampedes */
6051
private static final float prefetchDelayVariation;
61-
static String prefetchedFileListPath;
6252
static {
6353
// Consider doing this on demand with a configuration passed in rather
6454
// than in a static initializer.
@@ -71,7 +61,7 @@ public final class PrefetchExecutor {
7161
prefetchExecutorPool = new ScheduledThreadPoolExecutor(prefetchThreads, new ThreadFactory() {
7262
@Override
7363
public Thread newThread(Runnable r) {
74-
String name = "hfile-prefetch-" + System.currentTimeMillis();
64+
String name = "hfile-prefetch-" + EnvironmentEdgeManager.currentTime();
7565
Thread t = new Thread(r, name);
7666
t.setDaemon(true);
7767
return t;
@@ -88,13 +78,6 @@ public Thread newThread(Runnable r) {
8878
+ HConstants.HREGION_COMPACTIONDIR_NAME.replace(".", "\\.") + Path.SEPARATOR_CHAR + ")");
8979

9080
public static void request(Path path, Runnable runnable) {
91-
if (prefetchCompleted != null) {
92-
if (isFilePrefetched(path.getName())) {
93-
LOG.info(
94-
"File has already been prefetched before the restart, so skipping prefetch : " + path);
95-
return;
96-
}
97-
}
9881
if (!prefetchPathExclude.matcher(path.toString()).find()) {
9982
long delay;
10083
if (prefetchDelayMillis > 0) {
@@ -120,9 +103,8 @@ public static void request(Path path, Runnable runnable) {
120103
public static void complete(Path path) {
121104
prefetchFutures.remove(path);
122105
if (LOG.isDebugEnabled()) {
123-
LOG.debug("Prefetch completed for " + path);
106+
LOG.debug("Prefetch completed for {}", path.getName());
124107
}
125-
prefetchCompleted.put(path.getName(), true);
126108
}
127109

128110
public static void cancel(Path path) {
@@ -131,14 +113,8 @@ public static void cancel(Path path) {
131113
// ok to race with other cancellation attempts
132114
future.cancel(true);
133115
prefetchFutures.remove(path);
134-
if (LOG.isDebugEnabled()) {
135-
LOG.debug("Prefetch cancelled for " + path);
136-
}
116+
LOG.debug("Prefetch cancelled for {}", path);
137117
}
138-
if (LOG.isDebugEnabled()) {
139-
LOG.debug("Removing filename from the prefetched persistence list: {}", path.getName());
140-
}
141-
removePrefetchedFileWhileEvict(path.getName());
142118
}
143119

144120
public static boolean isCompleted(Path path) {
@@ -149,68 +125,6 @@ public static boolean isCompleted(Path path) {
149125
return true;
150126
}
151127

152-
public static void persistToFile(String path) throws IOException {
153-
prefetchedFileListPath = path;
154-
if (prefetchedFileListPath == null) {
155-
LOG.info("Exception while persisting prefetch!");
156-
throw new IOException("Error persisting prefetched HFiles set!");
157-
}
158-
if (!prefetchCompleted.isEmpty()) {
159-
try (FileOutputStream fos = new FileOutputStream(prefetchedFileListPath, false)) {
160-
PrefetchProtoUtils.toPB(prefetchCompleted).writeDelimitedTo(fos);
161-
}
162-
}
163-
}
164-
165-
public static void retrieveFromFile(String path) throws IOException {
166-
prefetchedFileListPath = path;
167-
File prefetchPersistenceFile = new File(prefetchedFileListPath);
168-
if (!prefetchPersistenceFile.exists()) {
169-
LOG.warn("Prefetch persistence file does not exist!");
170-
return;
171-
}
172-
LOG.info("Retrieving from prefetch persistence file " + path);
173-
assert (prefetchedFileListPath != null);
174-
try (FileInputStream fis = deleteFileOnClose(prefetchPersistenceFile)) {
175-
PersistentPrefetchProtos.PrefetchedHfileName proto =
176-
PersistentPrefetchProtos.PrefetchedHfileName.parseDelimitedFrom(fis);
177-
Map<String, Boolean> protoPrefetchedFilesMap = proto.getPrefetchedFilesMap();
178-
prefetchCompleted.putAll(protoPrefetchedFilesMap);
179-
}
180-
}
181-
182-
private static FileInputStream deleteFileOnClose(final File file) throws IOException {
183-
return new FileInputStream(file) {
184-
private File myFile;
185-
186-
private FileInputStream init(File file) {
187-
myFile = file;
188-
return this;
189-
}
190-
191-
@Override
192-
public void close() throws IOException {
193-
if (myFile == null) {
194-
return;
195-
}
196-
197-
super.close();
198-
if (!myFile.delete()) {
199-
throw new IOException("Failed deleting persistence file " + myFile.getAbsolutePath());
200-
}
201-
myFile = null;
202-
}
203-
}.init(file);
204-
}
205-
206-
public static void removePrefetchedFileWhileEvict(String hfileName) {
207-
prefetchCompleted.remove(hfileName);
208-
}
209-
210-
public static boolean isFilePrefetched(String hfileName) {
211-
return prefetchCompleted.containsKey(hfileName);
212-
}
213-
214128
private PrefetchExecutor() {
215129
}
216130
}

0 commit comments

Comments
 (0)