Skip to content

Commit 27dbcc4

Browse files
ragarkarwchevreuil
authored andcommitted
HBASE-27998 Enhance region metrics to include prefetch ratio for each… (#5342)
Signed-off-by: Wellington Chevreuil <wchevreuil@apache.org> (cherry picked from commit 9e74cc0)
1 parent a91e933 commit 27dbcc4

File tree

16 files changed

+264
-130
lines changed

16 files changed

+264
-130
lines changed

hbase-client/src/main/java/org/apache/hadoop/hbase/RegionMetrics.java

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -132,4 +132,10 @@ default String getNameAsString() {
132132

133133
/** Returns the compaction state of this region */
134134
CompactionState getCompactionState();
135+
136+
/** Returns the total size of the hfiles in the region */
137+
Size getRegionSizeMB();
138+
139+
/** Returns current prefetch ratio of this region on this server */
140+
float getCurrentRegionCachedRatio();
135141
}

hbase-client/src/main/java/org/apache/hadoop/hbase/RegionMetricsBuilder.java

Lines changed: 35 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -79,7 +79,8 @@ public static RegionMetrics toRegionMetrics(ClusterStatusProtos.RegionLoad regio
7979
ClusterStatusProtos.StoreSequenceId::getSequenceId)))
8080
.setUncompressedStoreFileSize(
8181
new Size(regionLoadPB.getStoreUncompressedSizeMB(), Size.Unit.MEGABYTE))
82-
.build();
82+
.setRegionSizeMB(new Size(regionLoadPB.getRegionSizeMB(), Size.Unit.MEGABYTE))
83+
.setCurrentRegionCachedRatio(regionLoadPB.getCurrentRegionCachedRatio()).build();
8384
}
8485

8586
private static List<ClusterStatusProtos.StoreSequenceId>
@@ -118,7 +119,8 @@ public static ClusterStatusProtos.RegionLoad toRegionLoad(RegionMetrics regionMe
118119
.addAllStoreCompleteSequenceId(toStoreSequenceId(regionMetrics.getStoreSequenceId()))
119120
.setStoreUncompressedSizeMB(
120121
(int) regionMetrics.getUncompressedStoreFileSize().get(Size.Unit.MEGABYTE))
121-
.build();
122+
.setRegionSizeMB((int) regionMetrics.getRegionSizeMB().get(Size.Unit.MEGABYTE))
123+
.setCurrentRegionCachedRatio(regionMetrics.getCurrentRegionCachedRatio()).build();
122124
}
123125

