Author: jbellis
Date: Mon Sep 6 20:43:43 2010
New Revision: 993148
URL: http://svn.apache.org/viewvc?rev=993148&view=rev
Log:
eager-create tcp connections to other cluster members. patch by jbellis;
reviewed by Nate McCall for CASSANDRA-1465
Modified:
cassandra/branches/cassandra-0.6/CHANGES.txt
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnection.java
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnectionPool.java
Modified: cassandra/branches/cassandra-0.6/CHANGES.txt
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.6/CHANGES.txt?rev=993148&r1=993147&r2=993148&view=diff
==============================================================================
--- cassandra/branches/cassandra-0.6/CHANGES.txt (original)
+++ cassandra/branches/cassandra-0.6/CHANGES.txt Mon Sep 6 20:43:43 2010
@@ -9,6 +9,7 @@
* document DoConsistencyChecksBoolean option to disable Read Repair
* remove failed bootstrap attempt from pending ranges when gossip times
it out after 1h (CASSANDRA-1463)
+ * eager-create tcp connections to other cluster members (CASSANDRA-1465)
0.6.5
Modified:
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnection.java
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnection.java?rev=993148&r1=993147&r2=993148&view=diff
==============================================================================
---
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnection.java
(original)
+++
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnection.java
Mon Sep 6 20:43:43 2010
@@ -41,16 +41,14 @@ public class OutboundTcpConnection exten
private static final ByteBuffer CLOSE_SENTINEL = ByteBuffer.allocate(0);
private static final int OPEN_RETRY_DELAY = 100; // ms between retries
- private final OutboundTcpConnectionPool pool;
private final InetAddress endpoint;
private final BlockingQueue<ByteBuffer> queue = new
LinkedBlockingQueue<ByteBuffer>();
private DataOutputStream output;
private Socket socket;
- public OutboundTcpConnection(final OutboundTcpConnectionPool pool, final
InetAddress remoteEp)
+ public OutboundTcpConnection(InetAddress remoteEp)
{
super("WRITE-" + remoteEp);
- this.pool = pool;
this.endpoint = remoteEp;
}
Modified:
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnectionPool.java
URL:
http://svn.apache.org/viewvc/cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnectionPool.java?rev=993148&r1=993147&r2=993148&view=diff
==============================================================================
---
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnectionPool.java
(original)
+++
cassandra/branches/cassandra-0.6/src/java/org/apache/cassandra/net/OutboundTcpConnectionPool.java
Mon Sep 6 20:43:43 2010
@@ -18,55 +18,35 @@
package org.apache.cassandra.net;
-import java.io.IOException;
import java.net.InetAddress;
-import org.apache.log4j.Logger;
-
import org.apache.cassandra.concurrent.StageManager;
class OutboundTcpConnectionPool
{
- private InetAddress remoteEp_;
- private OutboundTcpConnection cmdCon;
- private OutboundTcpConnection ackCon;
+ private final OutboundTcpConnection cmdCon;
+ private final OutboundTcpConnection ackCon;
OutboundTcpConnectionPool(InetAddress remoteEp)
{
- remoteEp_ = remoteEp;
+ cmdCon = new OutboundTcpConnection(remoteEp);
+ ackCon = new OutboundTcpConnection(remoteEp);
}
/**
* returns the appropriate connection based on message type.
* returns null if a connection could not be established.
*/
- synchronized OutboundTcpConnection getConnection(Message msg)
+ OutboundTcpConnection getConnection(Message msg)
{
- if (StageManager.RESPONSE_STAGE.equals(msg.getMessageType())
- || StageManager.GOSSIP_STAGE.equals(msg.getMessageType()))
- {
- if (ackCon == null)
- {
- ackCon = new OutboundTcpConnection(this, remoteEp_);
- ackCon.start();
- }
- return ackCon;
- }
- else
- {
- if (cmdCon == null)
- {
- cmdCon = new OutboundTcpConnection(this, remoteEp_);
- cmdCon.start();
- }
- return cmdCon;
- }
+ return msg.getMessageType().equals(StageManager.RESPONSE_STAGE) ||
msg.getMessageType().equals(StageManager.GOSSIP_STAGE)
+ ? ackCon
+ : cmdCon;
}
synchronized void reset()
{
for (OutboundTcpConnection con : new OutboundTcpConnection[] { cmdCon,
ackCon })
- if (con != null)
- con.closeSocket();
+ con.closeSocket();
}
}