Skip to content

Commit

Permalink
Somewhat better defaults for the binary protocol
Browse files Browse the repository at this point in the history
  • Loading branch information
Sylvain Lebresne committed Nov 29, 2012
1 parent 452f596 commit a0db8ff
Show file tree
Hide file tree
Showing 6 changed files with 24 additions and 10 deletions.
9 changes: 6 additions & 3 deletions conf/cassandra.yaml
Expand Up @@ -310,9 +310,12 @@ listen_address: localhost
start_native_transport: false
# port for the CQL native transport to listen for clients on
native_transport_port: 9042
# The maximum of thread handling requests. The meaning is the same than
# rpc_max_threads. The default is unlimited.
#native_transport_max_threads: 2048
# The minimum and maximum threads for handling requests when the native
# transport is used. The meaning is those is similar to the one of
# rpc_min_threads and rpc_max_threads, though the default differ slightly and
# are the ones below:
# native_transport_min_threads: 16
# native_transport_max_threads: 128


# Whether to start the thrift rpc server.
Expand Down
3 changes: 2 additions & 1 deletion src/java/org/apache/cassandra/config/Config.java
Expand Up @@ -86,7 +86,8 @@ public class Config

public Boolean start_native_transport = false;
public Integer native_transport_port = 8000;
public Integer native_transport_max_threads = Integer.MAX_VALUE;
public Integer native_transport_min_threads = 16;
public Integer native_transport_max_threads = 128;

public Integer thrift_max_message_length_in_mb = 16;
public Integer thrift_framed_transport_size_in_mb = 15;
Expand Down
5 changes: 5 additions & 0 deletions src/java/org/apache/cassandra/config/DatabaseDescriptor.java
Expand Up @@ -986,6 +986,11 @@ public static int getNativeTransportPort()
return Integer.parseInt(System.getProperty("cassandra.native_transport_port", conf.native_transport_port.toString()));
}

public static Integer getNativeTransportMinThreads()
{
return conf.native_transport_min_threads;
}

public static Integer getNativeTransportMaxThreads()
{
return conf.native_transport_max_threads;
Expand Down
Expand Up @@ -17,22 +17,23 @@
*/
package org.apache.cassandra.transport;

import java.util.concurrent.ArrayBlockingQueue;
import java.util.concurrent.TimeUnit;

import org.jboss.netty.handler.execution.OrderedMemoryAwareThreadPoolExecutor;

import org.apache.cassandra.config.DatabaseDescriptor;
import org.apache.cassandra.concurrent.DebuggableThreadPoolExecutor;
import org.apache.cassandra.concurrent.NamedThreadFactory;
import org.apache.cassandra.config.DatabaseDescriptor;

public class RequestThreadPoolExecutor extends OrderedMemoryAwareThreadPoolExecutor
public class RequestThreadPoolExecutor extends DebuggableThreadPoolExecutor
{
private final static int CORE_THREAD_TIMEOUT_SEC = 30;

public RequestThreadPoolExecutor()
{
super(DatabaseDescriptor.getNativeTransportMaxThreads(),
0, 0,
super(DatabaseDescriptor.getNativeTransportMinThreads(),
DatabaseDescriptor.getNativeTransportMaxThreads(),
CORE_THREAD_TIMEOUT_SEC, TimeUnit.SECONDS,
new ArrayBlockingQueue(32), // Seems to help smooth latency compared to SynchronousQueue.
new NamedThreadFactory("Native-Transport-Requests"));
}
}
2 changes: 2 additions & 0 deletions src/java/org/apache/cassandra/transport/Server.java
Expand Up @@ -106,6 +106,8 @@ public void run()
factory = new NioServerSocketChannelFactory(Executors.newCachedThreadPool(), Executors.newCachedThreadPool());
ServerBootstrap bootstrap = new ServerBootstrap(factory);

bootstrap.setOption("child.tcpNoDelay", true);

// Set up the event pipeline factory.
bootstrap.setPipelineFactory(new PipelineFactory(this));

Expand Down
2 changes: 2 additions & 0 deletions src/java/org/apache/cassandra/transport/SimpleClient.java
Expand Up @@ -92,6 +92,8 @@ protected void establishConnection() throws IOException
Executors.newCachedThreadPool(),
Executors.newCachedThreadPool()));

bootstrap.setOption("tcpNoDelay", true);

// Configure the pipeline factory.
bootstrap.setPipelineFactory(new PipelineFactory());
ChannelFuture future = bootstrap.connect(new InetSocketAddress(host, port));
Expand Down

0 comments on commit a0db8ff

Please sign in to comment.