124126
public static RegionMetricsBuilder newBuilder(byte[] name) {
@@ -151,6 +153,8 @@ public static RegionMetricsBuilder newBuilder(byte[] name) {
151153
private long blocksLocalWithSsdWeight;
152154
private long blocksTotalWeight;
153155
private CompactionState compactionState;
156+
private Size regionSizeMB = Size.ZERO;
157+
private float currentRegionCachedRatio;
154158

155159
private RegionMetricsBuilder(byte[] name) {
156160
this.name = name;
@@ -281,14 +285,24 @@ public RegionMetricsBuilder setCompactionState(CompactionState compactionState)
281285
return this;
282286
}
283287

288+
public RegionMetricsBuilder setRegionSizeMB(Size value) {
289+
this.regionSizeMB = value;
290+
return this;
291+
}
292+
293+
public RegionMetricsBuilder setCurrentRegionCachedRatio(float value) {
294+
this.currentRegionCachedRatio = value;
295+
return this;
296+
}
297+
284298
public RegionMetrics build() {
285299
return new RegionMetricsImpl(name, storeCount, storeFileCount, storeRefCount,
286300
maxCompactedStoreFileRefCount, compactingCellCount, compactedCellCount, storeFileSize,
287301
memStoreSize, indexSize, rootLevelIndexSize, uncompressedDataIndexSize, bloomFilterSize,
288302
uncompressedStoreFileSize, writeRequestCount, readRequestCount, filteredReadRequestCount,
289303
completedSequenceId, storeSequenceIds, dataLocality, lastMajorCompactionTimestamp,
290304
dataLocalityForSsd, blocksLocalWeight, blocksLocalWithSsdWeight, blocksTotalWeight,
291-
compactionState);
305+
compactionState, regionSizeMB, currentRegionCachedRatio);
292306
}
293307

294308
private static class RegionMetricsImpl implements RegionMetrics {
@@ -318,6 +332,8 @@ private static class RegionMetricsImpl implements RegionMetrics {
318332
private final long blocksLocalWithSsdWeight;
319333
private final long blocksTotalWeight;
320334
private final CompactionState compactionState;
335+
private final Size regionSizeMB;
336+
private final float currentRegionCachedRatio;
321337

322338
RegionMetricsImpl(byte[] name, int storeCount, int storeFileCount, int storeRefCount,
323339
int maxCompactedStoreFileRefCount, final long compactingCellCount, long compactedCellCount,
@@ -326,7 +342,8 @@ private static class RegionMetricsImpl implements RegionMetrics {
326342
long writeRequestCount, long readRequestCount, long filteredReadRequestCount,
327343
long completedSequenceId, Map<byte[], Long> storeSequenceIds, float dataLocality,
328344
long lastMajorCompactionTimestamp, float dataLocalityForSsd, long blocksLocalWeight,
329-
long blocksLocalWithSsdWeight, long blocksTotalWeight, CompactionState compactionState) {
345+
long blocksLocalWithSsdWeight, long blocksTotalWeight, CompactionState compactionState,
346+
Size regionSizeMB, float currentRegionCachedRatio) {
330347
this.name = Preconditions.checkNotNull(name);
331348
this.storeCount = storeCount;
332349
this.storeFileCount = storeFileCount;
@@ -353,6 +370,8 @@ private static class RegionMetricsImpl implements RegionMetrics {
353370
this.blocksLocalWithSsdWeight = blocksLocalWithSsdWeight;
354371
this.blocksTotalWeight = blocksTotalWeight;
355372
this.compactionState = compactionState;
373+
this.regionSizeMB = regionSizeMB;
374+
this.currentRegionCachedRatio = currentRegionCachedRatio;
356375
}
357376

358377
@Override
@@ -485,6 +504,16 @@ public CompactionState getCompactionState() {
485504
return compactionState;
486505
}
487506

507+
@Override
508+
public Size getRegionSizeMB() {
509+
return regionSizeMB;
510+
}
511+
512+
@Override
513+
public float getCurrentRegionCachedRatio() {
514+
return currentRegionCachedRatio;
515+
}
516+
488517
@Override
489518
public String toString() {
490519
StringBuilder sb =
@@ -524,6 +553,8 @@ public String toString() {
524553
Strings.appendKeyValue(sb, "blocksLocalWithSsdWeight", blocksLocalWithSsdWeight);
525554
Strings.appendKeyValue(sb, "blocksTotalWeight", blocksTotalWeight);
526555
Strings.appendKeyValue(sb, "compactionState", compactionState);
556+
Strings.appendKeyValue(sb, "regionSizeMB", regionSizeMB);
557+
Strings.appendKeyValue(sb, "currentRegionCachedRatio", currentRegionCachedRatio);
527558
return sb.toString();
528559
}
529560
}

hbase-client/src/main/java/org/apache/hadoop/hbase/ServerMetrics.java

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -100,4 +100,10 @@ default String getVersion() {
100100
@Nullable
101101
List<ServerTask> getTasks();
102102

103+
/**
104+
* Returns the region cache information for the regions hosted on this server
105+
* @return map of region encoded name and the size of the region cached on this region server
106+
* rounded to MB
107+
*/
108+
Map<String, Integer> getRegionCachedInfo();
103109
}

hbase-client/src/main/java/org/apache/hadoop/hbase/ServerMetricsBuilder.java

Lines changed: 18 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -83,6 +83,7 @@ public static ServerMetrics toServerMetrics(ServerName serverName, int versionNu
8383
: null)
8484
.setTasks(serverLoadPB.getTasksList().stream().map(ProtobufUtil::getServerTask)
8585
.collect(Collectors.toList()))
86+
.setRegionCachedInfo(serverLoadPB.getRegionCachedInfoMap())
8687
.setReportTimestamp(serverLoadPB.getReportEndTime())
8788
.setLastReportTimestamp(serverLoadPB.getReportStartTime()).setVersionNumber(versionNumber)
8889
.setVersion(version).build();
@@ -109,6 +110,7 @@ public static ClusterStatusProtos.ServerLoad toServerLoad(ServerMetrics metrics)
109110
.map(ProtobufUtil::toReplicationLoadSource).collect(Collectors.toList()))
110111
.addAllTasks(
111112
metrics.getTasks().stream().map(ProtobufUtil::toServerTask).collect(Collectors.toList()))
113+
.putAllRegionCachedInfo(metrics.getRegionCachedInfo())
112114
.setReportStartTime(metrics.getLastReportTimestamp())
113115
.setReportEndTime(metrics.getReportTimestamp());
114116
if (metrics.getReplicationLoadSink() != null) {
@@ -138,6 +140,7 @@ public static ServerMetricsBuilder newBuilder(ServerName sn) {
138140
private long reportTimestamp = EnvironmentEdgeManager.currentTime();
139141
private long lastReportTimestamp = 0;
140142
private final List<ServerTask> tasks = new ArrayList<>();
143+
private Map<String, Integer> regionCachedInfo = new HashMap<>();
141144

142145
private ServerMetricsBuilder(ServerName serverName) {
143146
this.serverName = serverName;
@@ -218,10 +221,15 @@ public ServerMetricsBuilder setTasks(List<ServerTask> tasks) {
218221
return this;
219222
}
220223

224+
public ServerMetricsBuilder setRegionCachedInfo(Map<String, Integer> value) {
225+
this.regionCachedInfo = value;
226+
return this;
227+
}
228+
221229
public ServerMetrics build() {
222230
return new ServerMetricsImpl(serverName, versionNumber, version, requestCountPerSecond,
223231
requestCount, usedHeapSize, maxHeapSize, infoServerPort, sources, sink, regionStatus,
224-
coprocessorNames, reportTimestamp, lastReportTimestamp, userMetrics, tasks);
232+
coprocessorNames, reportTimestamp, lastReportTimestamp, userMetrics, tasks, regionCachedInfo);
225233
}
226234

227235
private static class ServerMetricsImpl implements ServerMetrics {
@@ -242,12 +250,14 @@ private static class ServerMetricsImpl implements ServerMetrics {
242250
private final long lastReportTimestamp;
243251
private final Map<byte[], UserMetrics> userMetrics;
244252
private final List<ServerTask> tasks;
253+
private final Map<String, Integer> regionCachedInfo;
245254

246255
ServerMetricsImpl(ServerName serverName, int versionNumber, String version,
247256
long requestCountPerSecond, long requestCount, Size usedHeapSize, Size maxHeapSize,
248257
int infoServerPort, List<ReplicationLoadSource> sources, ReplicationLoadSink sink,
249258
Map<byte[], RegionMetrics> regionStatus, Set<String> coprocessorNames, long reportTimestamp,
250-
long lastReportTimestamp, Map<byte[], UserMetrics> userMetrics, List<ServerTask> tasks) {
259+
long lastReportTimestamp, Map<byte[], UserMetrics> userMetrics, List<ServerTask> tasks,
260+
Map<String, Integer> regionCachedInfo) {
251261
this.serverName = Preconditions.checkNotNull(serverName);
252262
this.versionNumber = versionNumber;
253263
this.version = version;
@@ -264,6 +274,7 @@ private static class ServerMetricsImpl implements ServerMetrics {
264274
this.reportTimestamp = reportTimestamp;
265275
this.lastReportTimestamp = lastReportTimestamp;
266276
this.tasks = tasks;
277+
this.regionCachedInfo = regionCachedInfo;
267278
}
268279

269280
@Override
@@ -356,6 +367,11 @@ public List<ServerTask> getTasks() {
356367
return tasks;
357368
}
358369

370+
@Override
371+
public Map<String, Integer> getRegionCachedInfo() {
372+
return Collections.unmodifiableMap(regionCachedInfo);
373+
}
374+
359375
@Override
360376
public String toString() {
361377
int storeCount = 0;

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

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -32,7 +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;
35+
map<string, RegionFileSizeMap> cached_files = 7;
3636
}
3737

3838
message BackingMap {
@@ -81,3 +81,9 @@ enum BlockPriority {
8181
multi = 1;
8282
memory = 2;
8383
}
84+
85+
message RegionFileSizeMap {
86+
required string region_name = 1;
87+
required uint64 region_cached_size = 2;
88+
}
89+

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

Lines changed: 11 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -177,6 +177,12 @@ message RegionLoad {
177177
MAJOR = 2;
178178
MAJOR_AND_MINOR = 3;
179179
}
180+
181+
/** Total region size in MB */
182+
optional uint32 region_size_MB = 28;
183+
184+
/** Current region cache ratio on this server */
185+
optional float current_region_cached_ratio = 29;
180186
}
181187

182188
message UserLoad {
@@ -301,6 +307,11 @@ message ServerLoad {
301307
*/
302308
repeated UserLoad userLoads = 12;
303309

310+
/**
311+
* The metrics for region cached on this region server
312+
*/
313+
map<string, uint32> regionCachedInfo = 13;
314+
304315
/**
305316
* The active monitored tasks
306317
*/

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

Lines changed: 0 additions & 36 deletions
This file was deleted.

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

Lines changed: 5 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,8 @@
2020
import java.util.Iterator;
2121
import java.util.Map;
2222
import java.util.Optional;
23+
import org.apache.hadoop.fs.Path;
24+
import org.apache.hadoop.hbase.util.Pair;
2325
import org.apache.yetus.audience.InterfaceAudience;
2426

2527
/**
@@ -154,8 +156,8 @@ default boolean isMetaBlock(BlockType blockType) {
154156
* made into the cache).
155157
* @param fileName the file that has been completely cached.
156158
*/
157-
default void notifyFileCachingCompleted(String fileName, int totalBlockCount,
158-
int dataBlockCount) {
159+
default void notifyFileCachingCompleted(Path fileName, int totalBlockCount, int dataBlockCount,
160+
long size) {
159161
// noop
160162
}
161163

@@ -225,7 +227,7 @@ default Optional<Integer> getBlockSize(BlockCacheKey key) {
225227
* @return empty optional if this method is not supported, otherwise the returned optional
226228
* contains a map of all files that have been fully cached.
227229
*/
228-
default Optional<Map<String, Boolean>> getFullyCachedFiles() {
230+
default Optional<Map<String, Pair<String, Long>>> getFullyCachedFiles() {
229231
return Optional.empty();
230232
}
231233
}

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

Lines changed: 7 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -22,8 +22,10 @@
2222
import java.util.Optional;
2323
import org.apache.commons.lang3.mutable.Mutable;
2424
import org.apache.commons.lang3.mutable.MutableBoolean;
25+
import org.apache.hadoop.fs.Path;
2526
import org.apache.hadoop.hbase.io.HeapSize;
2627
import org.apache.hadoop.hbase.io.hfile.bucket.BucketCache;
28+
import org.apache.hadoop.hbase.util.Pair;
2729
import org.apache.yetus.audience.InterfaceAudience;
2830
import org.slf4j.Logger;
2931
import org.slf4j.LoggerFactory;
@@ -423,7 +425,7 @@ public BlockCache[] getBlockCaches() {
423425
* Returns the list of fully cached files
424426
*/
425427
@Override
426-
public Optional<Map<String, Boolean>> getFullyCachedFiles() {
428+
public Optional<Map<String, Pair<String, Long>>> getFullyCachedFiles() {
427429
return this.l2Cache.getFullyCachedFiles();
428430
}
429431

@@ -447,10 +449,11 @@ public BlockCache getSecondLevelCache() {
447449
}
448450

449451
@Override
450-
public void notifyFileCachingCompleted(String fileName, int totalBlockCount, int dataBlockCount) {
452+
public void notifyFileCachingCompleted(Path fileName, int totalBlockCount, int dataBlockCount,
453+
long size) {
451454
l1Cache.getBlockCount();
452-
l1Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount);
453-
l2Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount);
455+
l1Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount, size);
456+
l2Cache.notifyFileCachingCompleted(fileName, totalBlockCount, dataBlockCount, size);
454457

455458
}
456459

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

Lines changed: 2 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -128,8 +128,8 @@ public void run() {
128128
}
129129
}
130130
if (!interrupted) {
131-
cacheConf.getBlockCache().get().notifyFileCachingCompleted(path.getName(), blockCount,
132-
dataBlockCount);
131+
cacheConf.getBlockCache().get().notifyFileCachingCompleted(path, blockCount,
132+
dataBlockCount, offset);
133133
}
134134
} catch (IOException e) {
135135
// IOExceptions are probably due to region closes (relocation, etc.)
@@ -147,7 +147,6 @@ public void run() {
147147
LOG.warn("Close prefetch stream reader failed, path: " + path, e);
148148
}
149149
}
150-
String regionName = getRegionName(path);
151150
PrefetchExecutor.complete(path);
152151
}
153152
}

0 commit comments

Comments
 (0)