2009-06-24 15:12:47 +02:00
/* *
2010-12-22 15:35:50 +01:00
* Copyright ( C ) 2009 - 2011 Scalable Solutions AB < http : //scalablesolutions.se>
2009-06-24 15:12:47 +02:00
*/
2011-01-03 12:42:30 +01:00
package akka.remote.netty
2009-06-24 15:12:47 +02:00
2011-05-23 11:31:01 +02:00
import akka.dispatch. { DefaultPromise , Promise , Future }
2011-05-18 17:25:30 +02:00
import akka.remote. { MessageSerializer , RemoteClientSettings , RemoteServerSettings }
2010-10-26 12:49:25 +02:00
import akka.remote.protocol.RemoteProtocol._
import akka.serialization.RemoteActorSerialization
2011-01-03 12:42:30 +01:00
import akka.serialization.RemoteActorSerialization._
import akka.remoteinterface._
2011-05-18 17:25:30 +02:00
import akka.actor. {
PoisonPill ,
Index ,
LocalActorRef ,
Actor ,
RemoteActorRef ,
ActorRef ,
IllegalActorStateException ,
RemoteActorSystemMessage ,
uuidFrom ,
Uuid ,
Exit ,
2011-05-19 14:29:21 +02:00
LifeCycleMessage
2011-05-18 17:25:30 +02:00
}
2010-12-15 17:52:31 +01:00
import akka.actor.Actor._
2011-05-24 19:04:25 +02:00
import akka.config.Config
import Config._
2010-12-15 17:52:31 +01:00
import akka.util._
2011-03-24 12:28:01 +01:00
import akka.event.EventHandler
2009-07-18 00:16:32 +02:00
2009-06-24 15:12:47 +02:00
import org.jboss.netty.channel._
2011-05-18 17:25:30 +02:00
import org.jboss.netty.channel.group. { DefaultChannelGroup , ChannelGroup , ChannelGroupFuture }
2010-12-15 17:52:31 +01:00
import org.jboss.netty.channel.socket.nio.NioClientSocketChannelFactory
2009-06-24 15:12:47 +02:00
import org.jboss.netty.channel.socket.nio.NioServerSocketChannelFactory
2011-05-18 17:25:30 +02:00
import org.jboss.netty.bootstrap. { ServerBootstrap , ClientBootstrap }
2010-12-15 17:52:31 +01:00
import org.jboss.netty.handler.codec.frame. { LengthFieldBasedFrameDecoder , LengthFieldPrepender }
import org.jboss.netty.handler.codec.compression. { ZlibDecoder , ZlibEncoder }
import org.jboss.netty.handler.codec.protobuf. { ProtobufDecoder , ProtobufEncoder }
2011-03-14 11:21:41 +01:00
import org.jboss.netty.handler.timeout. { ReadTimeoutHandler , ReadTimeoutException }
2011-03-23 11:31:19 +01:00
import org.jboss.netty.handler.execution. { OrderedMemoryAwareThreadPoolExecutor , ExecutionHandler }
2010-12-15 17:52:31 +01:00
import org.jboss.netty.util. { TimerTask , Timeout , HashedWheelTimer }
2010-04-25 20:32:52 +02:00
2011-03-24 12:28:01 +01:00
import scala.collection.mutable.HashMap
2011-03-25 08:36:43 +01:00
import scala.collection.JavaConversions._
2011-03-18 23:04:48 +01:00
2011-03-24 12:28:01 +01:00
import java.net.InetSocketAddress
2010-12-15 17:52:31 +01:00
import java.lang.reflect.InvocationTargetException
2011-05-18 17:25:30 +02:00
import java.util.concurrent.atomic. { AtomicReference , AtomicBoolean }
2011-03-24 16:23:37 +01:00
import java.util.concurrent._
2011-03-29 17:07:41 +02:00
import akka.AkkaException
2011-04-29 17:15:00 +02:00
class RemoteClientMessageBufferException ( message : String , cause : Throwable = null ) extends AkkaException ( message , cause )
2011-01-03 12:42:30 +01:00
2011-03-05 14:55:58 +01:00
object RemoteEncoder {
def encode ( rmp : RemoteMessageProtocol ) : AkkaRemoteProtocol = {
val arp = AkkaRemoteProtocol . newBuilder
arp . setMessage ( rmp )
arp . build
}
def encode ( rcp : RemoteControlProtocol ) : AkkaRemoteProtocol = {
val arp = AkkaRemoteProtocol . newBuilder
arp . setInstruction ( rcp )
arp . build
}
}
2011-05-18 17:25:30 +02:00
trait NettyRemoteClientModule extends RemoteClientModule { self : ListenerManagement ⇒
2011-01-03 11:25:27 +01:00
private val remoteClients = new HashMap [ Address , RemoteClient ]
2011-05-18 17:25:30 +02:00
private val remoteActors = new Index [ Address , Uuid ]
private val lock = new ReadWriteGuard
2010-12-15 17:52:31 +01:00
protected [ akka ] def send [ T ] ( message : Any ,
senderOption : Option [ ActorRef ] ,
2011-05-23 11:31:01 +02:00
senderFuture : Option [ Promise [ T ] ] ,
2010-12-15 17:52:31 +01:00
remoteAddress : InetSocketAddress ,
timeout : Long ,
isOneWay : Boolean ,
actorRef : ActorRef ,
2011-05-23 11:31:01 +02:00
loader : Option [ ClassLoader ] ) : Option [ Promise [ T ] ] =
2011-05-19 14:29:21 +02:00
withClientFor ( remoteAddress , loader ) ( _ . send [ T ] ( message , senderOption , senderFuture , remoteAddress , timeout , isOneWay , actorRef ) )
2010-12-15 17:52:31 +01:00
2011-01-03 12:42:30 +01:00
private [ akka ] def withClientFor [ T ] (
2011-05-18 17:25:30 +02:00
address : InetSocketAddress , loader : Option [ ClassLoader ] ) ( fun : RemoteClient ⇒ T ) : T = {
2011-03-27 00:01:18 +01:00
loader . foreach ( MessageSerializer . setClassLoader ( _ ) )
2011-01-03 12:42:30 +01:00
val key = Address ( address )
2011-01-01 21:01:44 +01:00
lock . readLock . lock
2011-01-01 23:18:51 +01:00
try {
2011-01-03 12:42:30 +01:00
val c = remoteClients . get ( key ) match {
2011-05-18 17:25:30 +02:00
case s : Some [ RemoteClient ] ⇒ s . get
case None ⇒
2011-01-01 23:18:51 +01:00
lock . readLock . unlock
lock . writeLock . lock //Lock upgrade, not supported natively
try {
2011-01-03 11:25:27 +01:00
try {
remoteClients . get ( key ) match { //Recheck for addition, race between upgrades
2011-05-18 17:25:30 +02:00
case s : Some [ RemoteClient ] ⇒ s . get //If already populated by other writer
case None ⇒ //Populate map
2011-01-03 11:25:27 +01:00
val client = new ActiveRemoteClient ( this , address , loader , self . notifyListeners _ )
client . connect ( )
remoteClients += key -> client
client
}
} finally { lock . readLock . lock } //downgrade
} finally { lock . writeLock . unlock }
2011-01-01 23:18:51 +01:00
}
2011-01-03 12:42:30 +01:00
fun ( c )
2011-01-01 23:18:51 +01:00
} finally { lock . readLock . unlock }
2010-12-15 17:52:31 +01:00
}
2011-01-01 22:20:43 +01:00
def shutdownClientConnection ( address : InetSocketAddress ) : Boolean = lock withWriteGuard {
2011-01-03 12:42:30 +01:00
remoteClients . remove ( Address ( address ) ) match {
2011-05-18 17:25:30 +02:00
case s : Some [ RemoteClient ] ⇒ s . get . shutdown ( )
case None ⇒ false
2010-12-29 17:59:38 +01:00
}
}
2011-01-01 22:20:43 +01:00
def restartClientConnection ( address : InetSocketAddress ) : Boolean = lock withReadGuard {
2011-01-03 12:42:30 +01:00
remoteClients . get ( Address ( address ) ) match {
2011-05-18 17:25:30 +02:00
case s : Some [ RemoteClient ] ⇒ s . get . connect ( reconnectIfAlreadyConnected = true )
case None ⇒ false
2010-12-15 17:52:31 +01:00
}
}
2010-12-20 16:58:05 +01:00
2010-12-15 17:52:31 +01:00
/* *
* Clean - up all open connections .
*/
2011-04-29 10:20:16 +02:00
def shutdownClientModule ( ) {
shutdownRemoteClients ( )
2011-01-01 21:01:44 +01:00
//TODO: Should we empty our remoteActors too?
//remoteActors.clear
2010-12-15 17:52:31 +01:00
}
2011-04-29 10:20:16 +02:00
def shutdownRemoteClients ( ) = lock withWriteGuard {
2011-05-18 17:25:30 +02:00
remoteClients . foreach ( { case ( addr , client ) ⇒ client . shutdown ( ) } )
2011-04-29 10:20:16 +02:00
remoteClients . clear ( )
2010-12-15 17:52:31 +01:00
}
}
2011-01-03 12:42:30 +01:00
/* *
2011-03-27 22:58:50 +02:00
* This is the abstract baseclass for netty remote clients , currently there 's only an
2011-04-23 08:11:31 +02:00
* ActiveRemoteClient , but others could be feasible , like a PassiveRemoteClient that
2011-01-03 12:42:30 +01:00
* reuses an already established connection .
*/
2011-01-01 21:01:44 +01:00
abstract class RemoteClient private [ akka ] (
2010-12-29 16:08:43 +01:00
val module : NettyRemoteClientModule ,
2011-02-28 22:54:32 +01:00
val remoteAddress : InetSocketAddress ) {
2010-12-15 17:52:31 +01:00
2011-06-10 20:49:00 +01:00
val useTransactionLog = config . getBool ( "akka.cluster.client.buffering.retry-message-send-on-failure" , true )
val transactionLogCapacity = config . getInt ( "akka.cluster.client.buffering.capacity" , - 1 )
2011-03-27 22:58:50 +02:00
2011-03-24 16:23:37 +01:00
val name = this . getClass . getSimpleName + "@" +
2011-05-18 17:25:30 +02:00
remoteAddress . getAddress . getHostAddress + "::" +
remoteAddress . getPort
2010-12-15 17:52:31 +01:00
2011-05-23 11:31:01 +02:00
protected val futures = new ConcurrentHashMap [ Uuid , Promise [ _ ] ]
2011-03-29 17:07:41 +02:00
protected val pendingRequests = {
if ( transactionLogCapacity < 0 ) new ConcurrentLinkedQueue [ ( Boolean , Uuid , RemoteMessageProtocol ) ]
2011-05-18 17:25:30 +02:00
else new LinkedBlockingQueue [ ( Boolean , Uuid , RemoteMessageProtocol ) ] ( transactionLogCapacity )
2011-03-29 17:07:41 +02:00
}
2011-03-24 16:23:37 +01:00
2011-05-18 17:25:30 +02:00
private [ remote ] val runSwitch = new Switch ( )
2010-12-15 17:52:31 +01:00
private [ remote ] def isRunning = runSwitch . isOn
2011-05-18 17:25:30 +02:00
protected def notifyListeners ( msg : ⇒ Any ) : Unit
2011-03-24 16:23:37 +01:00
2011-01-01 21:01:44 +01:00
protected def currentChannel : Channel
2010-12-15 17:52:31 +01:00
2011-01-01 21:01:44 +01:00
def connect ( reconnectIfAlreadyConnected : Boolean = false ) : Boolean
2011-03-24 16:23:37 +01:00
2011-04-29 10:20:16 +02:00
def shutdown ( ) : Boolean
2010-12-15 17:52:31 +01:00
2011-03-25 08:36:43 +01:00
/* *
* Returns an array with the current pending messages not yet delivered .
*/
2011-03-25 16:06:55 +01:00
def pendingMessages : Array [ Any ] = {
var messages = Vector [ Any ] ( )
val iter = pendingRequests . iterator
while ( iter . hasNext ) {
val ( _ , _ , message ) = iter . next
messages = messages : + MessageSerializer . deserialize ( message . getMessage )
}
messages . toArray
}
2011-03-25 08:36:43 +01:00
2011-01-03 12:42:30 +01:00
/* *
* Converts the message to the wireprotocol and sends the message across the wire
*/
2010-12-15 17:52:31 +01:00
def send [ T ] (
message : Any ,
senderOption : Option [ ActorRef ] ,
2011-05-23 11:31:01 +02:00
senderFuture : Option [ Promise [ T ] ] ,
2010-12-15 17:52:31 +01:00
remoteAddress : InetSocketAddress ,
timeout : Long ,
isOneWay : Boolean ,
2011-05-23 11:31:01 +02:00
actorRef : ActorRef ) : Option [ Promise [ T ] ] =
2010-12-15 17:52:31 +01:00
send ( createRemoteMessageProtocolBuilder (
2011-05-20 19:40:11 +02:00
Some ( actorRef ) , Left ( actorRef . uuid ) , actorRef . address , timeout , Right ( message ) , isOneWay , senderOption ) . build ,
senderFuture )
2010-12-15 17:52:31 +01:00
2011-01-03 12:42:30 +01:00
/* *
* Sends the message across the wire
*/
2010-12-15 17:52:31 +01:00
def send [ T ] (
2011-03-24 12:28:01 +01:00
request : RemoteMessageProtocol ,
2011-05-23 11:31:01 +02:00
senderFuture : Option [ Promise [ T ] ] ) : Option [ Promise [ T ] ] = {
2011-05-24 19:04:25 +02:00
2010-12-15 17:52:31 +01:00
if ( isRunning ) {
2011-05-25 16:18:35 +02:00
EventHandler . debug ( this , "Sending to connection [%s] message [%s]" . format ( remoteAddress , request ) )
2011-05-24 19:04:25 +02:00
2010-12-15 17:52:31 +01:00
if ( request . getOneWay ) {
2011-03-25 16:06:55 +01:00
try {
val future = currentChannel . write ( RemoteEncoder . encode ( request ) )
future . awaitUninterruptibly ( )
if ( ! future . isCancelled && ! future . isSuccess ) {
notifyListeners ( RemoteClientWriteFailed ( request , future . getCause , module , remoteAddress ) )
throw future . getCause
}
} catch {
2011-05-18 17:25:30 +02:00
case e : Throwable ⇒
2011-03-27 22:58:50 +02:00
// add the request to the tx log after a failing send
2011-03-25 16:06:55 +01:00
notifyListeners ( RemoteClientError ( e , module , remoteAddress ) )
2011-03-29 17:07:41 +02:00
if ( useTransactionLog ) {
if ( ! pendingRequests . offer ( ( true , null , request ) ) )
throw new RemoteClientMessageBufferException ( "Buffer limit [" + transactionLogCapacity + "] reached" )
2011-05-18 17:25:30 +02:00
} else throw e
2011-03-23 15:00:29 +01:00
}
2010-12-15 17:52:31 +01:00
None
2011-05-24 19:04:25 +02:00
2010-12-15 17:52:31 +01:00
} else {
2011-03-24 16:23:37 +01:00
val futureResult = if ( senderFuture . isDefined ) senderFuture . get
2011-05-23 11:31:01 +02:00
else new DefaultPromise [ T ] ( request . getActorInfo . getTimeout )
2011-03-24 16:23:37 +01:00
val futureUuid = uuidFrom ( request . getUuid . getHigh , request . getUuid . getLow )
futures . put ( futureUuid , futureResult ) // Add future prematurely, remove it if write fails
2011-03-25 16:06:55 +01:00
def handleRequestReplyError ( future : ChannelFuture ) = {
notifyListeners ( RemoteClientWriteFailed ( request , future . getCause , module , remoteAddress ) )
2011-03-27 22:58:50 +02:00
if ( useTransactionLog ) {
2011-03-29 17:07:41 +02:00
if ( ! pendingRequests . offer ( ( false , futureUuid , request ) ) ) // Add the request to the tx log after a failing send
throw new RemoteClientMessageBufferException ( "Buffer limit [" + transactionLogCapacity + "] reached" )
2011-03-27 22:58:50 +02:00
} else {
2011-05-18 17:25:30 +02:00
val f = futures . remove ( futureUuid ) // Clean up future
2011-03-27 22:58:50 +02:00
if ( f ne null ) f . completeWithException ( future . getCause )
}
2011-03-25 16:06:55 +01:00
}
var future : ChannelFuture = null
try {
// try to send the original one
future = currentChannel . write ( RemoteEncoder . encode ( request ) )
future . awaitUninterruptibly ( )
2011-05-18 17:25:30 +02:00
if ( future . isCancelled ) futures . remove ( futureUuid ) // Clean up future
2011-03-25 16:06:55 +01:00
else if ( ! future . isSuccess ) handleRequestReplyError ( future )
} catch {
2011-05-18 17:25:30 +02:00
case e : Exception ⇒ handleRequestReplyError ( future )
2011-03-25 16:06:55 +01:00
}
2011-03-24 16:23:37 +01:00
Some ( futureResult )
2010-12-15 17:52:31 +01:00
}
2011-05-24 19:04:25 +02:00
2010-12-15 17:52:31 +01:00
} else {
2011-04-08 15:29:14 +02:00
val exception = new RemoteClientException ( "RemoteModule client is not running, make sure you have invoked 'RemoteClient.connect' before using it." , module , remoteAddress )
2010-12-29 16:08:43 +01:00
notifyListeners ( RemoteClientError ( exception , module , remoteAddress ) )
2010-12-15 17:52:31 +01:00
throw exception
}
}
2011-03-25 16:06:55 +01:00
private [ remote ] def sendPendingRequests ( ) = pendingRequests synchronized { // ensure only one thread at a time can flush the log
val nrOfMessages = pendingRequests . size
if ( nrOfMessages > 0 ) EventHandler . info ( this , "Resending [%s] previously failed messages after remote client reconnect" format nrOfMessages )
var pendingRequest = pendingRequests . peek
while ( pendingRequest ne null ) {
val ( isOneWay , futureUuid , message ) = pendingRequest
2011-03-24 16:23:37 +01:00
if ( isOneWay ) { // sendOneWay
val future = currentChannel . write ( RemoteEncoder . encode ( message ) )
future . awaitUninterruptibly ( )
if ( ! future . isCancelled && ! future . isSuccess ) {
notifyListeners ( RemoteClientWriteFailed ( message , future . getCause , module , remoteAddress ) )
throw future . getCause
}
} else { // sendRequestReply
val future = currentChannel . write ( RemoteEncoder . encode ( message ) )
future . awaitUninterruptibly ( )
2011-03-25 16:06:55 +01:00
if ( future . isCancelled ) futures . remove ( futureUuid ) // Clean up future
else if ( ! future . isSuccess ) {
2011-03-24 16:23:37 +01:00
val f = futures . remove ( futureUuid ) // Clean up future
if ( f ne null ) f . completeWithException ( future . getCause )
notifyListeners ( RemoteClientWriteFailed ( message , future . getCause , module , remoteAddress ) )
}
}
2011-03-25 16:06:55 +01:00
pendingRequests . remove ( pendingRequest )
2011-05-18 17:25:30 +02:00
pendingRequest = pendingRequests . peek // try to grab next message
2011-03-24 16:23:37 +01:00
}
}
2011-01-01 21:01:44 +01:00
}
/* *
2011-01-03 12:42:30 +01:00
* RemoteClient represents a connection to an Akka node . Is used to send messages to remote actors on the node .
2011-01-01 21:01:44 +01:00
*
* @author < a href = "http://jonasboner.com" > Jonas Bon & # 233 ; r </ a >
*/
class ActiveRemoteClient private [ akka ] (
2011-01-03 13:49:39 +01:00
module : NettyRemoteClientModule , remoteAddress : InetSocketAddress ,
2011-05-18 17:25:30 +02:00
val loader : Option [ ClassLoader ] = None , notifyListenersFun : ( ⇒ Any ) ⇒ Unit ) extends RemoteClient ( module , remoteAddress ) {
2011-01-03 12:42:30 +01:00
import RemoteClientSettings._
2011-03-24 16:23:37 +01:00
2011-01-01 21:01:44 +01:00
//FIXME rewrite to a wrapper object (minimize volatile access and maximize encapsulation)
2011-05-18 17:25:30 +02:00
@volatile
private var bootstrap : ClientBootstrap = _
@volatile
private [ remote ] var connection : ChannelFuture = _
@volatile
private [ remote ] var openChannels : DefaultChannelGroup = _
@volatile
private var timer : HashedWheelTimer = _
@volatile
private var reconnectionTimeWindowStart = 0L
def notifyListeners ( msg : ⇒ Any ) : Unit = notifyListenersFun ( msg )
2011-01-01 21:01:44 +01:00
def currentChannel = connection . getChannel
def connect ( reconnectIfAlreadyConnected : Boolean = false ) : Boolean = {
runSwitch switchOn {
2011-01-31 12:41:39 +01:00
openChannels = new DefaultDisposableChannelGroup ( classOf [ RemoteClient ] . getName )
2011-01-01 21:01:44 +01:00
timer = new HashedWheelTimer
bootstrap = new ClientBootstrap ( new NioClientSocketChannelFactory ( Executors . newCachedThreadPool , Executors . newCachedThreadPool ) )
2011-04-07 13:03:23 +02:00
bootstrap . setPipelineFactory ( new ActiveRemoteClientPipelineFactory ( name , futures , bootstrap , remoteAddress , timer , this ) )
2011-01-01 21:01:44 +01:00
bootstrap . setOption ( "tcpNoDelay" , true )
bootstrap . setOption ( "keepAlive" , true )
// Wait until the connection attempt succeeds or fails.
connection = bootstrap . connect ( remoteAddress )
openChannels . add ( connection . awaitUninterruptibly . getChannel )
if ( ! connection . isSuccess ) {
notifyListeners ( RemoteClientError ( connection . getCause , module , remoteAddress ) )
false
} else {
2011-05-20 19:40:11 +02:00
//Send cookie
val handshake = RemoteControlProtocol . newBuilder . setCommandType ( CommandType . CONNECT )
if ( SECURE_COOKIE . nonEmpty )
handshake . setCookie ( SECURE_COOKIE . get )
connection . getChannel . write ( RemoteEncoder . encode ( handshake . build ) )
2011-03-05 14:59:22 +01:00
//Add a task that does GCing of expired Futures
2011-02-08 15:06:13 +01:00
timer . newTimeout ( new TimerTask ( ) {
def run ( timeout : Timeout ) = {
2011-05-18 17:25:30 +02:00
if ( isRunning ) {
2011-02-08 15:06:13 +01:00
val i = futures . entrySet . iterator
2011-05-18 17:25:30 +02:00
while ( i . hasNext ) {
2011-02-08 15:06:13 +01:00
val e = i . next
if ( e . getValue . isExpired )
futures . remove ( e . getKey )
}
}
}
} , RemoteClientSettings . REAP_FUTURES_DELAY . length , RemoteClientSettings . REAP_FUTURES_DELAY . unit )
2011-01-01 21:01:44 +01:00
notifyListeners ( RemoteClientStarted ( module , remoteAddress ) )
true
}
} match {
2011-05-18 17:25:30 +02:00
case true ⇒ true
case false if reconnectIfAlreadyConnected ⇒
2011-01-01 21:01:44 +01:00
openChannels . remove ( connection . getChannel )
connection . getChannel . close
connection = bootstrap . connect ( remoteAddress )
openChannels . add ( connection . awaitUninterruptibly . getChannel ) // Wait until the connection attempt succeeds or fails.
if ( ! connection . isSuccess ) {
notifyListeners ( RemoteClientError ( connection . getCause , module , remoteAddress ) )
false
2011-05-20 19:40:11 +02:00
} else {
//Send cookie
val handshake = RemoteControlProtocol . newBuilder . setCommandType ( CommandType . CONNECT )
if ( SECURE_COOKIE . nonEmpty )
handshake . setCookie ( SECURE_COOKIE . get )
connection . getChannel . write ( RemoteEncoder . encode ( handshake . build ) )
true
}
2011-05-18 17:25:30 +02:00
case false ⇒ false
2011-01-01 21:01:44 +01:00
}
}
2011-03-14 11:21:41 +01:00
//Please note that this method does _not_ remove the ARC from the NettyRemoteClientModule's map of clients
2011-04-29 10:20:16 +02:00
def shutdown ( ) = runSwitch switchOff {
2011-01-01 21:01:44 +01:00
notifyListeners ( RemoteClientShutdown ( module , remoteAddress ) )
2011-04-12 10:53:56 +02:00
timer . stop ( )
2011-01-01 21:01:44 +01:00
timer = null
openChannels . close . awaitUninterruptibly
openChannels = null
2011-04-29 10:20:16 +02:00
bootstrap . releaseExternalResources ( )
2011-01-01 21:01:44 +01:00
bootstrap = null
connection = null
2011-04-29 10:20:16 +02:00
pendingRequests . clear ( )
2011-01-01 21:01:44 +01:00
}
2010-12-15 17:52:31 +01:00
private [ akka ] def isWithinReconnectionTimeWindow : Boolean = {
if ( reconnectionTimeWindowStart == 0L ) {
reconnectionTimeWindowStart = System . currentTimeMillis
true
} else {
2011-03-05 15:01:07 +01:00
/* Time left > 0 */ ( RECONNECTION_TIME_WINDOW - ( System . currentTimeMillis - reconnectionTimeWindowStart ) ) > 0
2010-12-15 17:52:31 +01:00
}
}
private [ akka ] def resetReconnectionTimeWindow = reconnectionTimeWindowStart = 0L
}
/* *
* @author < a href = "http://jonasboner.com" > Jonas Bon & # 233 ; r </ a >
*/
2011-01-01 21:01:44 +01:00
class ActiveRemoteClientPipelineFactory (
2010-12-15 17:52:31 +01:00
name : String ,
2011-05-23 11:31:01 +02:00
futures : ConcurrentMap [ Uuid , Promise [ _ ] ] ,
2010-12-15 17:52:31 +01:00
bootstrap : ClientBootstrap ,
2011-03-14 11:21:41 +01:00
remoteAddress : InetSocketAddress ,
2010-12-15 17:52:31 +01:00
timer : HashedWheelTimer ,
2011-01-01 21:01:44 +01:00
client : ActiveRemoteClient ) extends ChannelPipelineFactory {
2010-12-15 17:52:31 +01:00
def getPipeline : ChannelPipeline = {
2011-05-18 17:25:30 +02:00
val timeout = new ReadTimeoutHandler ( timer , RemoteClientSettings . READ_TIMEOUT . length , RemoteClientSettings . READ_TIMEOUT . unit )
val lenDec = new LengthFieldBasedFrameDecoder ( RemoteClientSettings . MESSAGE_FRAME_SIZE , 0 , 4 , 0 , 4 )
val lenPrep = new LengthFieldPrepender ( 4 )
2011-03-05 14:55:58 +01:00
val protobufDec = new ProtobufDecoder ( AkkaRemoteProtocol . getDefaultInstance )
2010-12-15 17:52:31 +01:00
val protobufEnc = new ProtobufEncoder
2011-05-18 17:25:30 +02:00
val ( enc , dec ) = RemoteServerSettings . COMPRESSION_SCHEME match {
case "zlib" ⇒ ( new ZlibEncoder ( RemoteServerSettings . ZLIB_COMPRESSION_LEVEL ) : : Nil , new ZlibDecoder : : Nil )
case _ ⇒ ( Nil , Nil )
2010-12-15 17:52:31 +01:00
}
2011-04-07 13:03:23 +02:00
val remoteClient = new ActiveRemoteClientHandler ( name , futures , bootstrap , remoteAddress , timer , client )
2011-03-09 13:24:17 +01:00
val stages : List [ ChannelHandler ] = timeout : : dec ::: lenDec :: protobufDec :: enc ::: lenPrep :: protobufEnc :: remoteClient :: Nil
2010-12-15 17:52:31 +01:00
new StaticChannelPipeline ( stages : _ * )
}
}
/* *
* @author < a href = "http://jonasboner.com" > Jonas Bon & # 233 ; r </ a >
*/
@ChannelHandler . Sharable
2011-01-01 21:01:44 +01:00
class ActiveRemoteClientHandler (
2010-12-15 17:52:31 +01:00
val name : String ,
2011-05-23 11:31:01 +02:00
val futures : ConcurrentMap [ Uuid , Promise [ _ ] ] ,
2010-12-15 17:52:31 +01:00
val bootstrap : ClientBootstrap ,
2011-03-14 11:21:41 +01:00
val remoteAddress : InetSocketAddress ,
2010-12-15 17:52:31 +01:00
val timer : HashedWheelTimer ,
2011-01-01 21:01:44 +01:00
val client : ActiveRemoteClient )
2011-02-28 22:54:32 +01:00
extends SimpleChannelUpstreamHandler {
2010-12-15 17:52:31 +01:00
override def messageReceived ( ctx : ChannelHandlerContext , event : MessageEvent ) {
try {
2010-12-17 16:09:21 +01:00
event . getMessage match {
2011-05-18 17:25:30 +02:00
case arp : AkkaRemoteProtocol if arp . hasInstruction ⇒
2011-03-05 14:55:58 +01:00
val rcp = arp . getInstruction
rcp . getCommandType match {
2011-05-18 17:25:30 +02:00
case CommandType . SHUTDOWN ⇒ spawn { client . module . shutdownClientConnection ( remoteAddress ) }
2011-03-05 14:55:58 +01:00
}
2011-05-18 17:25:30 +02:00
case arp : AkkaRemoteProtocol if arp . hasMessage ⇒
2011-03-05 14:55:58 +01:00
val reply = arp . getMessage
2010-12-17 16:09:21 +01:00
val replyUuid = uuidFrom ( reply . getActorInfo . getUuid . getHigh , reply . getActorInfo . getUuid . getLow )
2011-05-23 11:31:01 +02:00
val future = futures . remove ( replyUuid ) . asInstanceOf [ Promise [ Any ] ]
2010-12-20 12:10:46 +01:00
2010-12-17 16:09:21 +01:00
if ( reply . hasMessage ) {
if ( future eq null ) throw new IllegalActorStateException ( "Future mapped to UUID " + replyUuid + " does not exist" )
val message = MessageSerializer . deserialize ( reply . getMessage )
future . completeWithResult ( message )
} else {
2011-04-07 13:03:23 +02:00
future . completeWithException ( parseException ( reply , client . loader ) )
2010-12-15 17:52:31 +01:00
}
2011-05-18 17:25:30 +02:00
case other ⇒
2010-12-29 16:08:43 +01:00
throw new RemoteClientException ( "Unknown message received in remote client handler: " + other , client . module , client . remoteAddress )
2010-12-15 17:52:31 +01:00
}
} catch {
2011-05-18 17:25:30 +02:00
case e : Throwable ⇒
2011-03-18 23:04:48 +01:00
EventHandler . error ( e , this , e . getMessage )
2010-12-29 16:08:43 +01:00
client . notifyListeners ( RemoteClientError ( e , client . module , client . remoteAddress ) )
2010-12-15 17:52:31 +01:00
throw e
}
}
override def channelClosed ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = client . runSwitch ifOn {
if ( client . isWithinReconnectionTimeWindow ) {
timer . newTimeout ( new TimerTask ( ) {
def run ( timeout : Timeout ) = {
2011-02-08 15:06:13 +01:00
if ( client . isRunning ) {
client . openChannels . remove ( event . getChannel )
client . connect ( reconnectIfAlreadyConnected = true )
}
2010-12-15 17:52:31 +01:00
}
2011-01-03 12:42:30 +01:00
} , RemoteClientSettings . RECONNECT_DELAY . toMillis , TimeUnit . MILLISECONDS )
2011-03-14 11:21:41 +01:00
} else spawn { client . module . shutdownClientConnection ( remoteAddress ) }
2010-12-15 17:52:31 +01:00
}
override def channelConnected ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = {
2011-03-25 16:06:55 +01:00
try {
2011-03-27 22:58:50 +02:00
if ( client . useTransactionLog ) client . sendPendingRequests ( ) // try to send pending requests (still there after client/server crash ard reconnect
2011-03-25 16:06:55 +01:00
client . notifyListeners ( RemoteClientConnected ( client . module , client . remoteAddress ) )
client . resetReconnectionTimeWindow
} catch {
2011-05-18 17:25:30 +02:00
case e : Throwable ⇒
2011-03-25 16:06:55 +01:00
EventHandler . error ( e , this , e . getMessage )
client . notifyListeners ( RemoteClientError ( e , client . module , client . remoteAddress ) )
throw e
}
2010-12-15 17:52:31 +01:00
}
override def channelDisconnected ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = {
2010-12-29 16:08:43 +01:00
client . notifyListeners ( RemoteClientDisconnected ( client . module , client . remoteAddress ) )
2010-12-15 17:52:31 +01:00
}
override def exceptionCaught ( ctx : ChannelHandlerContext , event : ExceptionEvent ) = {
2011-03-14 11:21:41 +01:00
event . getCause match {
2011-05-18 17:25:30 +02:00
case e : ReadTimeoutException ⇒
2011-03-14 11:21:41 +01:00
spawn { client . module . shutdownClientConnection ( remoteAddress ) }
2011-05-18 17:25:30 +02:00
case e ⇒
2011-03-14 11:21:41 +01:00
client . notifyListeners ( RemoteClientError ( e , client . module , client . remoteAddress ) )
event . getChannel . close //FIXME Is this the correct behavior?
}
2010-12-15 17:52:31 +01:00
}
private def parseException ( reply : RemoteMessageProtocol , loader : Option [ ClassLoader ] ) : Throwable = {
val exception = reply . getException
val classname = exception . getClassname
2010-12-20 12:10:46 +01:00
try {
val exceptionClass = if ( loader . isDefined ) loader . get . loadClass ( classname )
2011-05-18 17:25:30 +02:00
else Class . forName ( classname )
2010-12-20 12:10:46 +01:00
exceptionClass
. getConstructor ( Array [ Class [ _ ] ] ( classOf [ String ] ) : _ * )
. newInstance ( exception . getMessage ) . asInstanceOf [ Throwable ]
} catch {
2011-05-18 17:25:30 +02:00
case problem : Throwable ⇒
2011-03-18 23:04:48 +01:00
EventHandler . error ( problem , this , problem . getMessage )
2011-03-25 16:25:36 +01:00
CannotInstantiateRemoteExceptionDueToRemoteProtocolParsingErrorException ( problem , classname , exception . getMessage )
2010-12-20 12:10:46 +01:00
}
2010-12-15 17:52:31 +01:00
}
}
2009-11-24 17:41:08 +01:00
/* *
2010-12-15 17:52:31 +01:00
* Provides the implementation of the Netty remote support
2009-11-24 17:41:08 +01:00
*/
2010-12-17 16:09:21 +01:00
class NettyRemoteSupport extends RemoteSupport with NettyRemoteServerModule with NettyRemoteClientModule {
2011-03-24 12:48:40 +01:00
// Needed for remote testing and switching on/off under run
2011-01-05 17:30:56 +01:00
val optimizeLocal = new AtomicBoolean ( true )
2010-12-17 16:09:21 +01:00
def optimizeLocalScoped_? ( ) = optimizeLocal . get
2010-12-15 17:52:31 +01:00
2011-05-25 16:18:35 +02:00
protected [ akka ] def actorFor (
actorAddress : String ,
timeout : Long ,
host : String ,
port : Int ,
loader : Option [ ClassLoader ] ) : ActorRef = {
val homeInetSocketAddress = this . address
2010-12-21 14:36:47 +01:00
if ( optimizeLocalScoped_? ) {
2011-05-25 16:18:35 +02:00
if ( ( host == homeInetSocketAddress . getAddress . getHostAddress ||
host == homeInetSocketAddress . getHostName ) &&
port == homeInetSocketAddress . getPort ) { //TODO: switch to InetSocketAddress.equals?
2011-05-24 19:04:25 +02:00
val localRef = findActorByAddressOrUuid ( actorAddress , actorAddress )
2010-12-21 14:36:47 +01:00
if ( localRef ne null ) return localRef //Code significantly simpler with the return statement
}
}
2011-05-25 16:18:35 +02:00
val remoteInetSocketAddress = new InetSocketAddress ( host , port )
EventHandler . debug ( this ,
"Creating RemoteActorRef with address [%s] connected to [%s]"
. format ( actorAddress , remoteInetSocketAddress ) )
RemoteActorRef ( remoteInetSocketAddress , actorAddress , timeout , loader )
2010-12-15 17:52:31 +01:00
}
}
2010-12-20 16:58:05 +01:00
class NettyRemoteServer ( serverModule : NettyRemoteServerModule , val host : String , val port : Int , val loader : Option [ ClassLoader ] ) {
2009-12-18 21:26:03 +01:00
2010-12-20 16:58:05 +01:00
val name = "NettyRemoteServer@" + host + ":" + port
2011-05-18 17:25:30 +02:00
val address = new InetSocketAddress ( host , port )
2009-06-25 23:47:30 +02:00
2011-05-18 17:25:30 +02:00
private val factory = new NioServerSocketChannelFactory ( Executors . newCachedThreadPool , Executors . newCachedThreadPool )
2009-06-24 15:12:47 +02:00
2009-06-25 13:07:58 +02:00
private val bootstrap = new ServerBootstrap ( factory )
2009-06-24 15:12:47 +02:00
2009-12-18 21:26:03 +01:00
// group of open channels, used for clean-up
2011-01-31 12:41:39 +01:00
private val openChannels : ChannelGroup = new DefaultDisposableChannelGroup ( "akka-remote-server" )
2009-12-18 21:26:03 +01:00
2010-12-20 16:58:05 +01:00
val pipelineFactory = new RemoteServerPipelineFactory ( name , openChannels , loader , serverModule )
bootstrap . setPipelineFactory ( pipelineFactory )
2011-02-07 10:31:05 +01:00
bootstrap . setOption ( "backlog" , RemoteServerSettings . BACKLOG )
2010-12-20 16:58:05 +01:00
bootstrap . setOption ( "child.tcpNoDelay" , true )
bootstrap . setOption ( "child.keepAlive" , true )
bootstrap . setOption ( "child.reuseAddress" , true )
2011-01-03 12:42:30 +01:00
bootstrap . setOption ( "child.connectTimeoutMillis" , RemoteServerSettings . CONNECTION_TIMEOUT_MILLIS . toMillis )
2010-12-20 16:58:05 +01:00
2010-12-21 14:36:47 +01:00
openChannels . add ( bootstrap . bind ( address ) )
2010-12-20 16:58:05 +01:00
serverModule . notifyListeners ( RemoteServerStarted ( serverModule ) )
2011-04-29 10:20:16 +02:00
def shutdown ( ) {
2010-12-20 16:58:05 +01:00
try {
2011-03-05 14:55:58 +01:00
val shutdownSignal = {
2011-05-20 19:40:11 +02:00
val b = RemoteControlProtocol . newBuilder . setCommandType ( CommandType . SHUTDOWN )
2011-03-05 14:55:58 +01:00
if ( RemoteClientSettings . SECURE_COOKIE . nonEmpty )
b . setCookie ( RemoteClientSettings . SECURE_COOKIE . get )
2011-05-20 19:40:11 +02:00
2011-03-05 14:55:58 +01:00
b . build
}
openChannels . write ( RemoteEncoder . encode ( shutdownSignal ) ) . awaitUninterruptibly
2010-12-20 16:58:05 +01:00
openChannels . disconnect
openChannels . close . awaitUninterruptibly
2011-04-29 10:20:16 +02:00
bootstrap . releaseExternalResources ( )
2010-12-20 16:58:05 +01:00
serverModule . notifyListeners ( RemoteServerShutdown ( serverModule ) )
} catch {
2011-05-18 17:25:30 +02:00
case e : Exception ⇒
2011-03-18 23:04:48 +01:00
EventHandler . error ( e , this , e . getMessage )
2010-12-20 16:58:05 +01:00
}
}
}
2011-05-18 17:25:30 +02:00
trait NettyRemoteServerModule extends RemoteServerModule { self : RemoteModule ⇒
2011-01-03 12:42:30 +01:00
import RemoteServerSettings._
2010-12-20 16:58:05 +01:00
private [ akka ] val currentServer = new AtomicReference [ Option [ NettyRemoteServer ] ] ( None )
2011-03-14 11:21:41 +01:00
2010-12-21 14:36:47 +01:00
def address = currentServer . get match {
2011-05-18 17:25:30 +02:00
case s : Some [ NettyRemoteServer ] ⇒ s . get . address
case None ⇒ ReflectiveAccess . RemoteModule . configDefaultAddress
2010-12-20 16:58:05 +01:00
}
def name = currentServer . get match {
2011-05-18 17:25:30 +02:00
case s : Some [ NettyRemoteServer ] ⇒ s . get . name
case None ⇒
val a = ReflectiveAccess . RemoteModule . configDefaultAddress
2011-03-02 18:48:13 +01:00
"NettyRemoteServer@" + a . getAddress . getHostAddress + ":" + a . getPort
2010-12-20 16:58:05 +01:00
}
private val _isRunning = new Switch ( false )
2010-12-14 18:22:46 +01:00
def isRunning = _isRunning . isOn
2009-10-13 11:18:21 +02:00
2010-12-15 17:52:31 +01:00
def start ( _hostname : String , _port : Int , loader : Option [ ClassLoader ] = None ) : RemoteServerModule = guard withGuard {
2009-12-27 06:35:25 +01:00
try {
2010-12-14 18:22:46 +01:00
_isRunning switchOn {
2010-12-20 16:58:05 +01:00
currentServer . set ( Some ( new NettyRemoteServer ( this , _hostname , _port , loader ) ) )
2010-04-06 12:45:09 +02:00
}
2009-12-27 06:35:25 +01:00
} catch {
2011-05-18 17:25:30 +02:00
case e : Exception ⇒
2011-03-18 23:04:48 +01:00
EventHandler . error ( e , this , e . getMessage )
2010-09-12 11:24:27 +02:00
notifyListeners ( RemoteServerError ( e , this ) )
2009-06-25 23:47:30 +02:00
}
2010-05-16 10:59:06 +02:00
this
2009-06-25 23:47:30 +02:00
}
2009-11-24 17:41:08 +01:00
2011-04-29 10:20:16 +02:00
def shutdownServerModule ( ) = guard withGuard {
2010-12-14 18:22:46 +01:00
_isRunning switchOff {
2011-05-18 17:25:30 +02:00
currentServer . getAndSet ( None ) foreach { instance ⇒
2011-04-29 10:20:16 +02:00
instance . shutdown ( )
2010-07-14 14:38:56 +02:00
}
2010-04-06 12:45:09 +02:00
}
2009-11-24 17:41:08 +01:00
}
2010-02-16 15:39:09 +01:00
2010-02-17 15:32:17 +01:00
/* *
2011-04-08 15:29:14 +02:00
* Register RemoteModule Actor by a specific 'id' passed as argument .
2010-05-07 11:19:19 +02:00
* < p />
2010-05-21 20:08:49 +02:00
* NOTE : If you use this method to register your remote actor then you must unregister the actor by this ID yourself .
2010-02-17 15:32:17 +01:00
*/
2010-12-14 18:22:46 +01:00
def register ( id : String , actorRef : ActorRef ) : Unit = guard withGuard {
2010-10-26 15:23:50 +02:00
if ( id . startsWith ( UUID_PREFIX ) ) register ( id . substring ( UUID_PREFIX . length ) , actorRef , actorsByUuid )
else register ( id , actorRef , actors )
2010-09-20 12:33:30 +02:00
}
2010-12-14 18:22:46 +01:00
def registerByUuid ( actorRef : ActorRef ) : Unit = guard withGuard {
register ( actorRef . uuid . toString , actorRef , actorsByUuid )
}
private def register [ Key ] ( id : Key , actorRef : ActorRef , registry : ConcurrentHashMap [ Key , ActorRef ] ) {
if ( _isRunning . isOn ) {
registry . put ( id , actorRef ) //TODO change to putIfAbsent
2011-04-12 09:55:32 +02:00
if ( ! actorRef . isRunning ) actorRef . start ( )
2010-12-14 18:22:46 +01:00
}
2010-11-14 13:10:13 -06:00
}
2010-11-29 20:49:15 +01:00
/* *
2011-04-08 15:29:14 +02:00
* Register RemoteModule Session Actor by a specific 'id' passed as argument .
2010-11-29 20:49:15 +01:00
* < p />
* NOTE : If you use this method to register your remote actor then you must unregister the actor by this ID yourself .
*/
2011-05-18 17:25:30 +02:00
def registerPerSession ( id : String , factory : ⇒ ActorRef ) : Unit = synchronized {
registerPerSession ( id , ( ) ⇒ factory , actorsFactories )
2010-09-20 12:33:30 +02:00
}
2011-05-18 17:25:30 +02:00
private def registerPerSession [ Key ] ( id : Key , factory : ( ) ⇒ ActorRef , registry : ConcurrentHashMap [ Key , ( ) ⇒ ActorRef ] ) {
2010-12-14 18:22:46 +01:00
if ( _isRunning . isOn )
2010-11-22 11:57:17 +01:00
registry . put ( id , factory ) //TODO change to putIfAbsent
2010-11-14 13:10:13 -06:00
}
2010-05-05 22:45:19 +02:00
/* *
2011-04-08 15:29:14 +02:00
* Unregister RemoteModule Actor that is registered using its 'id' field ( not custom ID ) .
2010-05-05 22:45:19 +02:00
*/
2010-12-14 18:22:46 +01:00
def unregister ( actorRef : ActorRef ) : Unit = guard withGuard {
if ( _isRunning . isOn ) {
2011-04-08 21:16:05 +02:00
actors . remove ( actorRef . address , actorRef )
2010-10-26 15:23:50 +02:00
actorsByUuid . remove ( actorRef . uuid , actorRef )
2010-05-07 11:19:19 +02:00
}
}
/* *
2011-04-08 15:29:14 +02:00
* Unregister RemoteModule Actor by specific 'id' .
2010-05-07 11:19:19 +02:00
* < p />
2010-05-21 20:08:49 +02:00
* NOTE : You need to call this method if you have registered an actor by a custom ID .
2010-05-07 11:19:19 +02:00
*/
2010-12-14 18:22:46 +01:00
def unregister ( id : String ) : Unit = guard withGuard {
if ( _isRunning . isOn ) {
2010-10-29 16:33:31 +02:00
if ( id . startsWith ( UUID_PREFIX ) ) actorsByUuid . remove ( id . substring ( UUID_PREFIX . length ) )
2010-10-26 15:23:50 +02:00
else {
val actorRef = actors get id
actorsByUuid . remove ( actorRef . uuid , actorRef )
2011-05-18 17:25:30 +02:00
actors . remove ( id , actorRef )
2010-09-20 12:33:30 +02:00
}
2010-05-05 22:45:19 +02:00
}
}
2010-08-18 09:52:11 +02:00
2010-11-14 13:10:13 -06:00
/* *
2011-04-08 15:29:14 +02:00
* Unregister RemoteModule Actor by specific 'id' .
2010-11-14 13:10:13 -06:00
* < p />
* NOTE : You need to call this method if you have registered an actor by a custom ID .
*/
2010-12-14 18:22:46 +01:00
def unregisterPerSession ( id : String ) : Unit = {
if ( _isRunning . isOn ) {
2010-11-14 13:10:13 -06:00
actorsFactories . remove ( id )
}
}
2009-06-24 15:12:47 +02:00
}
2009-07-27 21:21:28 +02:00
/* *
* @author < a href = "http://jonasboner.com" > Jonas Bon & # 233 ; r </ a >
*/
2009-12-27 22:56:55 +01:00
class RemoteServerPipelineFactory (
2011-05-18 17:25:30 +02:00
val name : String ,
val openChannels : ChannelGroup ,
val loader : Option [ ClassLoader ] ,
val server : NettyRemoteServerModule ) extends ChannelPipelineFactory {
2011-01-03 12:42:30 +01:00
import RemoteServerSettings._
2009-11-24 17:41:08 +01:00
2009-12-18 21:26:03 +01:00
def getPipeline : ChannelPipeline = {
2011-05-18 17:25:30 +02:00
val lenDec = new LengthFieldBasedFrameDecoder ( MESSAGE_FRAME_SIZE , 0 , 4 , 0 , 4 )
val lenPrep = new LengthFieldPrepender ( 4 )
2011-03-05 14:55:58 +01:00
val protobufDec = new ProtobufDecoder ( AkkaRemoteProtocol . getDefaultInstance )
2010-07-15 21:33:44 +02:00
val protobufEnc = new ProtobufEncoder
2011-05-18 17:25:30 +02:00
val ( enc , dec ) = COMPRESSION_SCHEME match {
case "zlib" ⇒ ( new ZlibEncoder ( ZLIB_COMPRESSION_LEVEL ) : : Nil , new ZlibDecoder : : Nil )
case _ ⇒ ( Nil , Nil )
2009-11-22 14:32:27 +01:00
}
2011-03-23 11:31:19 +01:00
val execution = new ExecutionHandler (
new OrderedMemoryAwareThreadPoolExecutor (
EXECUTION_POOL_SIZE ,
MAX_CHANNEL_MEMORY_SIZE ,
MAX_TOTAL_MEMORY_SIZE ,
EXECUTION_POOL_KEEPALIVE . length ,
2011-05-18 17:25:30 +02:00
EXECUTION_POOL_KEEPALIVE . unit ) )
2011-05-20 19:40:11 +02:00
val authenticator = if ( REQUIRE_COOKIE ) new RemoteServerAuthenticationHandler ( SECURE_COOKIE ) : : Nil else Nil
2010-09-07 10:12:26 +02:00
val remoteServer = new RemoteServerHandler ( name , openChannels , loader , server )
2011-05-20 19:40:11 +02:00
val stages : List [ ChannelHandler ] = dec : :: lenDec :: protobufDec :: enc ::: lenPrep :: protobufEnc :: execution :: authenticator ::: remoteServer :: Nil
2009-12-30 08:48:22 +01:00
new StaticChannelPipeline ( stages : _ * )
2009-07-18 00:16:32 +02:00
}
}
2011-05-20 19:40:11 +02:00
@ChannelHandler . Sharable
class RemoteServerAuthenticationHandler ( secureCookie : Option [ String ] ) extends SimpleChannelUpstreamHandler {
val authenticated = new AnyRef
override def messageReceived ( ctx : ChannelHandlerContext , event : MessageEvent ) = secureCookie match {
case None ⇒ ctx . sendUpstream ( event )
case Some ( cookie ) ⇒
ctx . getAttachment match {
case `authenticated` ⇒ ctx . sendUpstream ( event )
case null ⇒ event . getMessage match {
case remoteProtocol : AkkaRemoteProtocol if remoteProtocol . hasInstruction ⇒
remoteProtocol . getInstruction . getCookie match {
case `cookie` ⇒
ctx . setAttachment ( authenticated )
ctx . sendUpstream ( event )
case _ ⇒
throw new SecurityException (
"The remote client [" + ctx . getChannel . getRemoteAddress + "] secure cookie is not the same as remote server secure cookie" )
}
case _ ⇒
throw new SecurityException ( "The remote client [" + ctx . getChannel . getRemoteAddress + "] is not Authorized!" )
}
}
}
}
2009-07-27 21:21:28 +02:00
/* *
* @author < a href = "http://jonasboner.com" > Jonas Bon & # 233 ; r </ a >
*/
2010-03-29 09:33:32 +02:00
@ChannelHandler . Sharable
2009-12-27 22:56:55 +01:00
class RemoteServerHandler (
2011-05-18 17:25:30 +02:00
val name : String ,
val openChannels : ChannelGroup ,
val applicationLoader : Option [ ClassLoader ] ,
val server : NettyRemoteServerModule ) extends SimpleChannelUpstreamHandler {
2011-01-03 12:42:30 +01:00
import RemoteServerSettings._
2009-12-18 21:26:03 +01:00
2011-01-03 14:44:15 +01:00
applicationLoader . foreach ( MessageSerializer . setClassLoader ( _ ) ) //TODO: REVISIT: THIS FEELS A BIT DODGY
2010-11-19 11:53:51 -06:00
val sessionActors = new ChannelLocal [ ConcurrentHashMap [ String , ActorRef ] ] ( )
2010-11-14 13:10:13 -06:00
2011-01-03 14:44:15 +01:00
//Writes the specified message to the specified channel and propagates write errors to listeners
2011-03-05 14:55:58 +01:00
private def write ( channel : Channel , payload : AkkaRemoteProtocol ) : Unit = {
channel . write ( payload ) . addListener (
2011-01-03 14:44:15 +01:00
new ChannelFutureListener {
def operationComplete ( future : ChannelFuture ) : Unit = {
if ( future . isCancelled ) {
//Not interesting at the moment
} else if ( ! future . isSuccess ) {
val socketAddress = future . getChannel . getRemoteAddress match {
2011-05-18 17:25:30 +02:00
case i : InetSocketAddress ⇒ Some ( i )
case _ ⇒ None
2011-01-03 14:44:15 +01:00
}
2011-03-05 14:55:58 +01:00
server . notifyListeners ( RemoteServerWriteFailed ( payload , future . getCause , server , socketAddress ) )
2011-01-03 14:44:15 +01:00
}
}
} )
2011-01-05 11:43:49 +01:00
}
2009-12-27 22:56:55 +01:00
2009-12-18 21:26:03 +01:00
/* *
2011-01-03 12:42:30 +01:00
* ChannelOpen overridden to store open channels for a clean postStop of a node .
2010-06-15 13:15:00 +02:00
* If a channel is closed before , it is automatically removed from the open channels group .
2009-12-18 21:26:03 +01:00
*/
2010-08-21 16:37:33 +02:00
override def channelOpen ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = openChannels . add ( ctx . getChannel )
2010-08-10 21:42:27 +02:00
2010-08-21 16:37:33 +02:00
override def channelConnected ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = {
2010-10-31 07:13:00 +01:00
val clientAddress = getClientAddress ( ctx )
2010-11-19 11:53:51 -06:00
sessionActors . set ( event . getChannel ( ) , new ConcurrentHashMap [ String , ActorRef ] ( ) )
2011-03-09 13:24:17 +01:00
server . notifyListeners ( RemoteServerClientConnected ( server , clientAddress ) )
2010-04-25 20:32:52 +02:00
}
2010-10-31 07:13:00 +01:00
override def channelDisconnected ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = {
val clientAddress = getClientAddress ( ctx )
2011-01-03 14:44:15 +01:00
2010-11-14 16:26:33 -06:00
// stop all session actors
2011-05-18 17:25:30 +02:00
for (
map ← Option ( sessionActors . remove ( event . getChannel ) ) ;
2011-05-25 16:18:35 +02:00
actor ← collectionAsScalaIterable ( map . values )
2011-05-18 17:25:30 +02:00
) {
try { actor ! PoisonPill } catch { case e : Exception ⇒ }
2010-11-14 16:26:33 -06:00
}
2011-03-18 18:03:10 +01:00
2010-10-31 07:13:00 +01:00
server . notifyListeners ( RemoteServerClientDisconnected ( server , clientAddress ) )
}
2010-08-21 16:37:33 +02:00
override def channelClosed ( ctx : ChannelHandlerContext , event : ChannelStateEvent ) = {
2010-10-31 07:13:00 +01:00
val clientAddress = getClientAddress ( ctx )
server . notifyListeners ( RemoteServerClientClosed ( server , clientAddress ) )
2010-08-21 16:37:33 +02:00
}
2011-05-24 19:04:25 +02:00
override def messageReceived ( ctx : ChannelHandlerContext , event : MessageEvent ) = {
event . getMessage match {
case null ⇒
throw new IllegalActorStateException ( "Message in remote MessageEvent is null: " + event )
case remote : AkkaRemoteProtocol if remote . hasMessage ⇒
handleRemoteMessageProtocol ( remote . getMessage , event . getChannel )
//case remote: AkkaRemoteProtocol if remote.hasInstruction => RemoteServer cannot receive control messages (yet)
case _ ⇒ //ignore
}
2009-06-24 15:12:47 +02:00
}
override def exceptionCaught ( ctx : ChannelHandlerContext , event : ExceptionEvent ) = {
event . getChannel . close
2010-09-12 11:24:27 +02:00
server . notifyListeners ( RemoteServerError ( event . getCause , server ) )
2009-06-24 15:12:47 +02:00
}
2011-01-03 13:49:39 +01:00
private def getClientAddress ( ctx : ChannelHandlerContext ) : Option [ InetSocketAddress ] =
ctx . getChannel . getRemoteAddress match {
2011-05-18 17:25:30 +02:00
case inet : InetSocketAddress ⇒ Some ( inet )
case _ ⇒ None
2011-01-03 13:49:39 +01:00
}
2010-10-31 07:13:00 +01:00
2011-05-24 19:04:25 +02:00
private def handleRemoteMessageProtocol ( request : RemoteMessageProtocol , channel : Channel ) = {
EventHandler . debug ( this , "Received remote message [%s]" . format ( request ) )
2011-05-19 14:29:21 +02:00
dispatchToActor ( request , channel )
2011-05-24 19:04:25 +02:00
}
2009-06-24 15:12:47 +02:00
2010-11-02 21:18:31 +01:00
private def dispatchToActor ( request : RemoteMessageProtocol , channel : Channel ) {
2010-07-26 18:47:25 +02:00
val actorInfo = request . getActorInfo
2010-10-29 16:33:31 +02:00
val actorRef =
2011-03-18 10:41:49 +01:00
try { createActor ( actorInfo , channel ) } catch {
2011-05-18 17:25:30 +02:00
case e : SecurityException ⇒
2011-03-18 23:04:48 +01:00
EventHandler . error ( e , this , e . getMessage )
2011-05-19 14:29:21 +02:00
write ( channel , createErrorReplyMessage ( e , request ) )
2010-10-28 21:18:25 +02:00
server . notifyListeners ( RemoteServerError ( e , server ) )
return
}
2010-07-26 18:47:25 +02:00
2010-06-24 08:48:48 +02:00
val message = MessageSerializer . deserialize ( request . getMessage )
2010-06-30 16:26:15 +02:00
val sender =
2010-07-02 00:16:11 +05:30
if ( request . hasSender ) Some ( RemoteActorSerialization . fromProtobufToRemoteActorRef ( request . getSender , applicationLoader ) )
2010-06-15 13:15:00 +02:00
else None
2010-07-26 18:47:25 +02:00
2010-08-12 15:36:05 +02:00
message match { // first match on system messages
2011-05-18 17:25:30 +02:00
case RemoteActorSystemMessage . Stop ⇒
2011-04-08 15:29:14 +02:00
if ( UNTRUSTED_MODE ) throw new SecurityException ( "RemoteModule server is operating is untrusted mode, can not stop the actor" )
2011-04-12 10:53:56 +02:00
else actorRef . stop ( )
2011-04-27 01:06:08 +02:00
2011-05-18 17:25:30 +02:00
case _ : LifeCycleMessage if ( UNTRUSTED_MODE ) ⇒
2011-04-08 15:29:14 +02:00
throw new SecurityException ( "RemoteModule server is operating is untrusted mode, can not pass on a LifeCycleMessage to the remote actor" )
2010-10-28 21:18:25 +02:00
2011-05-18 17:25:30 +02:00
case _ ⇒ // then match on user defined messages
2010-10-28 20:30:11 +02:00
if ( request . getOneWay ) actorRef . ! ( message ) ( sender )
2010-10-26 12:04:32 +02:00
else actorRef . postMessageToMailboxAndCreateFutureResultWithTimeout (
message ,
request . getActorInfo . getTimeout ,
None ,
2011-05-23 11:31:01 +02:00
Some ( new DefaultPromise [ Any ] ( request . getActorInfo . getTimeout ) .
2011-03-18 18:03:10 +01:00
onComplete ( _ . value . get match {
2011-05-19 14:29:21 +02:00
case l : Left [ Throwable , Any ] ⇒ write ( channel , createErrorReplyMessage ( l . a , request ) )
2011-05-18 17:25:30 +02:00
case r : Right [ Throwable , Any ] ⇒
val messageBuilder = RemoteActorSerialization . createRemoteMessageProtocolBuilder (
2010-11-12 12:11:53 +01:00
Some ( actorRef ) ,
Right ( request . getUuid ) ,
2011-04-08 21:16:05 +02:00
actorInfo . getAddress ,
2010-11-12 12:11:53 +01:00
actorInfo . getTimeout ,
2011-03-18 18:03:10 +01:00
r ,
2010-11-12 12:11:53 +01:00
true ,
2011-05-20 19:40:11 +02:00
Some ( actorRef ) )
2010-11-12 12:11:53 +01:00
// FIXME lift in the supervisor uuid management into toh createRemoteMessageProtocolBuilder method
if ( request . hasSupervisorUuid ) messageBuilder . setSupervisorUuid ( request . getSupervisorUuid )
2011-03-05 14:55:58 +01:00
write ( channel , RemoteEncoder . encode ( messageBuilder . build ) )
2011-05-18 17:25:30 +02:00
} ) ) )
2009-12-18 21:26:03 +01:00
}
2009-06-25 23:47:30 +02:00
}
2011-05-25 16:18:35 +02:00
/* *
* Creates a new instance of the actor with name , uuid and timeout specified as arguments .
*
* If actor already created then just return it from the registry .
*
* Does not start the actor .
*/
private def createActor ( actorInfo : ActorInfoProtocol , channel : Channel ) : ActorRef = {
val uuid = actorInfo . getUuid
val address = actorInfo . getAddress
EventHandler . debug ( this ,
"Creating an remotely available actor for address [%s] on node [%s]"
. format ( address , Config . nodename ) )
val actorRef = Actor . createActor ( address , ( ) ⇒ createSessionActor ( actorInfo , channel ) )
if ( actorRef eq null ) throw new IllegalActorStateException (
"Could not find a remote actor with address [" + address + "] or uuid [" + uuid + "]" )
actorRef
}
2010-11-14 16:26:33 -06:00
2010-04-06 12:45:09 +02:00
/* *
2010-11-18 12:48:31 -06:00
* gets the actor from the session , or creates one if there is a factory for it
2010-04-06 12:45:09 +02:00
*/
2010-11-18 12:48:31 -06:00
private def createSessionActor ( actorInfo : ActorInfoProtocol , channel : Channel ) : ActorRef = {
2010-09-16 13:50:57 +02:00
val uuid = actorInfo . getUuid
2011-04-08 21:16:05 +02:00
val address = actorInfo . getAddress
2011-01-03 13:49:39 +01:00
2011-04-08 21:16:05 +02:00
findSessionActor ( address , channel ) match {
2011-05-18 17:25:30 +02:00
case null ⇒ // we dont have it in the session either, see if we have a factory for it
2011-04-08 21:16:05 +02:00
server . findActorFactory ( address ) match {
2011-05-18 17:25:30 +02:00
case null ⇒ null
case factory ⇒
2011-01-03 13:49:39 +01:00
val actorRef = factory ( )
actorRef . uuid = parseUuid ( uuid ) //FIXME is this sensible?
2011-04-08 21:16:05 +02:00
sessionActors . get ( channel ) . put ( address , actorRef )
2011-04-12 09:55:32 +02:00
actorRef . start ( ) //Start it where's it's created
2011-01-03 13:49:39 +01:00
}
2011-05-18 17:25:30 +02:00
case sessionActor ⇒ sessionActor
2010-11-18 12:48:31 -06:00
}
}
2010-11-14 16:26:33 -06:00
2011-05-25 16:18:35 +02:00
private def findSessionActor ( id : String , channel : Channel ) : ActorRef =
sessionActors . get ( channel ) match {
case null ⇒ null
case map ⇒ map get id
2010-11-18 12:48:31 -06:00
}
2011-05-24 19:04:25 +02:00
2011-05-19 14:29:21 +02:00
private def createErrorReplyMessage ( exception : Throwable , request : RemoteMessageProtocol ) : AkkaRemoteProtocol = {
2010-07-26 18:47:25 +02:00
val actorInfo = request . getActorInfo
2010-11-02 18:11:58 +01:00
val messageBuilder = RemoteActorSerialization . createRemoteMessageProtocolBuilder (
None ,
2010-11-24 21:05:12 +01:00
Right ( request . getUuid ) ,
2011-04-08 21:16:05 +02:00
actorInfo . getAddress ,
2010-11-24 21:05:12 +01:00
actorInfo . getTimeout ,
2011-03-18 18:03:10 +01:00
Left ( exception ) ,
2010-11-24 21:05:12 +01:00
true ,
2010-11-02 18:11:58 +01:00
None )
if ( request . hasSupervisorUuid ) messageBuilder . setSupervisorUuid ( request . getSupervisorUuid )
2011-03-05 14:55:58 +01:00
RemoteEncoder . encode ( messageBuilder . build )
2010-07-26 18:47:25 +02:00
}
2010-10-26 12:04:32 +02:00
2011-05-18 17:25:30 +02:00
protected def parseUuid ( protocol : UuidProtocol ) : Uuid = uuidFrom ( protocol . getHigh , protocol . getLow )
2009-06-24 15:12:47 +02:00
}
2011-01-31 12:41:39 +01:00
class DefaultDisposableChannelGroup ( name : String ) extends DefaultChannelGroup ( name ) {
protected val guard = new ReadWriteGuard
2011-05-18 17:25:30 +02:00
protected val open = new AtomicBoolean ( true )
2011-01-31 12:41:39 +01:00
override def add ( channel : Channel ) : Boolean = guard withReadGuard {
2011-04-08 21:16:05 +02:00
if ( open . get ) {
2011-01-31 12:41:39 +01:00
super . add ( channel )
} else {
channel . close
false
}
}
override def close ( ) : ChannelGroupFuture = guard withWriteGuard {
if ( open . getAndSet ( false ) ) {
super . close
} else {
throw new IllegalStateException ( "ChannelGroup already closed, cannot add new channel" )
}
}
2011-03-27 22:58:50 +02:00
}