/*
*
* 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.cassandra.net;
import java.io.ByteArrayInputStream;
import java.io.ByteArrayOutputStream;
import java.io.IOException;
import java.net.InetAddress;
import java.net.UnknownHostException;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
import java.util.Map;
import java.util.Set;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.TimeUnit;
import com.google.common.collect.Iterables;
import com.codahale.metrics.Timer;
import org.apache.cassandra.auth.IInternodeAuthenticator;
import org.apache.cassandra.config.DatabaseDescriptor;
import org.apache.cassandra.config.DatabaseDescriptor;
import org.apache.cassandra.db.monitoring.ApproximateTime;
import org.apache.cassandra.exceptions.ConfigurationException;
import org.apache.cassandra.io.util.DataInputPlus.DataInputStreamPlus;
import org.apache.cassandra.io.util.DataOutputStreamPlus;
import org.apache.cassandra.io.util.WrappedDataOutputStreamPlus;
import org.caffinitas.ohc.histo.EstimatedHistogram;
import org.junit.After;
import org.junit.Before;
import org.junit.BeforeClass;
import org.junit.Test;
import static org.junit.Assert.*;
public class MessagingServiceTest
{
private final static long ONE_SECOND = TimeUnit.NANOSECONDS.convert(1, TimeUnit.SECONDS);
private final static long[] bucketOffsets = new EstimatedHistogram(160).getBucketOffsets();
public static final IInternodeAuthenticator ALLOW_NOTHING_AUTHENTICATOR = new IInternodeAuthenticator()
{
public boolean authenticate(InetAddress remoteAddress, int remotePort)
{
return false;
}
public void validateConfiguration() throws ConfigurationException
{
}
};
static final IInternodeAuthenticator originalAuthenticator = DatabaseDescriptor.getInternodeAuthenticator();
private final MessagingService messagingService = MessagingService.test();
@BeforeClass
public static void beforeClass() throws UnknownHostException
{
DatabaseDescriptor.daemonInitialization();
DatabaseDescriptor.setBackPressureStrategy(new MockBackPressureStrategy(Collections.emptyMap()));
DatabaseDescriptor.setBroadcastAddress(InetAddress.getByName("127.0.0.1"));
}
private static int metricScopeId = 0;
@Before
public void before() throws UnknownHostException
{
messagingService.resetDroppedMessagesMap(Integer.toString(metricScopeId++));
MockBackPressureStrategy.applied = false;
messagingService.destroyConnectionPool(InetAddress.getByName("127.0.0.2"));
messagingService.destroyConnectionPool(InetAddress.getByName("127.0.0.3"));
}
@Test
public void testDroppedMessages()
{
MessagingService.Verb verb = MessagingService.Verb.READ;
for (int i = 1; i <= 5000; i++)
messagingService.incrementDroppedMessages(verb, i, i % 2 == 0);
List<String> logs = messagingService.getDroppedMessagesLogs();
assertEquals(1, logs.size());
assertEquals("READ messages were dropped in last 5000 ms: 2500 internal and 2500 cross node. Mean internal dropped latency: 2730 ms and Mean cross-node dropped latency: 2731 ms", logs.get(0));
assertEquals(5000, (int) messagingService.getDroppedMessages().get(verb.toString()));
logs = messagingService.getDroppedMessagesLogs();
assertEquals(0, logs.size());
for (int i = 0; i < 2500; i++)
messagingService.incrementDroppedMessages(verb, i, i % 2 == 0);
logs = messagingService.getDroppedMessagesLogs();
assertEquals("READ messages were dropped in last 5000 ms: 1250 internal and 1250 cross node. Mean internal dropped latency: 2277 ms and Mean cross-node dropped latency: 2278 ms", logs.get(0));
assertEquals(7500, (int) messagingService.getDroppedMessages().get(verb.toString()));
}
@Test
public void testDCLatency() throws Exception
{
int latency = 100;
ConcurrentHashMap<String, Timer> dcLatency = MessagingService.instance().metrics.dcLatency;
dcLatency.clear();
long now = ApproximateTime.currentTimeMillis();
long sentAt = now - latency;
assertNull(dcLatency.get("datacenter1"));
addDCLatency(sentAt, now);
assertNotNull(dcLatency.get("datacenter1"));
assertEquals(1, dcLatency.get("datacenter1").getCount());
long expectedBucket = bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, TimeUnit.MILLISECONDS.toNanos(latency))) - 1];
assertEquals(expectedBucket, dcLatency.get("datacenter1").getSnapshot().getMax());
}
@Test
public void testNegativeDCLatency() throws Exception
{
// if clocks are off should just not track anything
int latency = -100;
ConcurrentHashMap<String, Timer> dcLatency = MessagingService.instance().metrics.dcLatency;
dcLatency.clear();
long now = ApproximateTime.currentTimeMillis();
long sentAt = now - latency;
assertNull(dcLatency.get("datacenter1"));
addDCLatency(sentAt, now);
assertNull(dcLatency.get("datacenter1"));
}
@Test
public void testQueueWaitLatency() throws Exception
{
int latency = 100;
String verb = MessagingService.Verb.MUTATION.toString();
ConcurrentHashMap<String, Timer> queueWaitLatency = MessagingService.instance().metrics.queueWaitLatency;
queueWaitLatency.clear();
assertNull(queueWaitLatency.get(verb));
MessagingService.instance().metrics.addQueueWaitTime(verb, latency);
assertNotNull(queueWaitLatency.get(verb));
assertEquals(1, queueWaitLatency.get(verb).getCount());
long expectedBucket = bucketOffsets[Math.abs(Arrays.binarySearch(bucketOffsets, TimeUnit.MILLISECONDS.toNanos(latency))) - 1];
assertEquals(expectedBucket, queueWaitLatency.get(verb).getSnapshot().getMax());
}
@Test
public void testNegativeQueueWaitLatency() throws Exception
{
int latency = -100;
String verb = MessagingService.Verb.MUTATION.toString();
ConcurrentHashMap<String, Timer> queueWaitLatency = MessagingService.instance().metrics.queueWaitLatency;
queueWaitLatency.clear();
assertNull(queueWaitLatency.get(verb));
MessagingService.instance().metrics.addQueueWaitTime(verb, latency);
assertNull(queueWaitLatency.get(verb));
}
@Test
public void testUpdatesBackPressureOnSendWhenEnabledAndWithSupportedCallback() throws UnknownHostException
{
MockBackPressureStrategy.MockBackPressureState backPressureState = (MockBackPressureStrategy.MockBackPressureState) messagingService.getConnectionPool(InetAddress.getByName("127.0.0.2")).getBackPressureState();
IAsyncCallback bpCallback = new BackPressureCallback();
IAsyncCallback noCallback = new NoBackPressureCallback();
MessageOut<?> ignored = null;
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnSend(InetAddress.getByName("127.0.0.2"), noCallback, ignored);
assertFalse(backPressureState.onSend);
DatabaseDescriptor.setBackPressureEnabled(false);
messagingService.updateBackPressureOnSend(InetAddress.getByName("127.0.0.2"), bpCallback, ignored);
assertFalse(backPressureState.onSend);
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnSend(InetAddress.getByName("127.0.0.2"), bpCallback, ignored);
assertTrue(backPressureState.onSend);
}
@Test
public void testUpdatesBackPressureOnReceiveWhenEnabledAndWithSupportedCallback() throws UnknownHostException
{
MockBackPressureStrategy.MockBackPressureState backPressureState = (MockBackPressureStrategy.MockBackPressureState) messagingService.getConnectionPool(InetAddress.getByName("127.0.0.2")).getBackPressureState();
IAsyncCallback bpCallback = new BackPressureCallback();
IAsyncCallback noCallback = new NoBackPressureCallback();
boolean timeout = false;
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), noCallback, timeout);
assertFalse(backPressureState.onReceive);
assertFalse(backPressureState.onTimeout);
DatabaseDescriptor.setBackPressureEnabled(false);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), bpCallback, timeout);
assertFalse(backPressureState.onReceive);
assertFalse(backPressureState.onTimeout);
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), bpCallback, timeout);
assertTrue(backPressureState.onReceive);
assertFalse(backPressureState.onTimeout);
}
@Test
public void testUpdatesBackPressureOnTimeoutWhenEnabledAndWithSupportedCallback() throws UnknownHostException
{
MockBackPressureStrategy.MockBackPressureState backPressureState = (MockBackPressureStrategy.MockBackPressureState) messagingService.getConnectionPool(InetAddress.getByName("127.0.0.2")).getBackPressureState();
IAsyncCallback bpCallback = new BackPressureCallback();
IAsyncCallback noCallback = new NoBackPressureCallback();
boolean timeout = true;
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), noCallback, timeout);
assertFalse(backPressureState.onReceive);
assertFalse(backPressureState.onTimeout);
DatabaseDescriptor.setBackPressureEnabled(false);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), bpCallback, timeout);
assertFalse(backPressureState.onReceive);
assertFalse(backPressureState.onTimeout);
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.updateBackPressureOnReceive(InetAddress.getByName("127.0.0.2"), bpCallback, timeout);
assertFalse(backPressureState.onReceive);
assertTrue(backPressureState.onTimeout);
}
@Test
public void testAppliesBackPressureWhenEnabled() throws UnknownHostException
{
DatabaseDescriptor.setBackPressureEnabled(false);
messagingService.applyBackPressure(Arrays.asList(InetAddress.getByName("127.0.0.2")), ONE_SECOND);
assertFalse(MockBackPressureStrategy.applied);
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.applyBackPressure(Arrays.asList(InetAddress.getByName("127.0.0.2")), ONE_SECOND);
assertTrue(MockBackPressureStrategy.applied);
}
@Test
public void testDoesntApplyBackPressureToBroadcastAddress() throws UnknownHostException
{
DatabaseDescriptor.setBackPressureEnabled(true);
messagingService.applyBackPressure(Arrays.asList(InetAddress.getByName("127.0.0.1")), ONE_SECOND);
assertFalse(MockBackPressureStrategy.applied);
}
private static void addDCLatency(long sentAt, long nowTime) throws IOException
{
ByteArrayOutputStream baos = new ByteArrayOutputStream();
try (DataOutputStreamPlus out = new WrappedDataOutputStreamPlus(baos))
{
out.writeInt((int) sentAt);
}
DataInputStreamPlus in = new DataInputStreamPlus(new ByteArrayInputStream(baos.toByteArray()));
MessageIn.readConstructionTime(InetAddress.getLocalHost(), in, nowTime);
}
public static class MockBackPressureStrategy implements BackPressureStrategy<MockBackPressureStrategy.MockBackPressureState>
{
public static volatile boolean applied = false;
public MockBackPressureStrategy(Map<String, Object> args)
{
}
@Override
public void apply(Set<MockBackPressureState> states, long timeout, TimeUnit unit)
{
if (!Iterables.isEmpty(states))
applied = true;
}
@Override
public MockBackPressureState newState(InetAddress host)
{
return new MockBackPressureState(host);
}
public static class MockBackPressureState implements BackPressureState
{
private final InetAddress host;
public volatile boolean onSend = false;
public volatile boolean onReceive = false;
public volatile boolean onTimeout = false;
private MockBackPressureState(InetAddress host)
{
this.host = host;
}
@Override
public void onMessageSent(MessageOut<?> message)
{
onSend = true;
}
@Override
public void onResponseReceived()
{
onReceive = true;
}
@Override
public void onResponseTimeout()
{
onTimeout = true;
}
@Override
public double getBackPressureRateLimit()
{
throw new UnsupportedOperationException("Not supported yet.");
}
@Override
public InetAddress getHost()
{
return host;
}
}
}
private static class BackPressureCallback implements IAsyncCallback
{
@Override
public boolean supportsBackPressure()
{
return true;
}
@Override
public boolean isLatencyForSnitch()
{
return false;
}
@Override
public void response(MessageIn msg)
{
throw new UnsupportedOperationException("Not supported.");
}
}
private static class NoBackPressureCallback implements IAsyncCallback
{
@Override
public boolean supportsBackPressure()
{
return false;
}
@Override
public boolean isLatencyForSnitch()
{
return false;
}
@Override
public void response(MessageIn msg)
{
throw new UnsupportedOperationException("Not supported.");
}
}
/**
* Make sure that if internode authenticatino fails for an outbound connection that all the code that relies
* on getting the connection pool handles the null return
* @throws Exception
*/
@Test
public void testFailedInternodeAuth() throws Exception
{
MessagingService ms = MessagingService.instance();
DatabaseDescriptor.setInternodeAuthenticator(ALLOW_NOTHING_AUTHENTICATOR);
InetAddress address = InetAddress.getByName("127.0.0.250");
//Should return null
assertNull(ms.getConnectionPool(address));
assertNull(ms.getConnection(address, new MessageOut(MessagingService.Verb.GOSSIP_DIGEST_ACK)));
//Should tolerate null
ms.convict(address);
ms.sendOneWay(new MessageOut(MessagingService.Verb.GOSSIP_DIGEST_ACK), address);
}
@Test
public void testOutboundTcpConnectionCleansUp() throws Exception
{
MessagingService ms = MessagingService.instance();
DatabaseDescriptor.setInternodeAuthenticator(ALLOW_NOTHING_AUTHENTICATOR);
InetAddress address = InetAddress.getByName("127.0.0.250");
OutboundTcpConnectionPool pool = new OutboundTcpConnectionPool(address, new MockBackPressureStrategy(null).newState(address));
ms.connectionManagers.put(address, pool);
pool.smallMessages.start();
pool.smallMessages.enqueue(new MessageOut(MessagingService.Verb.GOSSIP_DIGEST_ACK), 0);
pool.smallMessages.join();
assertFalse(ms.connectionManagers.containsKey(address));
}
@After
public void replaceAuthenticator()
{
DatabaseDescriptor.setInternodeAuthenticator(originalAuthenticator);
}
}