Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

HBASE-24205 Create metric to know the number of reads that happens fr… #1552

Merged
merged 15 commits into from
Jun 16, 2020
Merged
Original file line number Diff line number Diff line change
Expand Up @@ -53,6 +53,10 @@ public interface MetricsRegionSource extends Comparable<MetricsRegionSource> {
String COPROCESSOR_EXECUTION_STATISTICS_DESC = "Statistics for coprocessor execution times";
String REPLICA_ID = "replicaid";
String REPLICA_ID_DESC = "The replica ID of a region. 0 is primary, otherwise is secondary";
String ROW_READS_ONLY_ON_MEMSTORE = "memstoreOnlyRowReadsCount";
String ROW_READS_ONLY_ON_MEMSTORE_DESC = "Row reads happening completely out of memstore";
String MIXED_ROW_READS = "mixedRowReadsCount";
String MIXED_ROW_READS_ON_STORE_DESC = "Row reads happening out of files and memstore on store";

/**
* Close the region's metrics as this region is closing.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,8 @@

package org.apache.hadoop.hbase.regionserver;

import java.util.Map;
import java.util.Map.Entry;
import java.util.concurrent.atomic.AtomicBoolean;

import org.apache.hadoop.hbase.metrics.Interns;
Expand All @@ -33,6 +35,8 @@ public class MetricsRegionSourceImpl implements MetricsRegionSource {

private static final Logger LOG = LoggerFactory.getLogger(MetricsRegionSourceImpl.class);

private static final String _STORE = "_store_";

private AtomicBoolean closed = new AtomicBoolean(false);

// Non-final so that we can null out the wrapper
Expand All @@ -45,6 +49,8 @@ public class MetricsRegionSourceImpl implements MetricsRegionSource {
private final DynamicMetricsRegistry registry;

private final String regionNamePrefix;
private final String regionNamePrefix1;
private final String regionNamePrefix2;
private final String regionPutKey;
private final String regionDeleteKey;
private final String regionGetKey;
Expand Down Expand Up @@ -77,10 +83,11 @@ public MetricsRegionSourceImpl(MetricsRegionWrapper regionWrapper,

registry = agg.getMetricsRegistry();

regionNamePrefix = "Namespace_" + regionWrapper.getNamespace() +
"_table_" + regionWrapper.getTableName() +
"_region_" + regionWrapper.getRegionName() +
"_metric_";
regionNamePrefix1 = "Namespace_" + regionWrapper.getNamespace() +
"_table_" + regionWrapper.getTableName() +
"_region_" + regionWrapper.getRegionName();
regionNamePrefix2 = "_metric_";
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This 'metric' addition to name was of no value?

regionNamePrefix = regionNamePrefix1 + regionNamePrefix2;

String suffix = "Count";

Expand Down Expand Up @@ -302,6 +309,24 @@ void snapshot(MetricsRecordBuilder mrb, boolean ignored) {
regionNamePrefix + MetricsRegionSource.MAX_FLUSH_QUEUE_SIZE,
MetricsRegionSource.MAX_FLUSH_QUEUE_DESC),
this.regionWrapper.getMaxFlushQueueSize());
addCounter(mrb, this.regionWrapper.getMemstoreOnlyRowReadsCount(),
MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE,
MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE_DESC);
addCounter(mrb, this.regionWrapper.getMixedRowReadsCount(),
MetricsRegionSource.MIXED_ROW_READS,
MetricsRegionSource.MIXED_ROW_READS_ON_STORE_DESC);
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We still need this count? We don't have it already with the general read count?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

My aversion to the extra counting is that we already do so much; it costs us loads in cpu. Was trying to do less if we can.

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Are asking in terms of CPU that we add on while collecting the metric? Ifyou see we do collect the metric at the HStore level per row when the StoreScanner completes a row process. That is now a longadder. Seems it is more performant than AtomicLong. Also the above change that we have done at the region level is nothing but just get that metric when that runnable thread keeps running. We don do any metric collection at this level. Are you still thinking it may be a problem. @saintstack ? BTW thanks for your review here.

Copy link
Contributor Author

@ramkrish86 ramkrish86 Jun 17, 2020

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We don't have it already with the general read count?

This is a read count across all stores. But now what we get additionally is per store how much is the read count that hit both memstore and files - also one more where we say how many rows per store came out of memstore only. Ideally the sum of these values per store should be equal to the total read count per region.

}
}

private void addCounter(MetricsRecordBuilder mrb, Map<String, Long> metricMap, String metricName,
String metricDesc) {
if (metricMap != null) {
for (Entry<String, Long> entry : metricMap.entrySet()) {
// append 'store' and its name to the metric
mrb.addCounter(Interns.info(
this.regionNamePrefix1 + _STORE + entry.getKey() + this.regionNamePrefix2 + metricName,
metricDesc), entry.getValue());
}
}
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,8 @@

package org.apache.hadoop.hbase.regionserver;

import java.util.Map;

import org.apache.yetus.audience.InterfaceAudience;

/**
Expand Down Expand Up @@ -170,4 +172,15 @@ public interface MetricsRegionWrapper {
* all compacted store files that belong to this region
*/
long getMaxCompactedStoreFileRefCount();

/**
* @return the number of row reads completely on memstore per store
*/
Map<String, Long> getMemstoreOnlyRowReadsCount();

/**
* @return the number of row reads on memstore and file per store
*/
Map<String, Long> getMixedRowReadsCount();
Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yeah, do we need to keep this count? It doesn't overlap w/ another?


}
Original file line number Diff line number Diff line change
Expand Up @@ -92,7 +92,6 @@ public MetricsTableSource getOrCreateTableSource(String table,
@Override
public void getMetrics(MetricsCollector collector, boolean all) {
MetricsRecordBuilder mrb = collector.addRecord(metricsName);

if (tableSources != null) {
for (MetricsTableSource tableMetricSource : tableSources.values()) {
if (tableMetricSource instanceof MetricsTableSourceImpl) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -61,7 +61,10 @@
import static org.apache.hadoop.hbase.regionserver.MetricsRegionServerSource.SPLIT_SUCCESS_DESC;
import static org.apache.hadoop.hbase.regionserver.MetricsRegionServerSource.SPLIT_SUCCESS_KEY;

import java.util.Map;
import java.util.Map.Entry;
import java.util.concurrent.atomic.AtomicBoolean;

import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.metrics.Interns;
import org.apache.hadoop.metrics2.MetricHistogram;
Expand All @@ -75,6 +78,8 @@
@InterfaceAudience.Private
public class MetricsTableSourceImpl implements MetricsTableSource {

private static final String _STORE = "_store_";

private static final Logger LOG = LoggerFactory.getLogger(MetricsTableSourceImpl.class);

private AtomicBoolean closed = new AtomicBoolean(false);
Expand All @@ -87,6 +92,8 @@ public class MetricsTableSourceImpl implements MetricsTableSource {
private final MetricsTableAggregateSourceImpl agg;
private final DynamicMetricsRegistry registry;
private final String tableNamePrefix;
private final String tableNamePrefixPart1;
private final String tableNamePrefixPart2;
private final TableName tableName;
private final int hashCode;

Expand Down Expand Up @@ -127,8 +134,11 @@ public MetricsTableSourceImpl(String tblName,

this.tableWrapperAgg = tblWrapperAgg;
this.registry = agg.getMetricsRegistry();
this.tableNamePrefix = "Namespace_" + this.tableName.getNamespaceAsString() +
"_table_" + this.tableName.getQualifierAsString() + "_metric_";
this.tableNamePrefixPart1 = "Namespace_" + this.tableName.getNamespaceAsString() +
"_table_" + this.tableName.getQualifierAsString();
this.tableNamePrefixPart2 = "_metric_";
this.tableNamePrefix = tableNamePrefixPart1 +
tableNamePrefixPart2;
this.hashCode = this.tableName.hashCode();
}

Expand Down Expand Up @@ -311,6 +321,25 @@ void snapshot(MetricsRecordBuilder mrb, boolean ignored) {
mrb.addGauge(Interns.info(tableNamePrefix + MetricsRegionServerSource.NUM_REFERENCE_FILES,
MetricsRegionServerSource.NUM_REFERENCE_FILES_DESC),
tableWrapperAgg.getNumReferenceFiles(tableName.getNameAsString()));
addGauge(mrb, tableWrapperAgg.getMemstoreOnlyRowReadsCount(tableName.getNameAsString()),
MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE,
MetricsRegionSource.ROW_READS_ONLY_ON_MEMSTORE_DESC);
addGauge(mrb, tableWrapperAgg.getMixedRowReadsCount(tableName.getNameAsString()),
MetricsRegionSource.MIXED_ROW_READS,
MetricsRegionSource.MIXED_ROW_READS_ON_STORE_DESC);
}
}
}

private void addGauge(MetricsRecordBuilder mrb, Map<String, Long> metricMap, String metricName,
String metricDesc) {
if (metricMap != null) {
for (Entry<String, Long> entry : metricMap.entrySet()) {
// append 'store' and its name to the metric
mrb.addGauge(Interns.info(this.tableNamePrefixPart1 + _STORE
+ entry.getKey().split(MetricsTableWrapperAggregate.UNDERSCORE)[1]
Copy link
Contributor

@anoopsjohn anoopsjohn Jun 11, 2020

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Here the key will be <regionName>_<storeName>?
At this level we should call CF not store?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

You mean the key should be Columnfamily? So _STORE - that should be _ColumnFamily?

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Store is an instance of CF. So here what we have is an aggregated value across all instances (Stores) of a given CF in a table. So the name CF make sense than STORE? WDYT

Copy link
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This looks like it was addressed.

+ this.tableNamePrefixPart2 + metricName,
metricDesc), entry.getValue());
}
}
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,8 @@

package org.apache.hadoop.hbase.regionserver;

import java.util.Map;

import org.apache.yetus.audience.InterfaceAudience;

/**
Expand All @@ -26,7 +28,7 @@
*/
@InterfaceAudience.Private
public interface MetricsTableWrapperAggregate {

public String UNDERSCORE = "_";
/**
* Get the number of read requests that have been issued against this table
*/
Expand Down Expand Up @@ -107,6 +109,13 @@ public interface MetricsTableWrapperAggregate {
*/
long getNumReferenceFiles(String table);

/**
* @return number of row reads completely from memstore per store for this table
*/
Map<String, Long> getMemstoreOnlyRowReadsCount(String table);


/**
* @return number of row reads from file and memstore per store for this table
*/
Map<String, Long> getMixedRowReadsCount(String table);
}
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,9 @@

package org.apache.hadoop.hbase.regionserver;

import java.util.HashMap;
import java.util.Map;

public class MetricsTableWrapperStub implements MetricsTableWrapperAggregate {

private String tableName;
Expand Down Expand Up @@ -109,4 +112,18 @@ public long getAvgRegionSize(String table) {
public long getCpRequestsCount(String table) {
return 99;
}

@Override
public Map<String, Long> getMemstoreOnlyRowReadsCount(String table) {
Map<String, Long> map = new HashMap<String, Long>();
map.put("table_info", 3L);
return map;
}

@Override
public Map<String, Long> getMixedRowReadsCount(String table) {
Map<String, Long> map = new HashMap<String, Long>();
map.put("table_info", 3L);
return map;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,9 @@
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertNotEquals;

import java.util.HashMap;
import java.util.Map;

import org.apache.hadoop.hbase.CompatibilitySingletonFactory;
import org.apache.hadoop.hbase.HBaseClassTestRule;
import org.apache.hadoop.hbase.testclassification.MetricsTests;
Expand Down Expand Up @@ -216,5 +219,19 @@ public long getMaxFlushQueueSize() {
public long getTotalRequestCount() {
return 0;
}

@Override
public Map<String, Long> getMemstoreOnlyRowReadsCount() {
Map<String, Long> map = new HashMap<String, Long>();
map.put("info", 0L);
return map;
}

@Override
public Map<String, Long> getMixedRowReadsCount() {
Map<String, Long> map = new HashMap<String, Long>();
map.put("info", 0L);
return map;
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -46,12 +46,14 @@
import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.atomic.AtomicLong;
import java.util.concurrent.atomic.LongAdder;
import java.util.concurrent.locks.ReentrantLock;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import java.util.function.Predicate;
import java.util.function.ToLongFunction;
import java.util.stream.Collectors;
import java.util.stream.LongStream;

import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
Expand Down Expand Up @@ -94,6 +96,8 @@
import org.apache.hadoop.hbase.regionserver.wal.WALUtil;
import org.apache.hadoop.hbase.security.EncryptionUtil;
import org.apache.hadoop.hbase.security.User;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.ChecksumType;
import org.apache.hadoop.hbase.util.ClassSize;
Expand All @@ -102,9 +106,6 @@
import org.apache.hadoop.hbase.util.ReflectionUtils;
import org.apache.hadoop.util.StringUtils;
import org.apache.hadoop.util.StringUtils.TraditionalBinaryPrefix;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableCollection;
Expand All @@ -114,9 +115,9 @@
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
import org.apache.hbase.thirdparty.org.apache.commons.collections4.CollectionUtils;
import org.apache.hbase.thirdparty.org.apache.commons.collections4.IterableUtils;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor;

import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* A Store holds a column family in a Region. Its a memstore and a set of zero
* or more StoreFiles, which stretch backwards over time.
Expand Down Expand Up @@ -162,6 +163,9 @@ public class HStore implements Store, HeapSize, StoreConfigInformation,
volatile boolean forceMajor = false;
private AtomicLong storeSize = new AtomicLong();
private AtomicLong totalUncompressedBytes = new AtomicLong();
private LongAdder memstoreOnlyRowReadsCount = new LongAdder();
// rows that has cells from both memstore and files (or only files)
private LongAdder mixedRowReadsCount = new LongAdder();

private boolean cacheOnWriteLogged;

Expand Down Expand Up @@ -331,7 +335,8 @@ protected HStore(final HRegion region, final ColumnFamilyDescriptor family,
confPrintThreshold = 10;
}
this.parallelPutCountPrintThreshold = confPrintThreshold;
LOG.info("{} created, memstore type={}, storagePolicy={}, verifyBulkLoads={}, "

LOG.info("Store={}, memstore type={}, storagePolicy={}, verifyBulkLoads={}, "
+ "parallelPutCountPrintThreshold={}, encoding={}, compression={}",
this, memstore.getClass().getSimpleName(), policyName, verifyBulkLoads,
parallelPutCountPrintThreshold, family.getDataBlockEncoding(),
Expand Down Expand Up @@ -2544,7 +2549,7 @@ public CacheConfig getCacheConfig() {
}

public static final long FIXED_OVERHEAD =
ClassSize.align(ClassSize.OBJECT + (27 * ClassSize.REFERENCE) + (2 * Bytes.SIZEOF_LONG)
ClassSize.align(ClassSize.OBJECT + (29 * ClassSize.REFERENCE) + (2 * Bytes.SIZEOF_LONG)
+ (6 * Bytes.SIZEOF_INT) + (2 * Bytes.SIZEOF_BOOLEAN));

public static final long DEEP_OVERHEAD = ClassSize.align(FIXED_OVERHEAD
Expand Down Expand Up @@ -2882,8 +2887,7 @@ public int getStoreRefCount() {
}

/**
* @return get maximum ref count of storeFile among all compacted HStore Files
* for the HStore
* @return get maximum ref count of storeFile among all compacted HStore Files for the HStore
*/
public int getMaxCompactedStoreFileRefCount() {
OptionalInt maxCompactedStoreFileRefCount = this.storeEngine.getStoreFileManager()
Expand All @@ -2897,4 +2901,21 @@ public int getMaxCompactedStoreFileRefCount() {
? maxCompactedStoreFileRefCount.getAsInt() : 0;
}

@Override
public long getMemstoreOnlyRowReadsCount() {
return memstoreOnlyRowReadsCount.sum();
}

@Override
public long getMixedRowReadsCount() {
return mixedRowReadsCount.sum();
}

void updateMetricsStore(boolean memstoreRead) {
if (memstoreRead) {
memstoreOnlyRowReadsCount.increment();
} else {
mixedRowReadsCount.increment();
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -112,6 +112,10 @@ public Cell peek() {
return this.current.peek();
}

boolean isLatestCellFromMemstore() {
return !this.current.isFileScanner();
}

@Override
public Cell next() throws IOException {
if(this.current == null) {
Expand Down
Loading