This is an automated email from the ASF dual-hosted git repository.
aleksey pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git
The following commit(s) were added to refs/heads/trunk by this push:
new 1266fec Optimize NoSpamLogger use in hot paths
1266fec is described below
commit 1266fec349e76b964b522d11460f1df4adadcb48
Author: Caleb Rackliffe <[email protected]>
AuthorDate: Fri Jul 17 16:35:00 2020 -0500
Optimize NoSpamLogger use in hot paths
patch by Caleb Rackliffe; reviewed by Zhao Yang and Aleksey Yeschenko
for CASSANDRA-15766
---
CHANGES.txt | 1 +
.../cassandra/net/InboundMessageHandler.java | 16 +++++++----
.../apache/cassandra/net/OutboundConnection.java | 26 ++++++++++++++----
.../apache/cassandra/utils/memory/BufferPool.java | 32 ++++++++++++++++------
.../cassandra/utils/memory/LongBufferPoolTest.java | 6 ++--
.../cassandra/metrics/BufferPoolMetricsTest.java | 4 +--
.../cassandra/utils/memory/BufferPoolTest.java | 12 ++++----
7 files changed, 66 insertions(+), 31 deletions(-)
diff --git a/CHANGES.txt b/CHANGES.txt
index b56ee27..90d0129 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
4.0-beta2
+ * Optimize NoSpamLogger use in hot paths (CASSANDRA-15766)
* Verify sstable components on startup (CASSANDRA-15945)
Merged from 3.11:
* Frozen RawTuple is not annotated with frozen in the toString method
(CASSANDRA-15857)
diff --git a/src/java/org/apache/cassandra/net/InboundMessageHandler.java
b/src/java/org/apache/cassandra/net/InboundMessageHandler.java
index 1fc182b..534128e 100644
--- a/src/java/org/apache/cassandra/net/InboundMessageHandler.java
+++ b/src/java/org/apache/cassandra/net/InboundMessageHandler.java
@@ -325,7 +325,7 @@ public class InboundMessageHandler extends
ChannelInboundHandlerAdapter implemen
catch (IncompatibleSchemaException e)
{
callbacks.onFailedDeserialize(size, header, e);
- noSpamLogger.info("{} incompatible schema encountered while
deserializing a message", id(), e);
+ noSpamLogger.info("{} incompatible schema encountered while
deserializing a message", this, e);
}
catch (Throwable t)
{
@@ -415,20 +415,20 @@ public class InboundMessageHandler extends
ChannelInboundHandlerAdapter implemen
{
receivedBytes += frame.frameSize;
corruptFramesRecovered++;
- noSpamLogger.warn("{} invalid, recoverable CRC mismatch detected
while reading messages (corrupted self-contained frame)", id());
+ noSpamLogger.warn("{} invalid, recoverable CRC mismatch detected
while reading messages (corrupted self-contained frame)", this);
}
else if (null == largeMessage) // first frame of a large message
{
receivedBytes += frame.frameSize;
corruptFramesUnrecovered++;
- noSpamLogger.error("{} invalid, unrecoverable CRC mismatch
detected while reading messages (corrupted first frame of a large message)",
id());
+ noSpamLogger.error("{} invalid, unrecoverable CRC mismatch
detected while reading messages (corrupted first frame of a large message)",
this);
throw new InvalidCrc(frame.readCRC, frame.computedCRC);
}
else // subsequent frame of a large message
{
processSubsequentFrameOfLargeMessage(frame);
corruptFramesRecovered++;
- noSpamLogger.warn("{} invalid, recoverable CRC mismatch detected
while reading a large message", id());
+ noSpamLogger.warn("{} invalid, recoverable CRC mismatch detected
while reading a large message", this);
}
}
@@ -692,6 +692,12 @@ public class InboundMessageHandler extends
ChannelInboundHandlerAdapter implemen
return SocketFactory.channelId(peer, self, type,
channel.id().asShortText());
}
+ @Override
+ public String toString()
+ {
+ return id();
+ }
+
/*
* A large-message frame-accumulating state machine.
*
@@ -822,7 +828,7 @@ public class InboundMessageHandler extends
ChannelInboundHandlerAdapter implemen
catch (IncompatibleSchemaException e)
{
callbacks.onFailedDeserialize(size, header, e);
- noSpamLogger.info("{} incompatible schema encountered while
deserializing a message", id(), e);
+ noSpamLogger.info("{} incompatible schema encountered while
deserializing a message", InboundMessageHandler.this, e);
}
catch (Throwable t)
{
diff --git a/src/java/org/apache/cassandra/net/OutboundConnection.java
b/src/java/org/apache/cassandra/net/OutboundConnection.java
index 635f221..e7eca86 100644
--- a/src/java/org/apache/cassandra/net/OutboundConnection.java
+++ b/src/java/org/apache/cassandra/net/OutboundConnection.java
@@ -66,6 +66,7 @@ import static
org.apache.cassandra.net.OutboundConnections.LARGE_MESSAGE_THRESHO
import static org.apache.cassandra.net.ResourceLimits.*;
import static org.apache.cassandra.net.ResourceLimits.Outcome.*;
import static org.apache.cassandra.net.SocketFactory.*;
+import static org.apache.cassandra.utils.FBUtilities.prettyPrintMemory;
import static org.apache.cassandra.utils.MonotonicClock.approxTime;
import static org.apache.cassandra.utils.Throwables.isCausedBy;
@@ -120,6 +121,18 @@ public class OutboundConnection
* we allocate from here whenever queueSize > queueCapacity */
private final EndpointAndGlobal reserveCapacityInBytes;
+ /** Used in logging statements to lazily build a human-readable number of
pending bytes. */
+ private final Object readablePendingBytes =
+ new Object() { @Override public String toString() { return
prettyPrintMemory(pendingBytes()); } };
+
+ /** Used in logging statements to lazily build a human-readable number of
reserve endpoint bytes in use. */
+ private final Object readableReserveEndpointUsing =
+ new Object() { @Override public String toString() { return
prettyPrintMemory(reserveCapacityInBytes.endpoint.using()); } };
+
+ /** Used in logging statements to lazily build a human-readable number of
reserve global bytes in use. */
+ private final Object readableReserveGlobalUsing =
+ new Object() { @Override public String toString() { return
prettyPrintMemory(reserveCapacityInBytes.global.using()); } };
+
private volatile long submittedCount = 0; // updated with cas
private volatile long overloadedCount = 0; // updated with cas
private volatile long overloadedBytes = 0; // updated with cas
@@ -439,13 +452,14 @@ public class OutboundConnection
private void onOverloaded(Message<?> message)
{
overloadedCountUpdater.incrementAndGet(this);
- overloadedBytesUpdater.addAndGet(this, canonicalSize(message));
+
+ int canonicalSize = canonicalSize(message);
+ overloadedBytesUpdater.addAndGet(this, canonicalSize);
+
noSpamLogger.warn("{} overloaded; dropping {} message (queue: {}
local, {} endpoint, {} global)",
- id(),
-
FBUtilities.prettyPrintMemory(canonicalSize(message)),
- FBUtilities.prettyPrintMemory(pendingBytes()),
-
FBUtilities.prettyPrintMemory(reserveCapacityInBytes.endpoint.using()),
-
FBUtilities.prettyPrintMemory(reserveCapacityInBytes.global.using()));
+ this, FBUtilities.prettyPrintMemory(canonicalSize),
+ readablePendingBytes, readableReserveEndpointUsing,
readableReserveGlobalUsing);
+
callbacks.onOverloaded(message, template.to);
}
diff --git a/src/java/org/apache/cassandra/utils/memory/BufferPool.java
b/src/java/org/apache/cassandra/utils/memory/BufferPool.java
index c0686c4..b18f689 100644
--- a/src/java/org/apache/cassandra/utils/memory/BufferPool.java
+++ b/src/java/org/apache/cassandra/utils/memory/BufferPool.java
@@ -71,8 +71,8 @@ public class BufferPool
private final static BufferPoolMetrics metrics = new BufferPoolMetrics();
// TODO: this should not be using FileCacheSizeInMB
- @VisibleForTesting
- public static long MEMORY_USAGE_THRESHOLD =
DatabaseDescriptor.getFileCacheSizeInMB() * 1024L * 1024L;
+ private static long MEMORY_USAGE_THRESHOLD =
DatabaseDescriptor.getFileCacheSizeInMB() * 1024L * 1024L;
+ private static String READABLE_MEMORY_USAGE_THRESHOLD =
prettyPrintMemory(MEMORY_USAGE_THRESHOLD);
private static Debug debug;
@@ -160,6 +160,19 @@ public class BufferPool
return globalPool.sizeInBytes();
}
+ @VisibleForTesting
+ public static void setMemoryUsageThreshold(long threshold)
+ {
+ MEMORY_USAGE_THRESHOLD = threshold;
+ READABLE_MEMORY_USAGE_THRESHOLD =
prettyPrintMemory(MEMORY_USAGE_THRESHOLD);
+ }
+
+ @VisibleForTesting
+ public static long getMemoryUsageThreshold()
+ {
+ return MEMORY_USAGE_THRESHOLD;
+ }
+
interface Debug
{
void registerNormal(Chunk chunk);
@@ -187,6 +200,7 @@ public class BufferPool
{
/** The size of a bigger chunk, 1 MiB, must be a multiple of
NORMAL_CHUNK_SIZE */
static final int MACRO_CHUNK_SIZE = 64 * NORMAL_CHUNK_SIZE;
+ private static final String READABLE_MACRO_CHUNK_SIZE =
prettyPrintMemory(MACRO_CHUNK_SIZE);
static
{
@@ -194,8 +208,7 @@ public class BufferPool
assert Integer.bitCount(MACRO_CHUNK_SIZE) == 1; // must be a power
of 2
assert MACRO_CHUNK_SIZE % NORMAL_CHUNK_SIZE == 0; // must be a
multiple
- logger.info("Global buffer pool limit is {}",
- prettyPrintMemory(MEMORY_USAGE_THRESHOLD));
+ logger.info("Global buffer pool limit is {}",
prettyPrintMemory(MEMORY_USAGE_THRESHOLD));
}
private final Queue<Chunk> macroChunks = new ConcurrentLinkedQueue<>();
@@ -203,6 +216,10 @@ public class BufferPool
private final Queue<Chunk> chunks = new ConcurrentLinkedQueue<>();
private final AtomicLong memoryUsage = new AtomicLong();
+ /** Used in logging statements to lazily build a human-readable
current memory usage. */
+ private final Object readableMemoryUsage =
+ new Object() { @Override public String toString() { return
prettyPrintMemory(sizeInBytes()); } };
+
/** Return a chunk, the caller will take owership of the parent chunk.
*/
public Chunk get()
{
@@ -232,8 +249,7 @@ public class BufferPool
if (MEMORY_USAGE_THRESHOLD > 0)
{
noSpamLogger.info("Maximum memory usage reached ({}),
cannot allocate chunk of {}",
-
prettyPrintMemory(MEMORY_USAGE_THRESHOLD),
- prettyPrintMemory(MACRO_CHUNK_SIZE));
+ READABLE_MEMORY_USAGE_THRESHOLD,
READABLE_MACRO_CHUNK_SIZE);
}
return null;
}
@@ -252,9 +268,7 @@ public class BufferPool
noSpamLogger.error("Buffer pool failed to allocate chunk of
{}, current size {} ({}). " +
"Attempting to continue; buffers will be
allocated in on-heap memory which can degrade performance. " +
"Make sure direct memory size
(-XX:MaxDirectMemorySize) is large enough to accommodate off-heap memtables and
caches.",
- prettyPrintMemory(MACRO_CHUNK_SIZE),
- prettyPrintMemory(sizeInBytes()),
- oom.toString());
+ READABLE_MACRO_CHUNK_SIZE,
readableMemoryUsage, oom.getClass().getName());
return null;
}
diff --git
a/test/burn/org/apache/cassandra/utils/memory/LongBufferPoolTest.java
b/test/burn/org/apache/cassandra/utils/memory/LongBufferPoolTest.java
index c8368dd..0af4199 100644
--- a/test/burn/org/apache/cassandra/utils/memory/LongBufferPoolTest.java
+++ b/test/burn/org/apache/cassandra/utils/memory/LongBufferPoolTest.java
@@ -244,9 +244,9 @@ public class LongBufferPoolTest
DATE_FORMAT.format(new Date()),
threadCount,
TimeUnit.NANOSECONDS.toMinutes(duration)));
- long prevPoolSize = BufferPool.MEMORY_USAGE_THRESHOLD;
+ long prevPoolSize = BufferPool.getMemoryUsageThreshold();
logger.info("Overriding configured
BufferPool.MEMORY_USAGE_THRESHOLD={} and enabling BufferPool.DEBUG", poolSize);
- BufferPool.MEMORY_USAGE_THRESHOLD = poolSize;
+ BufferPool.setMemoryUsageThreshold(poolSize);
Debug debug = new Debug();
BufferPool.debug(debug);
@@ -288,7 +288,7 @@ public class LongBufferPoolTest
assertEquals(0, testEnv.executorService.shutdownNow().size());
logger.info("Reverting BufferPool.MEMORY_USAGE_THRESHOLD={}",
prevPoolSize);
- BufferPool.MEMORY_USAGE_THRESHOLD = prevPoolSize;
+ BufferPool.setMemoryUsageThreshold(prevPoolSize);
BufferPool.debug(null);
testEnv.assertCheckedThreadsSucceeded();
diff --git a/test/unit/org/apache/cassandra/metrics/BufferPoolMetricsTest.java
b/test/unit/org/apache/cassandra/metrics/BufferPoolMetricsTest.java
index fade96c..313aa3f 100644
--- a/test/unit/org/apache/cassandra/metrics/BufferPoolMetricsTest.java
+++ b/test/unit/org/apache/cassandra/metrics/BufferPoolMetricsTest.java
@@ -51,7 +51,7 @@ public class BufferPoolMetricsTest
@Before
public void setUp()
{
- BufferPool.MEMORY_USAGE_THRESHOLD = 16 * 1024L * 1024L;
+ BufferPool.setMemoryUsageThreshold(16 * 1024L * 1024L);
}
@After
@@ -77,7 +77,7 @@ public class BufferPoolMetricsTest
final long seed = System.currentTimeMillis();
final Random rand = new Random(seed);
final String assertionMessage = String.format("Failed with seed of
%s", seed);
- final long maxIterations = BufferPool.MEMORY_USAGE_THRESHOLD;
+ final long maxIterations = BufferPool.getMemoryUsageThreshold();
final int maxBufferSize = BufferPool.NORMAL_CHUNK_SIZE - 1;
int nextSizeToRequest;
long totalBytesRequestedFromPool = 0;
diff --git a/test/unit/org/apache/cassandra/utils/memory/BufferPoolTest.java
b/test/unit/org/apache/cassandra/utils/memory/BufferPoolTest.java
index 5df94ec..4bfc54a 100644
--- a/test/unit/org/apache/cassandra/utils/memory/BufferPoolTest.java
+++ b/test/unit/org/apache/cassandra/utils/memory/BufferPoolTest.java
@@ -1,4 +1,4 @@
-/**
+/*
* 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
@@ -46,7 +46,7 @@ public class BufferPoolTest
@Before
public void setUp()
{
- BufferPool.MEMORY_USAGE_THRESHOLD = 8 * 1024L * 1024L;
+ BufferPool.setMemoryUsageThreshold(8 * 1024L * 1024L);
}
@After
@@ -65,7 +65,7 @@ public class BufferPoolTest
}
@Test
- public void testGetPut() throws InterruptedException
+ public void testGetPut()
{
final int size = RandomAccessReader.DEFAULT_BUFFER_SIZE;
@@ -149,14 +149,14 @@ public class BufferPoolTest
@Test
public void testMaxMemoryExceeded_SameAsChunkSize()
{
- BufferPool.MEMORY_USAGE_THRESHOLD =
BufferPool.GlobalPool.MACRO_CHUNK_SIZE;
+
BufferPool.setMemoryUsageThreshold(BufferPool.GlobalPool.MACRO_CHUNK_SIZE);
requestDoubleMaxMemory();
}
@Test
public void testMaxMemoryExceeded_SmallerThanChunkSize()
{
- BufferPool.MEMORY_USAGE_THRESHOLD =
BufferPool.GlobalPool.MACRO_CHUNK_SIZE / 2;
+
BufferPool.setMemoryUsageThreshold(BufferPool.GlobalPool.MACRO_CHUNK_SIZE / 2);
requestDoubleMaxMemory();
}
@@ -168,7 +168,7 @@ public class BufferPoolTest
private void requestDoubleMaxMemory()
{
- requestUpToSize(RandomAccessReader.DEFAULT_BUFFER_SIZE, (int)(2 *
BufferPool.MEMORY_USAGE_THRESHOLD));
+ requestUpToSize(RandomAccessReader.DEFAULT_BUFFER_SIZE, (int)(2 *
BufferPool.getMemoryUsageThreshold()));
}
private void requestUpToSize(int bufferSize, int totalSize)
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]