virajjasani commented on a change in pull request #1681:
URL: https://github.com/apache/hbase/pull/1681#discussion_r426263602
##########
File path: hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java
##########
@@ -1532,6 +1532,16 @@
"hbase.regionserver.slowlog.buffer.enabled";
public static final boolean DEFAULT_ONLINE_LOG_PROVIDER_ENABLED = false;
+ /** The slowlog info family as a string*/
+ private static final String SLOWLOG_INFO_FAMILY_STR = "info";
+
+ /** The slowlog info family */
+ public static final byte [] SLOWLOG_INFO_FAMILY =
Bytes.toBytes(SLOWLOG_INFO_FAMILY_STR);
+
+ public static final String SLOW_LOG_SYS_TABLE_ENABLED_KEY =
+ "hbase.regionserver.slowlog.systable.enabled";
+ public static final boolean DEFAULT_SLOW_LOG_SYS_TABLE_ENABLED_KEY = false;
Review comment:
Yes, this is region name that comes as part of req param. We do record
it, but in ringbuffer and systable, we record the full name, whereas in
RpcServer log, we record truncated name. The truncation was one of the first
reasons behind the idea of online ringbuffer implementation.
##########
File path:
hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/slowlog/LogEventHandler.java
##########
@@ -53,12 +55,28 @@
class LogEventHandler implements EventHandler<RingBufferEnvelope> {
private static final Logger LOG =
LoggerFactory.getLogger(LogEventHandler.class);
+ private static final int SYS_TABLE_QUEUE_SIZE = 1000;
Review comment:
That will conflict with the existing one. We have
`hbase.regionserver.slowlog.ringbuffer.size` but this is only related to in
memory ring buffer and has nothing to do with system table. If you believe we
can increase this size, we can do it. It is rough estimate that within 10 min,
we can have slow/large logs count within 1000.
##########
File path:
hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/slowlog/LogEventHandler.java
##########
@@ -53,12 +55,28 @@
class LogEventHandler implements EventHandler<RingBufferEnvelope> {
private static final Logger LOG =
LoggerFactory.getLogger(LogEventHandler.class);
+ private static final int SYS_TABLE_QUEUE_SIZE = 1000;
- private final Queue<SlowLogPayload> queue;
+ private final Queue<SlowLogPayload> queueForRingBuffer;
+ private final Queue<SlowLogPayload> queueForSysTable;
+ private final boolean isSlowLogTableEnabled;
- LogEventHandler(int eventCount) {
+ private Configuration configuration;
+
+ private static final ReentrantLock LOCK = new ReentrantLock();
+
+ LogEventHandler(int eventCount, boolean isSlowLogTableEnabled, Configuration
conf) {
+ this.configuration = conf;
EvictingQueue<SlowLogPayload> evictingQueue =
EvictingQueue.create(eventCount);
- queue = Queues.synchronizedQueue(evictingQueue);
+ queueForRingBuffer = Queues.synchronizedQueue(evictingQueue);
+ this.isSlowLogTableEnabled = isSlowLogTableEnabled;
+ if (isSlowLogTableEnabled) {
+ EvictingQueue<SlowLogPayload> evictingQueueForTable =
EvictingQueue.create(
Review comment:
One Q is literally managed by user only and it's purpose is to server
online slowlogs from memory. Another Q is for cron to insert records in system
table, hence both should not be same. Even if user clears the Q, the one
intended is in-memory ring buffer Q. If user has opted for system table, the
other Q is totally managed internally and not upto user to manage. User might
also opt for lower size for config `hbase.regionserver.slowlog.ringbuffer.size`
to say 100/50. But there is no guarantee that within 10 min of cron run, the no
of slow RPC calls would be less than 50, and hence better to have another Q for
systable maintenance.
##########
File path:
hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/slowlog/LogEventHandler.java
##########
@@ -160,7 +183,7 @@ boolean clearSlowLogs() {
if (LOG.isDebugEnabled()) {
LOG.debug("Received request to clean up online slowlog buffer..");
}
- queue.clear();
+ queueForRingBuffer.clear();
Review comment:
Yes, this is intended since another Q is quite internal for systable
insertion by cron.
##########
File path:
hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/slowlog/LogEventHandler.java
##########
@@ -129,7 +147,12 @@ public void onEvent(RingBufferEnvelope event, long
sequence, boolean endOfBatch)
.setType(type)
.setUserName(userName)
.build();
- queue.add(slowLogPayload);
+ queueForRingBuffer.add(slowLogPayload);
Review comment:
Actually this is for in-memory ring buffer and has nothing to do with
below check. `isSlowLogTableEnabled` is for `queueForSysTable` only. If the
execution comes till this point, that means we do at least need to create
in-memory ring buffer and that one is `queueForRingBuffer` (this is previous
feature).
##########
File path:
hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/slowlog/SlowLogTableAccessor.java
##########
@@ -0,0 +1,140 @@
+/*
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License. You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hbase.regionserver.slowlog;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Random;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.HConstants;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.client.Connection;
+import org.apache.hadoop.hbase.client.ConnectionFactory;
+import org.apache.hadoop.hbase.client.Durability;
+import org.apache.hadoop.hbase.client.Put;
+import org.apache.hadoop.hbase.client.Table;
+import org.apache.hadoop.hbase.shaded.protobuf.generated.TooSlowLog;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
+import org.apache.yetus.audience.InterfaceAudience;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Slowlog Accessor to record slow/large RPC log identified at each
RegionServer RpcServer level.
+ * This can be done only optionally to record the entire history of slow/large
rpc calls
+ * since RingBuffer can handle only limited latest records.
+ */
[email protected]
+public class SlowLogTableAccessor {
+
+ private static final Logger LOG =
LoggerFactory.getLogger(SlowLogTableAccessor.class);
+
+ private static final Random RANDOM = new Random();
+
+ private static Connection connection;
+
+ private static void doPut(final Connection connection, final List<Put> puts)
+ throws IOException {
+ try (Table table = connection.getTable(TableName.SLOW_LOG_TABLE_NAME)) {
+ table.put(puts);
+ }
+ }
+
+ /**
+ * Add slow/large log records to hbase:slowlog table
+ * @param slowLogPayloads List of SlowLogPayload to process
+ * @param configuration Configuration to use for connection
+ */
+ public static void addSlowLogRecords(final List<TooSlowLog.SlowLogPayload>
slowLogPayloads,
+ final Configuration configuration) {
+ List<Put> puts = new ArrayList<>(slowLogPayloads.size());
+ for (TooSlowLog.SlowLogPayload slowLogPayload : slowLogPayloads) {
+ final byte[] rowKey = getRowKey(slowLogPayload);
+ final Put put = new Put(rowKey).setDurability(Durability.SKIP_WAL)
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("call_details"),
+ Bytes.toBytes(slowLogPayload.getCallDetails()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("client_address"),
+ Bytes.toBytes(slowLogPayload.getClientAddress()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("method_name"),
+ Bytes.toBytes(slowLogPayload.getMethodName()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY, Bytes.toBytes("param"),
+ Bytes.toBytes(slowLogPayload.getParam()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("processing_time"),
+ Bytes.toBytes(Integer.toString(slowLogPayload.getProcessingTime())))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY, Bytes.toBytes("queue_time"),
+ Bytes.toBytes(Integer.toString(slowLogPayload.getQueueTime())))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("region_name"),
+ Bytes.toBytes(slowLogPayload.getRegionName()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("response_size"),
+ Bytes.toBytes(Long.toString(slowLogPayload.getResponseSize())))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY,
Bytes.toBytes("server_class"),
+ Bytes.toBytes(slowLogPayload.getServerClass()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY, Bytes.toBytes("start_time"),
+ Bytes.toBytes(Long.toString(slowLogPayload.getStartTime())))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY, Bytes.toBytes("type"),
+ Bytes.toBytes(slowLogPayload.getType().name()))
+ .addColumn(HConstants.SLOWLOG_INFO_FAMILY, Bytes.toBytes("username"),
+ Bytes.toBytes(slowLogPayload.getUserName()));
+ puts.add(put);
+ }
+ try {
+ if (connection == null) {
+ synchronized (SlowLogTableAccessor.class) {
+ if (connection == null) {
+ Configuration conf = new Configuration(configuration);
+ // rpc timeout: 20s
+ conf.setInt(HConstants.HBASE_RPC_TIMEOUT_KEY, 20000);
+ // retry count: 5
+ conf.setInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, 5);
+ conf.setInt(HConstants.HBASE_CLIENT_SERVERSIDE_RETRIES_MULTIPLIER,
1);
+ connection = ConnectionFactory.createConnection(conf);
+ }
+ }
+ }
+ doPut(connection, puts);
Review comment:
Sure, taken care of in the latest commit.
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]