Moved remote-only stuff from akka-cluster to new module akka-remote.
Signed-off-by: Jonas Bonér <jonas@jonasboner.com>
This commit is contained in:
parent
702d59601e
commit
38c2fe1894
21 changed files with 14217 additions and 1 deletions
10788
akka-remote/src/main/java/akka/cluster/RemoteProtocol.java
Normal file
10788
akka-remote/src/main/java/akka/cluster/RemoteProtocol.java
Normal file
File diff suppressed because it is too large
Load diff
230
akka-remote/src/main/protocol/RemoteProtocol.proto
Normal file
230
akka-remote/src/main/protocol/RemoteProtocol.proto
Normal file
|
|
@ -0,0 +1,230 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
option java_package = "akka.cluster";
|
||||
option optimize_for = SPEED;
|
||||
|
||||
/******************************************
|
||||
Compile with:
|
||||
cd ./akka-cluster/src/main/protocol
|
||||
protoc RemoteProtocol.proto --java_out ../java
|
||||
*******************************************/
|
||||
|
||||
message AkkaRemoteProtocol {
|
||||
optional RemoteMessageProtocol message = 1;
|
||||
optional RemoteControlProtocol instruction = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a remote message.
|
||||
*/
|
||||
message RemoteMessageProtocol {
|
||||
required UuidProtocol uuid = 1;
|
||||
required ActorInfoProtocol actorInfo = 2;
|
||||
required bool oneWay = 3;
|
||||
optional MessageProtocol message = 4;
|
||||
optional ExceptionProtocol exception = 5;
|
||||
optional UuidProtocol supervisorUuid = 6;
|
||||
optional RemoteActorRefProtocol sender = 7;
|
||||
repeated MetadataEntryProtocol metadata = 8;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines some control messages for the remoting
|
||||
*/
|
||||
message RemoteControlProtocol {
|
||||
optional string cookie = 1;
|
||||
required CommandType commandType = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the type of the RemoteControlProtocol command type
|
||||
*/
|
||||
enum CommandType {
|
||||
CONNECT = 1;
|
||||
SHUTDOWN = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the type of the ReplicationStorage
|
||||
*/
|
||||
enum ReplicationStorageType {
|
||||
TRANSIENT = 1;
|
||||
TRANSACTION_LOG = 2;
|
||||
DATA_GRID = 3;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the type of the ReplicationStrategy
|
||||
*/
|
||||
enum ReplicationStrategyType {
|
||||
WRITE_THROUGH = 1;
|
||||
WRITE_BEHIND = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a remote ActorRef that "remembers" and uses its original Actor instance
|
||||
* on the original node.
|
||||
*/
|
||||
message RemoteActorRefProtocol {
|
||||
required string address = 1;
|
||||
required bytes inetSocketAddress = 2;
|
||||
optional uint64 timeout = 3;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a fully serialized remote ActorRef (with serialized Actor instance)
|
||||
* that is about to be instantiated on the remote node. It is fully disconnected
|
||||
* from its original host.
|
||||
*/
|
||||
message SerializedActorRefProtocol {
|
||||
required UuidProtocol uuid = 1;
|
||||
required string address = 2;
|
||||
required string actorClassname = 3;
|
||||
optional bytes actorInstance = 4;
|
||||
optional string serializerClassname = 5;
|
||||
optional uint64 timeout = 6;
|
||||
optional uint64 receiveTimeout = 7;
|
||||
optional LifeCycleProtocol lifeCycle = 8;
|
||||
optional RemoteActorRefProtocol supervisor = 9;
|
||||
optional bytes hotswapStack = 10;
|
||||
optional ReplicationStorageType replicationStorage = 11;
|
||||
optional ReplicationStrategyType replicationStrategy = 12;
|
||||
repeated RemoteMessageProtocol messages = 13;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a fully serialized remote ActorRef (with serialized typed actor instance)
|
||||
* that is about to be instantiated on the remote node. It is fully disconnected
|
||||
* from its original host.
|
||||
*/
|
||||
message SerializedTypedActorRefProtocol {
|
||||
required SerializedActorRefProtocol actorRef = 1;
|
||||
required string interfaceName = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a message.
|
||||
*/
|
||||
message MessageProtocol {
|
||||
required bytes message = 1;
|
||||
optional bytes messageManifest = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the actor info.
|
||||
*/
|
||||
message ActorInfoProtocol {
|
||||
required UuidProtocol uuid = 1;
|
||||
required uint64 timeout = 2;
|
||||
optional string address = 3;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a UUID.
|
||||
*/
|
||||
message UuidProtocol {
|
||||
required uint64 high = 1;
|
||||
required uint64 low = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a meta data entry.
|
||||
*/
|
||||
message MetadataEntryProtocol {
|
||||
required string key = 1;
|
||||
required bytes value = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the serialization scheme used to serialize the message and/or Actor instance.
|
||||
*/
|
||||
enum SerializationSchemeType {
|
||||
JAVA = 1;
|
||||
SBINARY = 2;
|
||||
SCALA_JSON = 3;
|
||||
JAVA_JSON = 4;
|
||||
PROTOBUF = 5;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the type of the life-cycle of a supervised Actor.
|
||||
*/
|
||||
enum LifeCycleType {
|
||||
PERMANENT = 1;
|
||||
TEMPORARY = 2;
|
||||
}
|
||||
|
||||
/*
|
||||
enum DispatcherType {
|
||||
GLOBAL_EVENT_EXECUTOR_BASED = 1;
|
||||
GLOBAL_REACTOR_SINGLE_THREAD_BASED = 2;
|
||||
GLOBAL_REACTOR_THREAD_POOL_BASED = 3;
|
||||
EVENT_EXECUTOR_BASED = 4;
|
||||
THREAD_BASED = 5;
|
||||
}
|
||||
*/
|
||||
|
||||
/**
|
||||
* Defines the life-cycle of a supervised Actor.
|
||||
*/
|
||||
message LifeCycleProtocol {
|
||||
required LifeCycleType lifeCycle = 1;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines a remote address.
|
||||
*/
|
||||
message AddressProtocol {
|
||||
required string hostname = 1;
|
||||
required uint32 port = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines an exception.
|
||||
*/
|
||||
message ExceptionProtocol {
|
||||
required string classname = 1;
|
||||
required string message = 2;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the remote daemon message.
|
||||
*/
|
||||
message RemoteDaemonMessageProtocol {
|
||||
required RemoteDaemonMessageType messageType = 1;
|
||||
optional UuidProtocol actorUuid = 2;
|
||||
optional string actorAddress = 3;
|
||||
optional bytes payload = 5;
|
||||
optional UuidProtocol replicateActorFromUuid = 6;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the remote daemon message type.
|
||||
*/
|
||||
enum RemoteDaemonMessageType {
|
||||
STOP = 1;
|
||||
USE = 2;
|
||||
RELEASE = 3;
|
||||
MAKE_AVAILABLE = 4;
|
||||
MAKE_UNAVAILABLE = 5;
|
||||
DISCONNECT = 6;
|
||||
RECONNECT = 7;
|
||||
RESIGN = 8;
|
||||
FAIL_OVER_CONNECTIONS = 9;
|
||||
FUNCTION_FUN0_UNIT = 10;
|
||||
FUNCTION_FUN0_ANY = 11;
|
||||
FUNCTION_FUN1_ARG_UNIT = 12;
|
||||
FUNCTION_FUN1_ARG_ANY = 13;
|
||||
}
|
||||
|
||||
/**
|
||||
* Defines the durable mailbox message.
|
||||
*/
|
||||
message DurableMailboxMessageProtocol {
|
||||
required string ownerActorAddress= 1;
|
||||
optional string senderActorAddress = 2;
|
||||
optional UuidProtocol futureUuid = 3;
|
||||
required bytes message = 4;
|
||||
}
|
||||
|
|
@ -0,0 +1,42 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import akka.actor.{ Actor, BootableActorLoaderService }
|
||||
import akka.util.{ ReflectiveAccess, Bootable }
|
||||
import akka.event.EventHandler
|
||||
|
||||
/**
|
||||
* This bundle/service is responsible for booting up and shutting down the remote actors facility.
|
||||
* <p/>
|
||||
* It is used in Kernel.
|
||||
*/
|
||||
trait BootableRemoteActorService extends Bootable {
|
||||
self: BootableActorLoaderService ⇒
|
||||
|
||||
protected lazy val remoteServerThread = new Thread(new Runnable() {
|
||||
def run = Actor.remote.start(self.applicationLoader.getOrElse(null)) //Use config host/port
|
||||
}, "Akka RemoteModule Service")
|
||||
|
||||
def startRemoteService() { remoteServerThread.start() }
|
||||
|
||||
abstract override def onLoad() {
|
||||
if (ReflectiveAccess.ClusterModule.isEnabled && RemoteServerSettings.isRemotingEnabled) {
|
||||
EventHandler.info(this, "Initializing Remote Actors Service...")
|
||||
startRemoteService()
|
||||
EventHandler.info(this, "Remote Actors Service initialized")
|
||||
}
|
||||
super.onLoad()
|
||||
}
|
||||
|
||||
abstract override def onUnload() {
|
||||
EventHandler.info(this, "Shutting down Remote Actors Service")
|
||||
|
||||
Actor.remote.shutdown()
|
||||
if (remoteServerThread.isAlive) remoteServerThread.join(1000)
|
||||
EventHandler.info(this, "Remote Actors Service has been shut down")
|
||||
super.onUnload()
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,32 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import akka.cluster.RemoteProtocol._
|
||||
import akka.serialization.Serialization
|
||||
|
||||
import com.google.protobuf.ByteString
|
||||
|
||||
object MessageSerializer {
|
||||
|
||||
def deserialize(messageProtocol: MessageProtocol, classLoader: Option[ClassLoader] = None): AnyRef = {
|
||||
val clazz = loadManifest(classLoader, messageProtocol)
|
||||
Serialization.deserialize(messageProtocol.getMessage.toByteArray,
|
||||
clazz, classLoader).fold(x ⇒ throw x, o ⇒ o)
|
||||
}
|
||||
|
||||
def serialize(message: AnyRef): MessageProtocol = {
|
||||
val builder = MessageProtocol.newBuilder
|
||||
val bytes = Serialization.serialize(message).fold(x ⇒ throw x, b ⇒ b)
|
||||
builder.setMessage(ByteString.copyFrom(bytes))
|
||||
builder.setMessageManifest(ByteString.copyFromUtf8(message.getClass.getName))
|
||||
builder.build
|
||||
}
|
||||
|
||||
private def loadManifest(classLoader: Option[ClassLoader], messageProtocol: MessageProtocol): Class[_] = {
|
||||
val manifest = messageProtocol.getMessageManifest.toStringUtf8
|
||||
classLoader map (_.loadClass(manifest)) getOrElse (Class.forName(manifest))
|
||||
}
|
||||
}
|
||||
67
akka-remote/src/main/scala/akka/cluster/RemoteConfig.scala
Normal file
67
akka-remote/src/main/scala/akka/cluster/RemoteConfig.scala
Normal file
|
|
@ -0,0 +1,67 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import akka.util.Duration
|
||||
import akka.config.Config._
|
||||
import akka.config.ConfigurationException
|
||||
|
||||
object RemoteClientSettings {
|
||||
val SECURE_COOKIE: Option[String] = config.getString("akka.cluster.secure-cookie", "") match {
|
||||
case "" ⇒ None
|
||||
case cookie ⇒ Some(cookie)
|
||||
}
|
||||
|
||||
val RECONNECTION_TIME_WINDOW = Duration(config.getInt("akka.cluster.client.reconnection-time-window", 600), TIME_UNIT).toMillis
|
||||
val READ_TIMEOUT = Duration(config.getInt("akka.cluster.client.read-timeout", 3600), TIME_UNIT)
|
||||
val RECONNECT_DELAY = Duration(config.getInt("akka.cluster.client.reconnect-delay", 5), TIME_UNIT)
|
||||
val REAP_FUTURES_DELAY = Duration(config.getInt("akka.cluster.client.reap-futures-delay", 5), TIME_UNIT)
|
||||
val MESSAGE_FRAME_SIZE = config.getInt("akka.cluster.client.message-frame-size", 1048576)
|
||||
}
|
||||
|
||||
object RemoteServerSettings {
|
||||
val isRemotingEnabled = config.getList("akka.enabled-modules").exists(_ == "cluster")
|
||||
val MESSAGE_FRAME_SIZE = config.getInt("akka.cluster.server.message-frame-size", 1048576)
|
||||
val SECURE_COOKIE = config.getString("akka.cluster.secure-cookie")
|
||||
val REQUIRE_COOKIE = {
|
||||
val requireCookie = config.getBool("akka.cluster.server.require-cookie", false)
|
||||
if (isRemotingEnabled && requireCookie && SECURE_COOKIE.isEmpty) throw new ConfigurationException(
|
||||
"Configuration option 'akka.cluster.server.require-cookie' is turned on but no secure cookie is defined in 'akka.cluster.secure-cookie'.")
|
||||
requireCookie
|
||||
}
|
||||
|
||||
val UNTRUSTED_MODE = config.getBool("akka.cluster.server.untrusted-mode", false)
|
||||
val PORT = config.getInt("akka.cluster.server.port", 2552)
|
||||
val CONNECTION_TIMEOUT = Duration(config.getInt("akka.cluster.server.connection-timeout", 100), TIME_UNIT)
|
||||
val COMPRESSION_SCHEME = config.getString("akka.cluster.compression-scheme", "")
|
||||
val ZLIB_COMPRESSION_LEVEL = {
|
||||
val level = config.getInt("akka.cluster.zlib-compression-level", 6)
|
||||
if (level < 1 && level > 9) throw new IllegalArgumentException(
|
||||
"zlib compression level has to be within 1-9, with 1 being fastest and 9 being the most compressed")
|
||||
level
|
||||
}
|
||||
|
||||
val BACKLOG = config.getInt("akka.cluster.server.backlog", 4096)
|
||||
|
||||
val EXECUTION_POOL_KEEPALIVE = Duration(config.getInt("akka.cluster.server.execution-pool-keepalive", 60), TIME_UNIT)
|
||||
|
||||
val EXECUTION_POOL_SIZE = {
|
||||
val sz = config.getInt("akka.cluster.server.execution-pool-size", 16)
|
||||
if (sz < 1) throw new IllegalArgumentException("akka.cluster.server.execution-pool-size is less than 1")
|
||||
sz
|
||||
}
|
||||
|
||||
val MAX_CHANNEL_MEMORY_SIZE = {
|
||||
val sz = config.getInt("akka.cluster.server.max-channel-memory-size", 0)
|
||||
if (sz < 0) throw new IllegalArgumentException("akka.cluster.server.max-channel-memory-size is less than 0")
|
||||
sz
|
||||
}
|
||||
|
||||
val MAX_TOTAL_MEMORY_SIZE = {
|
||||
val sz = config.getInt("akka.cluster.server.max-total-memory-size", 0)
|
||||
if (sz < 0) throw new IllegalArgumentException("akka.cluster.server.max-total-memory-size is less than 0")
|
||||
sz
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,393 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import akka.actor._
|
||||
import Actor._
|
||||
import akka.cluster._
|
||||
import akka.routing._
|
||||
import akka.event.EventHandler
|
||||
import akka.dispatch.{ Dispatchers, Future, PinnedDispatcher }
|
||||
import akka.util.{ ListenerManagement, Duration }
|
||||
|
||||
import scala.collection.immutable.Map
|
||||
import scala.collection.mutable
|
||||
import scala.annotation.tailrec
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
import java.util.concurrent.atomic.AtomicReference
|
||||
import System.{ currentTimeMillis ⇒ newTimestamp }
|
||||
|
||||
/**
|
||||
* Holds error event channel Actor instance and provides API for channel listener management.
|
||||
*/
|
||||
object RemoteFailureDetector {
|
||||
|
||||
private sealed trait RemoteFailureDetectorChannelEvent
|
||||
|
||||
private case class Register(listener: RemoteFailureListener, connectionAddress: InetSocketAddress)
|
||||
extends RemoteFailureDetectorChannelEvent
|
||||
|
||||
private case class Unregister(listener: RemoteFailureListener, connectionAddress: InetSocketAddress)
|
||||
extends RemoteFailureDetectorChannelEvent
|
||||
|
||||
private[akka] val channel = actorOf(Props(new Channel).copy(dispatcher = new PinnedDispatcher(), localOnly = true))
|
||||
|
||||
def register(listener: RemoteFailureListener, connectionAddress: InetSocketAddress) =
|
||||
channel ! Register(listener, connectionAddress)
|
||||
|
||||
def unregister(listener: RemoteFailureListener, connectionAddress: InetSocketAddress) =
|
||||
channel ! Unregister(listener, connectionAddress)
|
||||
|
||||
private class Channel extends Actor {
|
||||
|
||||
val listeners = new mutable.HashMap[InetSocketAddress, mutable.Set[RemoteFailureListener]]() {
|
||||
override def default(k: InetSocketAddress) = mutable.Set.empty[RemoteFailureListener]
|
||||
}
|
||||
|
||||
def receive = {
|
||||
case event: RemoteClientLifeCycleEvent ⇒
|
||||
listeners(event.remoteAddress) foreach (_ notify event)
|
||||
|
||||
case event: RemoteServerLifeCycleEvent ⇒ // FIXME handle RemoteServerLifeCycleEvent
|
||||
|
||||
case Register(listener, connectionAddress) ⇒
|
||||
listeners(connectionAddress) += listener
|
||||
|
||||
case Unregister(listener, connectionAddress) ⇒
|
||||
listeners(connectionAddress) -= listener
|
||||
|
||||
case _ ⇒ //ignore other
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Base class for remote failure detection management.
|
||||
*/
|
||||
abstract class RemoteFailureDetectorBase(initialConnections: Map[InetSocketAddress, ActorRef])
|
||||
extends FailureDetector
|
||||
with RemoteFailureListener {
|
||||
|
||||
// import ClusterActorRef._
|
||||
|
||||
type T <: AnyRef
|
||||
|
||||
protected case class State(
|
||||
version: Long,
|
||||
connections: Map[InetSocketAddress, ActorRef],
|
||||
meta: T = null.asInstanceOf[T])
|
||||
extends VersionedIterable[ActorRef] {
|
||||
def iterable: Iterable[ActorRef] = connections.values
|
||||
}
|
||||
|
||||
protected val state: AtomicReference[State] = {
|
||||
val ref = new AtomicReference[State]
|
||||
ref set newState()
|
||||
ref
|
||||
}
|
||||
|
||||
/**
|
||||
* State factory. To be defined by subclass that wants to add extra info in the 'meta: Option[T]' field.
|
||||
*/
|
||||
protected def newState(): State
|
||||
|
||||
/**
|
||||
* Returns true if the 'connection' is considered available.
|
||||
*
|
||||
* To be implemented by subclass.
|
||||
*/
|
||||
def isAvailable(connectionAddress: InetSocketAddress): Boolean
|
||||
|
||||
/**
|
||||
* Records a successful connection.
|
||||
*
|
||||
* To be implemented by subclass.
|
||||
*/
|
||||
def recordSuccess(connectionAddress: InetSocketAddress, timestamp: Long)
|
||||
|
||||
/**
|
||||
* Records a failed connection.
|
||||
*
|
||||
* To be implemented by subclass.
|
||||
*/
|
||||
def recordFailure(connectionAddress: InetSocketAddress, timestamp: Long)
|
||||
|
||||
def version: Long = state.get.version
|
||||
|
||||
def versionedIterable = state.get
|
||||
|
||||
def size: Int = state.get.connections.size
|
||||
|
||||
def connections: Map[InetSocketAddress, ActorRef] = state.get.connections
|
||||
|
||||
def stopAll() {
|
||||
state.get.iterable foreach (_.stop()) // shut down all remote connections
|
||||
}
|
||||
|
||||
@tailrec
|
||||
final def failOver(from: InetSocketAddress, to: InetSocketAddress) {
|
||||
EventHandler.debug(this, "ClusterActorRef failover from [%s] to [%s]".format(from, to))
|
||||
|
||||
val oldState = state.get
|
||||
var changed = false
|
||||
|
||||
val newMap = oldState.connections map {
|
||||
case (`from`, actorRef) ⇒
|
||||
changed = true
|
||||
//actorRef.stop()
|
||||
(to, createRemoteActorRef(actorRef.address, to))
|
||||
case other ⇒ other
|
||||
}
|
||||
|
||||
if (changed) {
|
||||
//there was a state change, so we are now going to update the state.
|
||||
val newState = oldState copy (version = oldState.version + 1, connections = newMap)
|
||||
|
||||
//if we are not able to update, the state, we are going to try again.
|
||||
if (!state.compareAndSet(oldState, newState)) failOver(from, to)
|
||||
}
|
||||
}
|
||||
|
||||
@tailrec
|
||||
final def remove(faultyConnection: ActorRef) {
|
||||
EventHandler.debug(this, "ClusterActorRef remove [%s]".format(faultyConnection.uuid))
|
||||
|
||||
val oldState = state.get()
|
||||
var changed = false
|
||||
|
||||
//remote the faultyConnection from the clustered-connections.
|
||||
var newConnections = Map.empty[InetSocketAddress, ActorRef]
|
||||
oldState.connections.keys foreach { address ⇒
|
||||
val actorRef: ActorRef = oldState.connections.get(address).get
|
||||
if (actorRef ne faultyConnection) {
|
||||
newConnections = newConnections + ((address, actorRef))
|
||||
} else {
|
||||
changed = true
|
||||
}
|
||||
}
|
||||
|
||||
if (changed) {
|
||||
//one or more occurrances of the actorRef were removed, so we need to update the state.
|
||||
val newState = oldState copy (version = oldState.version + 1, connections = newConnections)
|
||||
|
||||
//if we are not able to update the state, we just try again.
|
||||
if (!state.compareAndSet(oldState, newState)) remove(faultyConnection)
|
||||
}
|
||||
}
|
||||
|
||||
private[cluster] def createRemoteActorRef(actorAddress: String, inetSocketAddress: InetSocketAddress) = {
|
||||
RemoteActorRef(inetSocketAddress, actorAddress, Actor.TIMEOUT, None)
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Simple failure detector that removes the failing connection permanently on first error.
|
||||
*/
|
||||
class RemoveConnectionOnFirstFailureRemoteFailureDetector(
|
||||
initialConnections: Map[InetSocketAddress, ActorRef])
|
||||
extends RemoteFailureDetectorBase(initialConnections) {
|
||||
|
||||
protected def newState() = State(Long.MinValue, initialConnections)
|
||||
|
||||
def isAvailable(connectionAddress: InetSocketAddress): Boolean = connections.get(connectionAddress).isDefined
|
||||
|
||||
def recordSuccess(connectionAddress: InetSocketAddress, timestamp: Long) {}
|
||||
|
||||
def recordFailure(connectionAddress: InetSocketAddress, timestamp: Long) {}
|
||||
|
||||
override def remoteClientWriteFailed(
|
||||
request: AnyRef, cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
removeConnection(connectionAddress)
|
||||
}
|
||||
|
||||
override def remoteClientError(cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
removeConnection(connectionAddress)
|
||||
}
|
||||
|
||||
override def remoteClientDisconnected(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
removeConnection(connectionAddress)
|
||||
}
|
||||
|
||||
override def remoteClientShutdown(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
removeConnection(connectionAddress)
|
||||
}
|
||||
|
||||
private def removeConnection(connectionAddress: InetSocketAddress) =
|
||||
connections.get(connectionAddress) foreach { conn ⇒ remove(conn) }
|
||||
}
|
||||
|
||||
/**
|
||||
* Failure detector that bans the failing connection for 'timeToBan: Duration' and will try to use the connection
|
||||
* again after the ban period have expired.
|
||||
*/
|
||||
class BannagePeriodFailureDetector(
|
||||
initialConnections: Map[InetSocketAddress, ActorRef],
|
||||
timeToBan: Duration)
|
||||
extends RemoteFailureDetectorBase(initialConnections) {
|
||||
|
||||
// FIXME considering adding a Scheduler event to notify the BannagePeriodFailureDetector unban the banned connection after the timeToBan have exprired
|
||||
|
||||
type T = Map[InetSocketAddress, BannedConnection]
|
||||
|
||||
case class BannedConnection(bannedSince: Long, connection: ActorRef)
|
||||
|
||||
val timeToBanInMillis = timeToBan.toMillis
|
||||
|
||||
protected def newState() =
|
||||
State(Long.MinValue, initialConnections, Map.empty[InetSocketAddress, BannedConnection])
|
||||
|
||||
private def removeConnection(connectionAddress: InetSocketAddress) =
|
||||
connections.get(connectionAddress) foreach { conn ⇒ remove(conn) }
|
||||
|
||||
// ===================================================================================
|
||||
// FailureDetector callbacks
|
||||
// ===================================================================================
|
||||
|
||||
def isAvailable(connectionAddress: InetSocketAddress): Boolean = connections.get(connectionAddress).isDefined
|
||||
|
||||
@tailrec
|
||||
final def recordSuccess(connectionAddress: InetSocketAddress, timestamp: Long) {
|
||||
val oldState = state.get
|
||||
val bannedConnection = oldState.meta.get(connectionAddress)
|
||||
|
||||
if (bannedConnection.isDefined) {
|
||||
val BannedConnection(bannedSince, connection) = bannedConnection.get
|
||||
val currentlyBannedFor = newTimestamp - bannedSince
|
||||
|
||||
if (currentlyBannedFor > timeToBanInMillis) {
|
||||
// ban time has expired - add connection to available connections
|
||||
val newConnections = oldState.connections + (connectionAddress -> connection)
|
||||
val newBannedConnections = oldState.meta - connectionAddress
|
||||
|
||||
val newState = oldState copy (version = oldState.version + 1,
|
||||
connections = newConnections,
|
||||
meta = newBannedConnections)
|
||||
|
||||
if (!state.compareAndSet(oldState, newState)) recordSuccess(connectionAddress, timestamp)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
@tailrec
|
||||
final def recordFailure(connectionAddress: InetSocketAddress, timestamp: Long) {
|
||||
val oldState = state.get
|
||||
val connection = oldState.connections.get(connectionAddress)
|
||||
|
||||
if (connection.isDefined) {
|
||||
val newConnections = oldState.connections - connectionAddress
|
||||
val bannedConnection = BannedConnection(timestamp, connection.get)
|
||||
val newBannedConnections = oldState.meta + (connectionAddress -> bannedConnection)
|
||||
|
||||
val newState = oldState copy (version = oldState.version + 1,
|
||||
connections = newConnections,
|
||||
meta = newBannedConnections)
|
||||
|
||||
if (!state.compareAndSet(oldState, newState)) recordFailure(connectionAddress, timestamp)
|
||||
}
|
||||
}
|
||||
|
||||
// ===================================================================================
|
||||
// RemoteFailureListener callbacks
|
||||
// ===================================================================================
|
||||
|
||||
override def remoteClientStarted(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordSuccess(connectionAddress, newTimestamp)
|
||||
}
|
||||
|
||||
override def remoteClientConnected(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordSuccess(connectionAddress, newTimestamp)
|
||||
}
|
||||
|
||||
override def remoteClientWriteFailed(
|
||||
request: AnyRef, cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordFailure(connectionAddress, newTimestamp)
|
||||
}
|
||||
|
||||
override def remoteClientError(cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordFailure(connectionAddress, newTimestamp)
|
||||
}
|
||||
|
||||
override def remoteClientDisconnected(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordFailure(connectionAddress, newTimestamp)
|
||||
}
|
||||
|
||||
override def remoteClientShutdown(client: RemoteClientModule, connectionAddress: InetSocketAddress) {
|
||||
recordFailure(connectionAddress, newTimestamp)
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Failure detector that uses the Circuit Breaker pattern to detect and recover from failing connections.
|
||||
*
|
||||
* class CircuitBreakerRemoteFailureListener(initialConnections: Map[InetSocketAddress, ActorRef])
|
||||
* extends RemoteFailureDetectorBase(initialConnections) {
|
||||
*
|
||||
* def newState() = State(Long.MinValue, initialConnections, None)
|
||||
*
|
||||
* def isAvailable(connectionAddress: InetSocketAddress): Boolean = connections.get(connectionAddress).isDefined
|
||||
*
|
||||
* def recordSuccess(connectionAddress: InetSocketAddress, timestamp: Long) {}
|
||||
*
|
||||
* def recordFailure(connectionAddress: InetSocketAddress, timestamp: Long) {}
|
||||
*
|
||||
* // FIXME implement CircuitBreakerRemoteFailureListener
|
||||
* }
|
||||
*/
|
||||
|
||||
/**
|
||||
* Base trait for remote failure event listener.
|
||||
*/
|
||||
trait RemoteFailureListener {
|
||||
|
||||
final private[akka] def notify(event: RemoteLifeCycleEvent) = event match {
|
||||
case RemoteClientStarted(client, connectionAddress) ⇒
|
||||
remoteClientStarted(client, connectionAddress)
|
||||
|
||||
case RemoteClientConnected(client, connectionAddress) ⇒
|
||||
remoteClientConnected(client, connectionAddress)
|
||||
|
||||
case RemoteClientWriteFailed(request, cause, client, connectionAddress) ⇒
|
||||
remoteClientWriteFailed(request, cause, client, connectionAddress)
|
||||
|
||||
case RemoteClientError(cause, client, connectionAddress) ⇒
|
||||
remoteClientError(cause, client, connectionAddress)
|
||||
|
||||
case RemoteClientDisconnected(client, connectionAddress) ⇒
|
||||
remoteClientDisconnected(client, connectionAddress)
|
||||
|
||||
case RemoteClientShutdown(client, connectionAddress) ⇒
|
||||
remoteClientShutdown(client, connectionAddress)
|
||||
|
||||
case RemoteServerWriteFailed(request, cause, server, clientAddress) ⇒
|
||||
remoteServerWriteFailed(request, cause, server, clientAddress)
|
||||
|
||||
case RemoteServerError(cause, server) ⇒
|
||||
remoteServerError(cause, server)
|
||||
|
||||
case RemoteServerShutdown(server) ⇒
|
||||
remoteServerShutdown(server)
|
||||
}
|
||||
|
||||
def remoteClientStarted(client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteClientConnected(client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteClientWriteFailed(
|
||||
request: AnyRef, cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteClientError(cause: Throwable, client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteClientDisconnected(client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteClientShutdown(client: RemoteClientModule, connectionAddress: InetSocketAddress) {}
|
||||
|
||||
def remoteServerWriteFailed(
|
||||
request: AnyRef, cause: Throwable, server: RemoteServerModule, clientAddress: Option[InetSocketAddress]) {}
|
||||
|
||||
def remoteServerError(cause: Throwable, server: RemoteServerModule) {}
|
||||
|
||||
def remoteServerShutdown(server: RemoteServerModule) {}
|
||||
}
|
||||
File diff suppressed because it is too large
Load diff
13
akka-remote/src/main/scala/akka/package.scala
Normal file
13
akka-remote/src/main/scala/akka/package.scala
Normal file
|
|
@ -0,0 +1,13 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka
|
||||
|
||||
package object serialization {
|
||||
type JsValue = _root_.dispatch.json.JsValue
|
||||
val JsValue = _root_.dispatch.json.JsValue
|
||||
val Js = _root_.dispatch.json.Js
|
||||
val JsonSerialization = sjson.json.JsonSerialization
|
||||
val DefaultProtocol = sjson.json.DefaultProtocol
|
||||
}
|
||||
|
|
@ -0,0 +1,21 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.serialization
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
object Compression {
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
object LZF {
|
||||
import voldemort.store.compress.lzf._
|
||||
def compress(bytes: Array[Byte]): Array[Byte] = LZFEncoder encode bytes
|
||||
def uncompress(bytes: Array[Byte]): Array[Byte] = LZFDecoder decode bytes
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,361 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.serialization
|
||||
|
||||
import akka.config.Supervision._
|
||||
import akka.actor.{ uuidFrom, newUuid }
|
||||
import akka.actor._
|
||||
import DeploymentConfig._
|
||||
import akka.dispatch.MessageInvocation
|
||||
import akka.util.{ ReflectiveAccess, Duration }
|
||||
import akka.cluster.{ RemoteClientSettings, MessageSerializer }
|
||||
import akka.cluster.RemoteProtocol
|
||||
import RemoteProtocol._
|
||||
|
||||
import scala.collection.immutable.Stack
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
|
||||
import com.google.protobuf.ByteString
|
||||
|
||||
import com.eaio.uuid.UUID
|
||||
import akka.event.EventHandler
|
||||
import java.util.{ LinkedList, Collections }
|
||||
|
||||
/**
|
||||
* Module for local actor serialization.
|
||||
*/
|
||||
object ActorSerialization {
|
||||
implicit val defaultSerializer = akka.serialization.JavaSerializer // Format.Default
|
||||
|
||||
def fromBinary[T <: Actor](bytes: Array[Byte], homeAddress: InetSocketAddress): ActorRef =
|
||||
fromBinaryToLocalActorRef(bytes, None, Some(homeAddress))
|
||||
|
||||
def fromBinary[T <: Actor](bytes: Array[Byte], uuid: UUID): ActorRef =
|
||||
fromBinaryToLocalActorRef(bytes, Some(uuid), None)
|
||||
|
||||
def fromBinary[T <: Actor](bytes: Array[Byte]): ActorRef =
|
||||
fromBinaryToLocalActorRef(bytes, None, None)
|
||||
|
||||
def toBinary[T <: Actor](
|
||||
a: ActorRef,
|
||||
serializeMailBox: Boolean = true,
|
||||
replicationScheme: ReplicationScheme = Transient): Array[Byte] =
|
||||
toSerializedActorRefProtocol(a, serializeMailBox, replicationScheme).toByteArray
|
||||
|
||||
// wrapper for implicits to be used by Java
|
||||
def fromBinaryJ[T <: Actor](bytes: Array[Byte]): ActorRef =
|
||||
fromBinary(bytes)
|
||||
|
||||
// wrapper for implicits to be used by Java
|
||||
def toBinaryJ[T <: Actor](
|
||||
a: ActorRef,
|
||||
srlMailBox: Boolean,
|
||||
replicationScheme: ReplicationScheme): Array[Byte] =
|
||||
toBinary(a, srlMailBox, replicationScheme)
|
||||
|
||||
private[akka] def toSerializedActorRefProtocol[T <: Actor](
|
||||
actorRef: ActorRef,
|
||||
serializeMailBox: Boolean,
|
||||
replicationScheme: ReplicationScheme): SerializedActorRefProtocol = {
|
||||
|
||||
val localRef: Option[LocalActorRef] = actorRef match {
|
||||
case l: LocalActorRef ⇒ Some(l)
|
||||
case _ ⇒ None
|
||||
}
|
||||
|
||||
val lifeCycleProtocol: Option[LifeCycleProtocol] = {
|
||||
actorRef.lifeCycle match {
|
||||
case Permanent ⇒ Some(LifeCycleProtocol.newBuilder.setLifeCycle(LifeCycleType.PERMANENT).build)
|
||||
case Temporary ⇒ Some(LifeCycleProtocol.newBuilder.setLifeCycle(LifeCycleType.TEMPORARY).build)
|
||||
case UndefinedLifeCycle ⇒ None //No need to send the undefined lifecycle over the wire //builder.setLifeCycle(LifeCycleType.UNDEFINED)
|
||||
}
|
||||
}
|
||||
|
||||
val builder = SerializedActorRefProtocol.newBuilder
|
||||
.setUuid(UuidProtocol.newBuilder.setHigh(actorRef.uuid.getTime).setLow(actorRef.uuid.getClockSeqAndNode).build)
|
||||
.setAddress(actorRef.address)
|
||||
.setTimeout(actorRef.timeout)
|
||||
|
||||
if (localRef.isDefined)
|
||||
builder.setActorClassname(localRef.get.actorInstance.get.getClass.getName) //TODO FIXME Why is the classname needed anymore?
|
||||
|
||||
replicationScheme match {
|
||||
case _: Transient | Transient ⇒
|
||||
builder.setReplicationStorage(ReplicationStorageType.TRANSIENT)
|
||||
|
||||
case Replication(storage, strategy) ⇒
|
||||
val storageType = storage match {
|
||||
case _: TransactionLog | TransactionLog ⇒ ReplicationStorageType.TRANSACTION_LOG
|
||||
case _: DataGrid | DataGrid ⇒ ReplicationStorageType.DATA_GRID
|
||||
}
|
||||
builder.setReplicationStorage(storageType)
|
||||
|
||||
val strategyType = strategy match {
|
||||
case _: WriteBehind ⇒ ReplicationStrategyType.WRITE_BEHIND
|
||||
case _: WriteThrough ⇒ ReplicationStrategyType.WRITE_THROUGH
|
||||
}
|
||||
builder.setReplicationStrategy(strategyType)
|
||||
}
|
||||
|
||||
lifeCycleProtocol.foreach(builder.setLifeCycle(_))
|
||||
actorRef.supervisor.foreach(s ⇒ builder.setSupervisor(RemoteActorSerialization.toRemoteActorRefProtocol(s)))
|
||||
|
||||
localRef foreach { l ⇒
|
||||
if (serializeMailBox) {
|
||||
l.mailbox match {
|
||||
case null ⇒ throw new IllegalActorStateException("Can't serialize an actor that has not been started.")
|
||||
case q: java.util.Queue[_] ⇒
|
||||
val l = new scala.collection.mutable.ListBuffer[MessageInvocation]
|
||||
val it = q.iterator
|
||||
while (it.hasNext) l += it.next.asInstanceOf[MessageInvocation]
|
||||
|
||||
l map { m ⇒
|
||||
RemoteActorSerialization.createRemoteMessageProtocolBuilder(
|
||||
Option(m.receiver),
|
||||
Left(actorRef.uuid),
|
||||
actorRef.address,
|
||||
actorRef.timeout,
|
||||
Right(m.message),
|
||||
false,
|
||||
m.channel match {
|
||||
case a: ActorRef ⇒ Some(a)
|
||||
case _ ⇒ None
|
||||
})
|
||||
} foreach {
|
||||
builder.addMessages(_)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
l.receiveTimeout.foreach(builder.setReceiveTimeout(_))
|
||||
val actorInstance = l.actorInstance.get
|
||||
Serialization.serialize(actorInstance.asInstanceOf[T]) match {
|
||||
case Right(bytes) ⇒ builder.setActorInstance(ByteString.copyFrom(bytes))
|
||||
case Left(exception) ⇒ throw new Exception("Error serializing : " + actorInstance.getClass.getName)
|
||||
}
|
||||
val stack = l.hotswap
|
||||
if (!stack.isEmpty)
|
||||
builder.setHotswapStack(ByteString.copyFrom(akka.serialization.JavaSerializer.toBinary(stack)))
|
||||
}
|
||||
|
||||
builder.build
|
||||
}
|
||||
|
||||
private def fromBinaryToLocalActorRef[T <: Actor](
|
||||
bytes: Array[Byte],
|
||||
uuid: Option[UUID],
|
||||
homeAddress: Option[InetSocketAddress]): ActorRef = {
|
||||
val builder = SerializedActorRefProtocol.newBuilder.mergeFrom(bytes)
|
||||
fromProtobufToLocalActorRef(builder.build, uuid, None)
|
||||
}
|
||||
|
||||
private[akka] def fromProtobufToLocalActorRef[T <: Actor](
|
||||
protocol: SerializedActorRefProtocol,
|
||||
overriddenUuid: Option[UUID],
|
||||
loader: Option[ClassLoader]): ActorRef = {
|
||||
|
||||
EventHandler.debug(this, "Deserializing SerializedActorRefProtocol to LocalActorRef:\n%s".format(protocol))
|
||||
|
||||
// import ReplicationStorageType._
|
||||
// import ReplicationStrategyType._
|
||||
// val replicationScheme =
|
||||
// if (protocol.hasReplicationStorage) {
|
||||
// protocol.getReplicationStorage match {
|
||||
// case TRANSIENT ⇒ Transient
|
||||
// case store ⇒
|
||||
// val storage = store match {
|
||||
// case TRANSACTION_LOG ⇒ TransactionLog
|
||||
// case DATA_GRID ⇒ DataGrid
|
||||
// }
|
||||
// val strategy = if (protocol.hasReplicationStrategy) {
|
||||
// protocol.getReplicationStrategy match {
|
||||
// case WRITE_THROUGH ⇒ WriteThrough
|
||||
// case WRITE_BEHIND ⇒ WriteBehind
|
||||
// }
|
||||
// } else throw new IllegalActorStateException(
|
||||
// "Expected replication strategy for replication storage [" + storage + "]")
|
||||
// Replication(storage, strategy)
|
||||
// }
|
||||
// } else Transient
|
||||
|
||||
val storedHotswap =
|
||||
try {
|
||||
Serialization.deserialize(
|
||||
protocol.getHotswapStack.toByteArray,
|
||||
classOf[Stack[PartialFunction[Any, Unit]]],
|
||||
loader) match {
|
||||
case Right(r) ⇒ r.asInstanceOf[Stack[PartialFunction[Any, Unit]]]
|
||||
case Left(ex) ⇒ throw new Exception("Cannot de-serialize hotswapstack")
|
||||
}
|
||||
} catch {
|
||||
case e: Exception ⇒ Stack[PartialFunction[Any, Unit]]()
|
||||
}
|
||||
|
||||
val storedLifeCycle =
|
||||
if (protocol.hasLifeCycle) {
|
||||
protocol.getLifeCycle.getLifeCycle match {
|
||||
case LifeCycleType.PERMANENT ⇒ Permanent
|
||||
case LifeCycleType.TEMPORARY ⇒ Temporary
|
||||
case unknown ⇒ UndefinedLifeCycle
|
||||
}
|
||||
} else UndefinedLifeCycle
|
||||
|
||||
val storedSupervisor =
|
||||
if (protocol.hasSupervisor) Some(RemoteActorSerialization.fromProtobufToRemoteActorRef(protocol.getSupervisor, loader))
|
||||
else None
|
||||
|
||||
val classLoader = loader.getOrElse(this.getClass.getClassLoader)
|
||||
val bytes = protocol.getActorInstance.toByteArray
|
||||
val actorClass = classLoader.loadClass(protocol.getActorClassname)
|
||||
val factory = () ⇒ {
|
||||
Serialization.deserialize(bytes, actorClass, loader) match {
|
||||
case Right(r) ⇒ r.asInstanceOf[Actor]
|
||||
case Left(ex) ⇒ throw new Exception("Cannot de-serialize : " + actorClass)
|
||||
}
|
||||
}
|
||||
|
||||
val actorUuid = overriddenUuid match {
|
||||
case Some(uuid) ⇒ uuid
|
||||
case None ⇒ uuidFrom(protocol.getUuid.getHigh, protocol.getUuid.getLow)
|
||||
}
|
||||
|
||||
val props = Props(creator = factory,
|
||||
timeout = if (protocol.hasTimeout) protocol.getTimeout else Timeout.default,
|
||||
lifeCycle = storedLifeCycle,
|
||||
supervisor = storedSupervisor //TODO what dispatcher should it use?
|
||||
//TODO what faultHandler should it use?
|
||||
//
|
||||
)
|
||||
|
||||
val receiveTimeout = if (protocol.hasReceiveTimeout) Some(protocol.getReceiveTimeout) else None //TODO FIXME, I'm expensive and slow
|
||||
|
||||
val ar = new LocalActorRef(actorUuid, protocol.getAddress, props, receiveTimeout, storedHotswap)
|
||||
|
||||
//Deserialize messages
|
||||
{
|
||||
val iterator = protocol.getMessagesList.iterator()
|
||||
while (iterator.hasNext())
|
||||
ar ! MessageSerializer.deserialize(iterator.next().getMessage, Some(classLoader)) //TODO This is broken, why aren't we preserving the sender?
|
||||
}
|
||||
|
||||
ar
|
||||
}
|
||||
}
|
||||
|
||||
object RemoteActorSerialization {
|
||||
|
||||
/**
|
||||
* Deserializes a byte array (Array[Byte]) into an RemoteActorRef instance.
|
||||
*/
|
||||
def fromBinaryToRemoteActorRef(bytes: Array[Byte]): ActorRef =
|
||||
fromProtobufToRemoteActorRef(RemoteActorRefProtocol.newBuilder.mergeFrom(bytes).build, None)
|
||||
|
||||
/**
|
||||
* Deserializes a byte array (Array[Byte]) into an RemoteActorRef instance.
|
||||
*/
|
||||
def fromBinaryToRemoteActorRef(bytes: Array[Byte], loader: ClassLoader): ActorRef =
|
||||
fromProtobufToRemoteActorRef(RemoteActorRefProtocol.newBuilder.mergeFrom(bytes).build, Some(loader))
|
||||
|
||||
/**
|
||||
* Deserializes a RemoteActorRefProtocol Protocol Buffers (protobuf) Message into an RemoteActorRef instance.
|
||||
*/
|
||||
private[akka] def fromProtobufToRemoteActorRef(protocol: RemoteActorRefProtocol, loader: Option[ClassLoader]): ActorRef = {
|
||||
EventHandler.debug(this, "Deserializing RemoteActorRefProtocol to RemoteActorRef:\n %s".format(protocol))
|
||||
|
||||
val ref = RemoteActorRef(
|
||||
JavaSerializer.fromBinary(protocol.getInetSocketAddress.toByteArray, Some(classOf[InetSocketAddress]), loader).asInstanceOf[InetSocketAddress],
|
||||
protocol.getAddress,
|
||||
protocol.getTimeout,
|
||||
loader)
|
||||
|
||||
EventHandler.debug(this, "Newly deserialized RemoteActorRef has uuid: %s".format(ref.uuid))
|
||||
|
||||
ref
|
||||
}
|
||||
|
||||
/**
|
||||
* Serializes the ActorRef instance into a Protocol Buffers (protobuf) Message.
|
||||
*/
|
||||
def toRemoteActorRefProtocol(actor: ActorRef): RemoteActorRefProtocol = {
|
||||
val remoteAddress = actor match {
|
||||
case ar: RemoteActorRef ⇒
|
||||
ar.remoteAddress
|
||||
case ar: LocalActorRef ⇒
|
||||
Actor.remote.registerByUuid(ar)
|
||||
ReflectiveAccess.RemoteModule.configDefaultAddress
|
||||
case _ ⇒
|
||||
ReflectiveAccess.RemoteModule.configDefaultAddress
|
||||
}
|
||||
|
||||
EventHandler.debug(this, "Register serialized Actor [%s] as remote @ [%s]".format(actor.uuid, remoteAddress))
|
||||
|
||||
RemoteActorRefProtocol.newBuilder
|
||||
.setInetSocketAddress(ByteString.copyFrom(JavaSerializer.toBinary(remoteAddress)))
|
||||
.setAddress(actor.address)
|
||||
.setTimeout(actor.timeout)
|
||||
.build
|
||||
}
|
||||
|
||||
def createRemoteMessageProtocolBuilder(
|
||||
actorRef: Option[ActorRef],
|
||||
replyUuid: Either[Uuid, UuidProtocol],
|
||||
actorAddress: String,
|
||||
timeout: Long,
|
||||
message: Either[Throwable, Any],
|
||||
isOneWay: Boolean,
|
||||
senderOption: Option[ActorRef]): RemoteMessageProtocol.Builder = {
|
||||
|
||||
val uuidProtocol = replyUuid match {
|
||||
case Left(uid) ⇒ UuidProtocol.newBuilder.setHigh(uid.getTime).setLow(uid.getClockSeqAndNode).build
|
||||
case Right(protocol) ⇒ protocol
|
||||
}
|
||||
|
||||
val actorInfoBuilder = ActorInfoProtocol.newBuilder
|
||||
.setUuid(uuidProtocol)
|
||||
.setAddress(actorAddress)
|
||||
.setTimeout(timeout)
|
||||
|
||||
val actorInfo = actorInfoBuilder.build
|
||||
val messageBuilder = RemoteMessageProtocol.newBuilder
|
||||
.setUuid({
|
||||
val messageUuid = newUuid
|
||||
UuidProtocol.newBuilder.setHigh(messageUuid.getTime).setLow(messageUuid.getClockSeqAndNode).build
|
||||
})
|
||||
.setActorInfo(actorInfo)
|
||||
.setOneWay(isOneWay)
|
||||
|
||||
message match {
|
||||
case Right(message) ⇒
|
||||
messageBuilder.setMessage(MessageSerializer.serialize(message.asInstanceOf[AnyRef]))
|
||||
case Left(exception) ⇒
|
||||
messageBuilder.setException(ExceptionProtocol.newBuilder
|
||||
.setClassname(exception.getClass.getName)
|
||||
.setMessage(empty(exception.getMessage))
|
||||
.build)
|
||||
}
|
||||
|
||||
def empty(s: String): String = s match {
|
||||
case null ⇒ ""
|
||||
case s ⇒ s
|
||||
}
|
||||
|
||||
/* TODO invent new supervision strategy
|
||||
actorRef.foreach { ref =>
|
||||
ref.registerSupervisorAsRemoteActor.foreach { id =>
|
||||
messageBuilder.setSupervisorUuid(
|
||||
UuidProtocol.newBuilder
|
||||
.setHigh(id.getTime)
|
||||
.setLow(id.getClockSeqAndNode)
|
||||
.build)
|
||||
}
|
||||
} */
|
||||
|
||||
if (senderOption.isDefined)
|
||||
messageBuilder.setSender(toRemoteActorRefProtocol(senderOption.get))
|
||||
|
||||
messageBuilder
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,116 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import org.scalatest.WordSpec
|
||||
import org.scalatest.matchers.MustMatchers
|
||||
import org.scalatest.BeforeAndAfterAll
|
||||
|
||||
import akka.util.duration._
|
||||
import akka.util.Duration
|
||||
import System.{ currentTimeMillis ⇒ now }
|
||||
|
||||
import java.io.File
|
||||
|
||||
trait ClusterTestNode extends WordSpec with MustMatchers with BeforeAndAfterAll {
|
||||
|
||||
override def beforeAll() = {
|
||||
ClusterTestNode.waitForReady(getClass.getName)
|
||||
}
|
||||
|
||||
override def afterAll() = {
|
||||
ClusterTestNode.exit(getClass.getName)
|
||||
}
|
||||
}
|
||||
|
||||
object ClusterTestNode {
|
||||
val TestMarker = "MultiJvm"
|
||||
val HomeDir = "_akka_cluster"
|
||||
val TestDir = "multi-jvm"
|
||||
val Sleep = 100.millis
|
||||
val Timeout = 1.minute
|
||||
|
||||
def ready(className: String) = {
|
||||
readyFile(className).createNewFile()
|
||||
}
|
||||
|
||||
def waitForReady(className: String) = {
|
||||
if (!waitExists(readyFile(className))) {
|
||||
cleanUp(className)
|
||||
sys.error("Timeout waiting for cluster ready")
|
||||
}
|
||||
}
|
||||
|
||||
def exit(className: String) = {
|
||||
exitFile(className).createNewFile()
|
||||
}
|
||||
|
||||
def waitForExits(className: String, nodes: Int) = {
|
||||
if (!waitCount(exitDir(className), nodes)) {
|
||||
cleanUp(className)
|
||||
sys.error("Timeout waiting for node exits")
|
||||
}
|
||||
}
|
||||
|
||||
def cleanUp(className: String) = {
|
||||
deleteRecursive(testDir(className))
|
||||
}
|
||||
|
||||
def testName(name: String) = {
|
||||
val i = name.indexOf(TestMarker)
|
||||
if (i >= 0) name.substring(0, i) else name
|
||||
}
|
||||
|
||||
def nodeName(name: String) = {
|
||||
val i = name.indexOf(TestMarker)
|
||||
if (i >= 0) name.substring(i + TestMarker.length) else name
|
||||
}
|
||||
|
||||
def testDir(className: String) = {
|
||||
val home = new File(HomeDir)
|
||||
val tests = new File(home, TestDir)
|
||||
val dir = new File(tests, testName(className))
|
||||
dir.mkdirs()
|
||||
dir
|
||||
}
|
||||
|
||||
def readyFile(className: String) = {
|
||||
new File(testDir(className), "ready")
|
||||
}
|
||||
|
||||
def exitDir(className: String) = {
|
||||
val dir = new File(testDir(className), "exit")
|
||||
dir.mkdirs()
|
||||
dir
|
||||
}
|
||||
|
||||
def exitFile(className: String) = {
|
||||
new File(exitDir(className), nodeName(className))
|
||||
}
|
||||
|
||||
def waitExists(file: File) = waitFor(file.exists)
|
||||
|
||||
def waitCount(file: File, n: Int) = waitFor(file.list.size >= n)
|
||||
|
||||
def waitFor(test: ⇒ Boolean, sleep: Duration = Sleep, timeout: Duration = Timeout): Boolean = {
|
||||
val start = now
|
||||
val limit = start + timeout.toMillis
|
||||
var passed = test
|
||||
var expired = false
|
||||
while (!passed && !expired) {
|
||||
if (now > limit) expired = true
|
||||
else {
|
||||
Thread.sleep(sleep.toMillis)
|
||||
passed = test
|
||||
}
|
||||
}
|
||||
passed
|
||||
}
|
||||
|
||||
def deleteRecursive(file: File): Boolean = {
|
||||
if (file.isDirectory) file.listFiles.foreach(deleteRecursive)
|
||||
file.delete()
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,31 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import org.scalatest.WordSpec
|
||||
import org.scalatest.matchers.MustMatchers
|
||||
import org.scalatest.BeforeAndAfterAll
|
||||
|
||||
trait MasterClusterTestNode extends WordSpec with MustMatchers with BeforeAndAfterAll {
|
||||
def testNodes: Int
|
||||
|
||||
override def beforeAll() = {
|
||||
// LocalCluster.startLocalCluster()
|
||||
onReady()
|
||||
ClusterTestNode.ready(getClass.getName)
|
||||
}
|
||||
|
||||
def onReady() = {}
|
||||
|
||||
override def afterAll() = {
|
||||
ClusterTestNode.waitForExits(getClass.getName, testNodes - 1)
|
||||
ClusterTestNode.cleanUp(getClass.getName)
|
||||
onShutdown()
|
||||
// LocalCluster.shutdownLocalCluster()
|
||||
}
|
||||
|
||||
def onShutdown() = {}
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,20 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package org.scalatest.akka
|
||||
|
||||
import org.scalatest.tools.StandardOutReporter
|
||||
import org.scalatest.events._
|
||||
import java.lang.Boolean.getBoolean
|
||||
|
||||
class QuietReporter(inColor: Boolean) extends StandardOutReporter(false, inColor, false, false) {
|
||||
def this() = this(!getBoolean("akka.test.nocolor"))
|
||||
|
||||
override def apply(event: Event): Unit = event match {
|
||||
case _: RunStarting ⇒ ()
|
||||
case _ ⇒ super.apply(event)
|
||||
}
|
||||
|
||||
override def makeFinalReport(resourceName: String, duration: Option[Long], summaryOption: Option[Summary]): Unit = {}
|
||||
}
|
||||
565
akka-remote/src/test/java/akka/actor/ProtobufProtocol.java
Normal file
565
akka-remote/src/test/java/akka/actor/ProtobufProtocol.java
Normal file
|
|
@ -0,0 +1,565 @@
|
|||
// Generated by the protocol buffer compiler. DO NOT EDIT!
|
||||
// source: ProtobufProtocol.proto
|
||||
|
||||
package akka.actor;
|
||||
|
||||
public final class ProtobufProtocol {
|
||||
private ProtobufProtocol() {}
|
||||
public static void registerAllExtensions(
|
||||
com.google.protobuf.ExtensionRegistry registry) {
|
||||
}
|
||||
public interface MyMessageOrBuilder
|
||||
extends com.google.protobuf.MessageOrBuilder {
|
||||
|
||||
// required uint64 id = 1;
|
||||
boolean hasId();
|
||||
long getId();
|
||||
|
||||
// required string name = 2;
|
||||
boolean hasName();
|
||||
String getName();
|
||||
|
||||
// required bool status = 3;
|
||||
boolean hasStatus();
|
||||
boolean getStatus();
|
||||
}
|
||||
public static final class MyMessage extends
|
||||
com.google.protobuf.GeneratedMessage
|
||||
implements MyMessageOrBuilder {
|
||||
// Use MyMessage.newBuilder() to construct.
|
||||
private MyMessage(Builder builder) {
|
||||
super(builder);
|
||||
}
|
||||
private MyMessage(boolean noInit) {}
|
||||
|
||||
private static final MyMessage defaultInstance;
|
||||
public static MyMessage getDefaultInstance() {
|
||||
return defaultInstance;
|
||||
}
|
||||
|
||||
public MyMessage getDefaultInstanceForType() {
|
||||
return defaultInstance;
|
||||
}
|
||||
|
||||
public static final com.google.protobuf.Descriptors.Descriptor
|
||||
getDescriptor() {
|
||||
return akka.actor.ProtobufProtocol.internal_static_akka_actor_MyMessage_descriptor;
|
||||
}
|
||||
|
||||
protected com.google.protobuf.GeneratedMessage.FieldAccessorTable
|
||||
internalGetFieldAccessorTable() {
|
||||
return akka.actor.ProtobufProtocol.internal_static_akka_actor_MyMessage_fieldAccessorTable;
|
||||
}
|
||||
|
||||
private int bitField0_;
|
||||
// required uint64 id = 1;
|
||||
public static final int ID_FIELD_NUMBER = 1;
|
||||
private long id_;
|
||||
public boolean hasId() {
|
||||
return ((bitField0_ & 0x00000001) == 0x00000001);
|
||||
}
|
||||
public long getId() {
|
||||
return id_;
|
||||
}
|
||||
|
||||
// required string name = 2;
|
||||
public static final int NAME_FIELD_NUMBER = 2;
|
||||
private java.lang.Object name_;
|
||||
public boolean hasName() {
|
||||
return ((bitField0_ & 0x00000002) == 0x00000002);
|
||||
}
|
||||
public String getName() {
|
||||
java.lang.Object ref = name_;
|
||||
if (ref instanceof String) {
|
||||
return (String) ref;
|
||||
} else {
|
||||
com.google.protobuf.ByteString bs =
|
||||
(com.google.protobuf.ByteString) ref;
|
||||
String s = bs.toStringUtf8();
|
||||
if (com.google.protobuf.Internal.isValidUtf8(bs)) {
|
||||
name_ = s;
|
||||
}
|
||||
return s;
|
||||
}
|
||||
}
|
||||
private com.google.protobuf.ByteString getNameBytes() {
|
||||
java.lang.Object ref = name_;
|
||||
if (ref instanceof String) {
|
||||
com.google.protobuf.ByteString b =
|
||||
com.google.protobuf.ByteString.copyFromUtf8((String) ref);
|
||||
name_ = b;
|
||||
return b;
|
||||
} else {
|
||||
return (com.google.protobuf.ByteString) ref;
|
||||
}
|
||||
}
|
||||
|
||||
// required bool status = 3;
|
||||
public static final int STATUS_FIELD_NUMBER = 3;
|
||||
private boolean status_;
|
||||
public boolean hasStatus() {
|
||||
return ((bitField0_ & 0x00000004) == 0x00000004);
|
||||
}
|
||||
public boolean getStatus() {
|
||||
return status_;
|
||||
}
|
||||
|
||||
private void initFields() {
|
||||
id_ = 0L;
|
||||
name_ = "";
|
||||
status_ = false;
|
||||
}
|
||||
private byte memoizedIsInitialized = -1;
|
||||
public final boolean isInitialized() {
|
||||
byte isInitialized = memoizedIsInitialized;
|
||||
if (isInitialized != -1) return isInitialized == 1;
|
||||
|
||||
if (!hasId()) {
|
||||
memoizedIsInitialized = 0;
|
||||
return false;
|
||||
}
|
||||
if (!hasName()) {
|
||||
memoizedIsInitialized = 0;
|
||||
return false;
|
||||
}
|
||||
if (!hasStatus()) {
|
||||
memoizedIsInitialized = 0;
|
||||
return false;
|
||||
}
|
||||
memoizedIsInitialized = 1;
|
||||
return true;
|
||||
}
|
||||
|
||||
public void writeTo(com.google.protobuf.CodedOutputStream output)
|
||||
throws java.io.IOException {
|
||||
getSerializedSize();
|
||||
if (((bitField0_ & 0x00000001) == 0x00000001)) {
|
||||
output.writeUInt64(1, id_);
|
||||
}
|
||||
if (((bitField0_ & 0x00000002) == 0x00000002)) {
|
||||
output.writeBytes(2, getNameBytes());
|
||||
}
|
||||
if (((bitField0_ & 0x00000004) == 0x00000004)) {
|
||||
output.writeBool(3, status_);
|
||||
}
|
||||
getUnknownFields().writeTo(output);
|
||||
}
|
||||
|
||||
private int memoizedSerializedSize = -1;
|
||||
public int getSerializedSize() {
|
||||
int size = memoizedSerializedSize;
|
||||
if (size != -1) return size;
|
||||
|
||||
size = 0;
|
||||
if (((bitField0_ & 0x00000001) == 0x00000001)) {
|
||||
size += com.google.protobuf.CodedOutputStream
|
||||
.computeUInt64Size(1, id_);
|
||||
}
|
||||
if (((bitField0_ & 0x00000002) == 0x00000002)) {
|
||||
size += com.google.protobuf.CodedOutputStream
|
||||
.computeBytesSize(2, getNameBytes());
|
||||
}
|
||||
if (((bitField0_ & 0x00000004) == 0x00000004)) {
|
||||
size += com.google.protobuf.CodedOutputStream
|
||||
.computeBoolSize(3, status_);
|
||||
}
|
||||
size += getUnknownFields().getSerializedSize();
|
||||
memoizedSerializedSize = size;
|
||||
return size;
|
||||
}
|
||||
|
||||
private static final long serialVersionUID = 0L;
|
||||
@java.lang.Override
|
||||
protected java.lang.Object writeReplace()
|
||||
throws java.io.ObjectStreamException {
|
||||
return super.writeReplace();
|
||||
}
|
||||
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
com.google.protobuf.ByteString data)
|
||||
throws com.google.protobuf.InvalidProtocolBufferException {
|
||||
return newBuilder().mergeFrom(data).buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
com.google.protobuf.ByteString data,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws com.google.protobuf.InvalidProtocolBufferException {
|
||||
return newBuilder().mergeFrom(data, extensionRegistry)
|
||||
.buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(byte[] data)
|
||||
throws com.google.protobuf.InvalidProtocolBufferException {
|
||||
return newBuilder().mergeFrom(data).buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
byte[] data,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws com.google.protobuf.InvalidProtocolBufferException {
|
||||
return newBuilder().mergeFrom(data, extensionRegistry)
|
||||
.buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(java.io.InputStream input)
|
||||
throws java.io.IOException {
|
||||
return newBuilder().mergeFrom(input).buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
java.io.InputStream input,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws java.io.IOException {
|
||||
return newBuilder().mergeFrom(input, extensionRegistry)
|
||||
.buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseDelimitedFrom(java.io.InputStream input)
|
||||
throws java.io.IOException {
|
||||
Builder builder = newBuilder();
|
||||
if (builder.mergeDelimitedFrom(input)) {
|
||||
return builder.buildParsed();
|
||||
} else {
|
||||
return null;
|
||||
}
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseDelimitedFrom(
|
||||
java.io.InputStream input,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws java.io.IOException {
|
||||
Builder builder = newBuilder();
|
||||
if (builder.mergeDelimitedFrom(input, extensionRegistry)) {
|
||||
return builder.buildParsed();
|
||||
} else {
|
||||
return null;
|
||||
}
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
com.google.protobuf.CodedInputStream input)
|
||||
throws java.io.IOException {
|
||||
return newBuilder().mergeFrom(input).buildParsed();
|
||||
}
|
||||
public static akka.actor.ProtobufProtocol.MyMessage parseFrom(
|
||||
com.google.protobuf.CodedInputStream input,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws java.io.IOException {
|
||||
return newBuilder().mergeFrom(input, extensionRegistry)
|
||||
.buildParsed();
|
||||
}
|
||||
|
||||
public static Builder newBuilder() { return Builder.create(); }
|
||||
public Builder newBuilderForType() { return newBuilder(); }
|
||||
public static Builder newBuilder(akka.actor.ProtobufProtocol.MyMessage prototype) {
|
||||
return newBuilder().mergeFrom(prototype);
|
||||
}
|
||||
public Builder toBuilder() { return newBuilder(this); }
|
||||
|
||||
@java.lang.Override
|
||||
protected Builder newBuilderForType(
|
||||
com.google.protobuf.GeneratedMessage.BuilderParent parent) {
|
||||
Builder builder = new Builder(parent);
|
||||
return builder;
|
||||
}
|
||||
public static final class Builder extends
|
||||
com.google.protobuf.GeneratedMessage.Builder<Builder>
|
||||
implements akka.actor.ProtobufProtocol.MyMessageOrBuilder {
|
||||
public static final com.google.protobuf.Descriptors.Descriptor
|
||||
getDescriptor() {
|
||||
return akka.actor.ProtobufProtocol.internal_static_akka_actor_MyMessage_descriptor;
|
||||
}
|
||||
|
||||
protected com.google.protobuf.GeneratedMessage.FieldAccessorTable
|
||||
internalGetFieldAccessorTable() {
|
||||
return akka.actor.ProtobufProtocol.internal_static_akka_actor_MyMessage_fieldAccessorTable;
|
||||
}
|
||||
|
||||
// Construct using akka.actor.ProtobufProtocol.MyMessage.newBuilder()
|
||||
private Builder() {
|
||||
maybeForceBuilderInitialization();
|
||||
}
|
||||
|
||||
private Builder(BuilderParent parent) {
|
||||
super(parent);
|
||||
maybeForceBuilderInitialization();
|
||||
}
|
||||
private void maybeForceBuilderInitialization() {
|
||||
if (com.google.protobuf.GeneratedMessage.alwaysUseFieldBuilders) {
|
||||
}
|
||||
}
|
||||
private static Builder create() {
|
||||
return new Builder();
|
||||
}
|
||||
|
||||
public Builder clear() {
|
||||
super.clear();
|
||||
id_ = 0L;
|
||||
bitField0_ = (bitField0_ & ~0x00000001);
|
||||
name_ = "";
|
||||
bitField0_ = (bitField0_ & ~0x00000002);
|
||||
status_ = false;
|
||||
bitField0_ = (bitField0_ & ~0x00000004);
|
||||
return this;
|
||||
}
|
||||
|
||||
public Builder clone() {
|
||||
return create().mergeFrom(buildPartial());
|
||||
}
|
||||
|
||||
public com.google.protobuf.Descriptors.Descriptor
|
||||
getDescriptorForType() {
|
||||
return akka.actor.ProtobufProtocol.MyMessage.getDescriptor();
|
||||
}
|
||||
|
||||
public akka.actor.ProtobufProtocol.MyMessage getDefaultInstanceForType() {
|
||||
return akka.actor.ProtobufProtocol.MyMessage.getDefaultInstance();
|
||||
}
|
||||
|
||||
public akka.actor.ProtobufProtocol.MyMessage build() {
|
||||
akka.actor.ProtobufProtocol.MyMessage result = buildPartial();
|
||||
if (!result.isInitialized()) {
|
||||
throw newUninitializedMessageException(result);
|
||||
}
|
||||
return result;
|
||||
}
|
||||
|
||||
private akka.actor.ProtobufProtocol.MyMessage buildParsed()
|
||||
throws com.google.protobuf.InvalidProtocolBufferException {
|
||||
akka.actor.ProtobufProtocol.MyMessage result = buildPartial();
|
||||
if (!result.isInitialized()) {
|
||||
throw newUninitializedMessageException(
|
||||
result).asInvalidProtocolBufferException();
|
||||
}
|
||||
return result;
|
||||
}
|
||||
|
||||
public akka.actor.ProtobufProtocol.MyMessage buildPartial() {
|
||||
akka.actor.ProtobufProtocol.MyMessage result = new akka.actor.ProtobufProtocol.MyMessage(this);
|
||||
int from_bitField0_ = bitField0_;
|
||||
int to_bitField0_ = 0;
|
||||
if (((from_bitField0_ & 0x00000001) == 0x00000001)) {
|
||||
to_bitField0_ |= 0x00000001;
|
||||
}
|
||||
result.id_ = id_;
|
||||
if (((from_bitField0_ & 0x00000002) == 0x00000002)) {
|
||||
to_bitField0_ |= 0x00000002;
|
||||
}
|
||||
result.name_ = name_;
|
||||
if (((from_bitField0_ & 0x00000004) == 0x00000004)) {
|
||||
to_bitField0_ |= 0x00000004;
|
||||
}
|
||||
result.status_ = status_;
|
||||
result.bitField0_ = to_bitField0_;
|
||||
onBuilt();
|
||||
return result;
|
||||
}
|
||||
|
||||
public Builder mergeFrom(com.google.protobuf.Message other) {
|
||||
if (other instanceof akka.actor.ProtobufProtocol.MyMessage) {
|
||||
return mergeFrom((akka.actor.ProtobufProtocol.MyMessage)other);
|
||||
} else {
|
||||
super.mergeFrom(other);
|
||||
return this;
|
||||
}
|
||||
}
|
||||
|
||||
public Builder mergeFrom(akka.actor.ProtobufProtocol.MyMessage other) {
|
||||
if (other == akka.actor.ProtobufProtocol.MyMessage.getDefaultInstance()) return this;
|
||||
if (other.hasId()) {
|
||||
setId(other.getId());
|
||||
}
|
||||
if (other.hasName()) {
|
||||
setName(other.getName());
|
||||
}
|
||||
if (other.hasStatus()) {
|
||||
setStatus(other.getStatus());
|
||||
}
|
||||
this.mergeUnknownFields(other.getUnknownFields());
|
||||
return this;
|
||||
}
|
||||
|
||||
public final boolean isInitialized() {
|
||||
if (!hasId()) {
|
||||
|
||||
return false;
|
||||
}
|
||||
if (!hasName()) {
|
||||
|
||||
return false;
|
||||
}
|
||||
if (!hasStatus()) {
|
||||
|
||||
return false;
|
||||
}
|
||||
return true;
|
||||
}
|
||||
|
||||
public Builder mergeFrom(
|
||||
com.google.protobuf.CodedInputStream input,
|
||||
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
|
||||
throws java.io.IOException {
|
||||
com.google.protobuf.UnknownFieldSet.Builder unknownFields =
|
||||
com.google.protobuf.UnknownFieldSet.newBuilder(
|
||||
this.getUnknownFields());
|
||||
while (true) {
|
||||
int tag = input.readTag();
|
||||
switch (tag) {
|
||||
case 0:
|
||||
this.setUnknownFields(unknownFields.build());
|
||||
onChanged();
|
||||
return this;
|
||||
default: {
|
||||
if (!parseUnknownField(input, unknownFields,
|
||||
extensionRegistry, tag)) {
|
||||
this.setUnknownFields(unknownFields.build());
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
break;
|
||||
}
|
||||
case 8: {
|
||||
bitField0_ |= 0x00000001;
|
||||
id_ = input.readUInt64();
|
||||
break;
|
||||
}
|
||||
case 18: {
|
||||
bitField0_ |= 0x00000002;
|
||||
name_ = input.readBytes();
|
||||
break;
|
||||
}
|
||||
case 24: {
|
||||
bitField0_ |= 0x00000004;
|
||||
status_ = input.readBool();
|
||||
break;
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
private int bitField0_;
|
||||
|
||||
// required uint64 id = 1;
|
||||
private long id_ ;
|
||||
public boolean hasId() {
|
||||
return ((bitField0_ & 0x00000001) == 0x00000001);
|
||||
}
|
||||
public long getId() {
|
||||
return id_;
|
||||
}
|
||||
public Builder setId(long value) {
|
||||
bitField0_ |= 0x00000001;
|
||||
id_ = value;
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
public Builder clearId() {
|
||||
bitField0_ = (bitField0_ & ~0x00000001);
|
||||
id_ = 0L;
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
|
||||
// required string name = 2;
|
||||
private java.lang.Object name_ = "";
|
||||
public boolean hasName() {
|
||||
return ((bitField0_ & 0x00000002) == 0x00000002);
|
||||
}
|
||||
public String getName() {
|
||||
java.lang.Object ref = name_;
|
||||
if (!(ref instanceof String)) {
|
||||
String s = ((com.google.protobuf.ByteString) ref).toStringUtf8();
|
||||
name_ = s;
|
||||
return s;
|
||||
} else {
|
||||
return (String) ref;
|
||||
}
|
||||
}
|
||||
public Builder setName(String value) {
|
||||
if (value == null) {
|
||||
throw new NullPointerException();
|
||||
}
|
||||
bitField0_ |= 0x00000002;
|
||||
name_ = value;
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
public Builder clearName() {
|
||||
bitField0_ = (bitField0_ & ~0x00000002);
|
||||
name_ = getDefaultInstance().getName();
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
void setName(com.google.protobuf.ByteString value) {
|
||||
bitField0_ |= 0x00000002;
|
||||
name_ = value;
|
||||
onChanged();
|
||||
}
|
||||
|
||||
// required bool status = 3;
|
||||
private boolean status_ ;
|
||||
public boolean hasStatus() {
|
||||
return ((bitField0_ & 0x00000004) == 0x00000004);
|
||||
}
|
||||
public boolean getStatus() {
|
||||
return status_;
|
||||
}
|
||||
public Builder setStatus(boolean value) {
|
||||
bitField0_ |= 0x00000004;
|
||||
status_ = value;
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
public Builder clearStatus() {
|
||||
bitField0_ = (bitField0_ & ~0x00000004);
|
||||
status_ = false;
|
||||
onChanged();
|
||||
return this;
|
||||
}
|
||||
|
||||
// @@protoc_insertion_point(builder_scope:akka.actor.MyMessage)
|
||||
}
|
||||
|
||||
static {
|
||||
defaultInstance = new MyMessage(true);
|
||||
defaultInstance.initFields();
|
||||
}
|
||||
|
||||
// @@protoc_insertion_point(class_scope:akka.actor.MyMessage)
|
||||
}
|
||||
|
||||
private static com.google.protobuf.Descriptors.Descriptor
|
||||
internal_static_akka_actor_MyMessage_descriptor;
|
||||
private static
|
||||
com.google.protobuf.GeneratedMessage.FieldAccessorTable
|
||||
internal_static_akka_actor_MyMessage_fieldAccessorTable;
|
||||
|
||||
public static com.google.protobuf.Descriptors.FileDescriptor
|
||||
getDescriptor() {
|
||||
return descriptor;
|
||||
}
|
||||
private static com.google.protobuf.Descriptors.FileDescriptor
|
||||
descriptor;
|
||||
static {
|
||||
java.lang.String[] descriptorData = {
|
||||
"\n\026ProtobufProtocol.proto\022\nakka.actor\"5\n\t" +
|
||||
"MyMessage\022\n\n\002id\030\001 \002(\004\022\014\n\004name\030\002 \002(\t\022\016\n\006s" +
|
||||
"tatus\030\003 \002(\010"
|
||||
};
|
||||
com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner assigner =
|
||||
new com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner() {
|
||||
public com.google.protobuf.ExtensionRegistry assignDescriptors(
|
||||
com.google.protobuf.Descriptors.FileDescriptor root) {
|
||||
descriptor = root;
|
||||
internal_static_akka_actor_MyMessage_descriptor =
|
||||
getDescriptor().getMessageTypes().get(0);
|
||||
internal_static_akka_actor_MyMessage_fieldAccessorTable = new
|
||||
com.google.protobuf.GeneratedMessage.FieldAccessorTable(
|
||||
internal_static_akka_actor_MyMessage_descriptor,
|
||||
new java.lang.String[] { "Id", "Name", "Status", },
|
||||
akka.actor.ProtobufProtocol.MyMessage.class,
|
||||
akka.actor.ProtobufProtocol.MyMessage.Builder.class);
|
||||
return null;
|
||||
}
|
||||
};
|
||||
com.google.protobuf.Descriptors.FileDescriptor
|
||||
.internalBuildGeneratedFileFrom(descriptorData,
|
||||
new com.google.protobuf.Descriptors.FileDescriptor[] {
|
||||
}, assigner);
|
||||
}
|
||||
|
||||
// @@protoc_insertion_point(outer_class_scope)
|
||||
}
|
||||
18
akka-remote/src/test/protocol/ProtobufProtocol.proto
Normal file
18
akka-remote/src/test/protocol/ProtobufProtocol.proto
Normal file
|
|
@ -0,0 +1,18 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.actor;
|
||||
|
||||
/*
|
||||
Compile with:
|
||||
cd ./akka-remote/src/test/protocol
|
||||
protoc ProtobufProtocol.proto --java_out ../java
|
||||
*/
|
||||
|
||||
message MyMessage {
|
||||
required uint64 id = 1;
|
||||
required string name = 2;
|
||||
required bool status = 3;
|
||||
}
|
||||
|
||||
58
akka-remote/src/test/resources/log4j.properties
Normal file
58
akka-remote/src/test/resources/log4j.properties
Normal file
|
|
@ -0,0 +1,58 @@
|
|||
# Define some default values that can be overridden by system properties
|
||||
zookeeper.root.logger=INFO, CONSOLE
|
||||
zookeeper.console.threshold=OFF
|
||||
zookeeper.log.dir=.
|
||||
zookeeper.log.file=zookeeper.log
|
||||
zookeeper.log.threshold=DEBUG
|
||||
zookeeper.tracelog.dir=.
|
||||
zookeeper.tracelog.file=zookeeper_trace.log
|
||||
|
||||
#
|
||||
# ZooKeeper Logging Configuration
|
||||
#
|
||||
|
||||
# Format is "<default threshold> (, <appender>)+
|
||||
|
||||
# DEFAULT: console appender only
|
||||
log4j.rootLogger=${zookeeper.root.logger}
|
||||
|
||||
# Example with rolling log file
|
||||
#log4j.rootLogger=DEBUG, CONSOLE, ROLLINGFILE
|
||||
|
||||
# Example with rolling log file and tracing
|
||||
#log4j.rootLogger=TRACE, CONSOLE, ROLLINGFILE, TRACEFILE
|
||||
|
||||
#
|
||||
# Log INFO level and above messages to the console
|
||||
#
|
||||
log4j.appender.CONSOLE=org.apache.log4j.ConsoleAppender
|
||||
log4j.appender.CONSOLE.Threshold=${zookeeper.console.threshold}
|
||||
log4j.appender.CONSOLE.layout=org.apache.log4j.PatternLayout
|
||||
log4j.appender.CONSOLE.layout.ConversionPattern=%d{ISO8601} [myid:%X{myid}] - %-5p [%t:%C{1}@%L] - %m%n
|
||||
|
||||
#
|
||||
# Add ROLLINGFILE to rootLogger to get log file output
|
||||
# Log DEBUG level and above messages to a log file
|
||||
log4j.appender.ROLLINGFILE=org.apache.log4j.RollingFileAppender
|
||||
log4j.appender.ROLLINGFILE.Threshold=${zookeeper.log.threshold}
|
||||
log4j.appender.ROLLINGFILE.File=${zookeeper.log.dir}/${zookeeper.log.file}
|
||||
|
||||
# Max log file size of 10MB
|
||||
log4j.appender.ROLLINGFILE.MaxFileSize=10MB
|
||||
# uncomment the next line to limit number of backup files
|
||||
#log4j.appender.ROLLINGFILE.MaxBackupIndex=10
|
||||
|
||||
log4j.appender.ROLLINGFILE.layout=org.apache.log4j.PatternLayout
|
||||
log4j.appender.ROLLINGFILE.layout.ConversionPattern=%d{ISO8601} [myid:%X{myid}] - %-5p [%t:%C{1}@%L] - %m%n
|
||||
|
||||
|
||||
#
|
||||
# Add TRACEFILE to rootLogger to get log file output
|
||||
# Log DEBUG level and above messages to a log file
|
||||
log4j.appender.TRACEFILE=org.apache.log4j.FileAppender
|
||||
log4j.appender.TRACEFILE.Threshold=TRACE
|
||||
log4j.appender.TRACEFILE.File=${zookeeper.tracelog.dir}/${zookeeper.tracelog.file}
|
||||
|
||||
log4j.appender.TRACEFILE.layout=org.apache.log4j.PatternLayout
|
||||
### Notice we are including log4j's NDC here (%x)
|
||||
log4j.appender.TRACEFILE.layout.ConversionPattern=%d{ISO8601} [myid:%X{myid}] - %-5p [%t:%C{1}@%L][%x] - %m%n
|
||||
26
akka-remote/src/test/resources/logback-test.xml
Normal file
26
akka-remote/src/test/resources/logback-test.xml
Normal file
|
|
@ -0,0 +1,26 @@
|
|||
<?xml version="1.0" encoding="UTF-8"?>
|
||||
|
||||
<!-- For assistance related to logback-translator or configuration -->
|
||||
<!-- files in general, please contact the logback user mailing list -->
|
||||
<!-- at http://www.qos.ch/mailman/listinfo/logback-user -->
|
||||
<!-- -->
|
||||
<!-- For professional support please see -->
|
||||
<!-- http://www.qos.ch/shop/products/professionalSupport -->
|
||||
<!-- -->
|
||||
<configuration scan="false" debug="false">
|
||||
<!-- Errors were reported during translation. -->
|
||||
<!-- Could not find transformer for org.apache.log4j.SimpleLayout -->
|
||||
<appender name="stdout" class="ch.qos.logback.core.ConsoleAppender">
|
||||
<encoder>
|
||||
<pattern>[%4p] [%d{ISO8601}] [%t] %c{1}: %m%n</pattern>
|
||||
</encoder>
|
||||
</appender>
|
||||
<logger name="akka" level="DEBUG"/>
|
||||
|
||||
<logger name="org.mortbay.log" level="ERROR"/>
|
||||
<logger name="org.apache.jasper" level="ERROR"/>
|
||||
|
||||
<root level="INFO">
|
||||
<appender-ref ref="stdout"/>
|
||||
</root>
|
||||
</configuration>
|
||||
12
akka-remote/src/test/resources/zoo.cfg
Normal file
12
akka-remote/src/test/resources/zoo.cfg
Normal file
|
|
@ -0,0 +1,12 @@
|
|||
# The number of milliseconds of each tick
|
||||
tickTime=2000
|
||||
# The number of ticks that the initial
|
||||
# synchronization phase can take
|
||||
initLimit=10
|
||||
# The number of ticks that can pass between
|
||||
# sending a request and getting an acknowledgement
|
||||
syncLimit=5
|
||||
# the directory where the snapshot is stored.
|
||||
dataDir=/export/crawlspace/mahadev/zookeeper/server1/data
|
||||
# the port at which the clients will connect
|
||||
clientPort=2181
|
||||
102
akka-remote/src/test/scala/akka/cluster/NetworkFailureSpec.scala
Normal file
102
akka-remote/src/test/scala/akka/cluster/NetworkFailureSpec.scala
Normal file
|
|
@ -0,0 +1,102 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.cluster
|
||||
|
||||
import org.scalatest.{ Spec, WordSpec, BeforeAndAfterAll, BeforeAndAfterEach }
|
||||
import org.scalatest.matchers.MustMatchers
|
||||
import org.scalatest.junit.JUnitRunner
|
||||
|
||||
import org.junit.runner.RunWith
|
||||
|
||||
import akka.cluster.netty.NettyRemoteSupport
|
||||
import akka.actor.{ Actor, ActorRegistry }
|
||||
|
||||
import java.util.concurrent.{ TimeUnit, CountDownLatch }
|
||||
import java.util.concurrent.atomic.AtomicBoolean
|
||||
|
||||
trait NetworkFailureSpec { self: WordSpec ⇒
|
||||
import Actor._
|
||||
import akka.util.Duration
|
||||
|
||||
val BytesPerSecond = "60KByte/s"
|
||||
val DelayMillis = "350ms"
|
||||
val PortRang = "1024-65535"
|
||||
|
||||
def replyWithTcpResetFor(duration: Duration, dead: AtomicBoolean) = {
|
||||
spawn {
|
||||
try {
|
||||
enableTcpReset()
|
||||
println("===>>> Reply with [TCP RST] for [" + duration + "]")
|
||||
Thread.sleep(duration.toMillis)
|
||||
restoreIP
|
||||
} catch {
|
||||
case e ⇒
|
||||
dead.set(true)
|
||||
e.printStackTrace
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
def throttleNetworkFor(duration: Duration, dead: AtomicBoolean) = {
|
||||
spawn {
|
||||
try {
|
||||
enableNetworkThrottling()
|
||||
println("===>>> Throttling network with [" + BytesPerSecond + ", " + DelayMillis + "] for [" + duration + "]")
|
||||
Thread.sleep(duration.toMillis)
|
||||
restoreIP
|
||||
} catch {
|
||||
case e ⇒
|
||||
dead.set(true)
|
||||
e.printStackTrace
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
def dropNetworkFor(duration: Duration, dead: AtomicBoolean) = {
|
||||
spawn {
|
||||
try {
|
||||
enableNetworkDrop()
|
||||
println("===>>> Blocking network [TCP DENY] for [" + duration + "]")
|
||||
Thread.sleep(duration.toMillis)
|
||||
restoreIP
|
||||
} catch {
|
||||
case e ⇒
|
||||
dead.set(true)
|
||||
e.printStackTrace
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
def sleepFor(duration: Duration) = {
|
||||
println("===>>> Sleeping for [" + duration + "]")
|
||||
Thread sleep (duration.toMillis)
|
||||
}
|
||||
|
||||
def enableNetworkThrottling() = {
|
||||
restoreIP()
|
||||
assert(new ProcessBuilder("ipfw", "add", "pipe", "1", "ip", "from", "any", "to", "any").start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "add", "pipe", "2", "ip", "from", "any", "to", "any").start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "pipe", "1", "config", "bw", BytesPerSecond, "delay", DelayMillis).start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "pipe", "2", "config", "bw", BytesPerSecond, "delay", DelayMillis).start.waitFor == 0)
|
||||
}
|
||||
|
||||
def enableNetworkDrop() = {
|
||||
restoreIP()
|
||||
assert(new ProcessBuilder("ipfw", "add", "1", "deny", "tcp", "from", "any", "to", "any", PortRang).start.waitFor == 0)
|
||||
}
|
||||
|
||||
def enableTcpReset() = {
|
||||
restoreIP()
|
||||
assert(new ProcessBuilder("ipfw", "add", "1", "reset", "tcp", "from", "any", "to", "any", PortRang).start.waitFor == 0)
|
||||
}
|
||||
|
||||
def restoreIP() = {
|
||||
println("===>>> Restoring network")
|
||||
assert(new ProcessBuilder("ipfw", "del", "pipe", "1").start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "del", "pipe", "2").start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "flush").start.waitFor == 0)
|
||||
assert(new ProcessBuilder("ipfw", "pipe", "flush").start.waitFor == 0)
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,152 @@
|
|||
package akka.serialization
|
||||
|
||||
import org.scalatest.Spec
|
||||
import org.scalatest.matchers.ShouldMatchers
|
||||
import org.scalatest.BeforeAndAfterAll
|
||||
import org.scalatest.junit.JUnitRunner
|
||||
import org.junit.runner.RunWith
|
||||
|
||||
import com.google.protobuf.Message
|
||||
|
||||
import akka.serialization.ActorSerialization._
|
||||
import akka.actor._
|
||||
import Actor._
|
||||
import SerializeSpec._
|
||||
|
||||
case class MyMessage(id: Long, name: String, status: Boolean)
|
||||
|
||||
@RunWith(classOf[JUnitRunner])
|
||||
class ActorSerializeSpec extends Spec with ShouldMatchers with BeforeAndAfterAll {
|
||||
|
||||
describe("Serializable actor") {
|
||||
it("should be able to serialize and de-serialize a stateful actor with a given serializer") {
|
||||
|
||||
val actor1 = actorOf(Props[MyJavaSerializableActor].withLocalOnly(true)).asInstanceOf[LocalActorRef]
|
||||
(actor1 ? "hello").get should equal("world 1")
|
||||
(actor1 ? "hello").get should equal("world 2")
|
||||
|
||||
val bytes = toBinary(actor1)
|
||||
val actor2 = fromBinary(bytes).start().asInstanceOf[LocalActorRef]
|
||||
(actor2 ? "hello").get should equal("world 3")
|
||||
|
||||
actor2.receiveTimeout should equal(Some(1000))
|
||||
actor1.stop()
|
||||
actor2.stop()
|
||||
}
|
||||
|
||||
it("should be able to serialize and deserialize a MyStatelessActorWithMessagesInMailbox") {
|
||||
|
||||
val actor1 = actorOf(Props[MyStatelessActorWithMessagesInMailbox].withLocalOnly(true)).asInstanceOf[LocalActorRef]
|
||||
for (i ← 1 to 10) actor1 ! "hello"
|
||||
|
||||
actor1.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
val actor2 = fromBinary(toBinary(actor1)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor2.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
(actor2 ? "hello-reply").get should equal("world")
|
||||
|
||||
val actor3 = fromBinary(toBinary(actor1, false)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor3.getDispatcher.mailboxSize(actor1) should equal(0)
|
||||
(actor3 ? "hello-reply").get should equal("world")
|
||||
}
|
||||
|
||||
it("should be able to serialize and deserialize a PersonActorWithMessagesInMailbox") {
|
||||
|
||||
val p1 = Person("debasish ghosh", 25, SerializeSpec.Address("120", "Monroe Street", "Santa Clara", "95050"))
|
||||
val actor1 = actorOf(Props[PersonActorWithMessagesInMailbox].withLocalOnly(true)).asInstanceOf[LocalActorRef]
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
(actor1 ! p1)
|
||||
actor1.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
val actor2 = fromBinary(toBinary(actor1)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor2.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
(actor2 ? "hello-reply").get should equal("hello")
|
||||
|
||||
val actor3 = fromBinary(toBinary(actor1, false)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor3.getDispatcher.mailboxSize(actor1) should equal(0)
|
||||
(actor3 ? "hello-reply").get should equal("hello")
|
||||
}
|
||||
}
|
||||
|
||||
describe("serialize protobuf") {
|
||||
it("should serialize") {
|
||||
val msg = MyMessage(123, "debasish ghosh", true)
|
||||
import akka.serialization.Serialization._
|
||||
val b = serialize(ProtobufProtocol.MyMessage.newBuilder.setId(msg.id).setName(msg.name).setStatus(msg.status).build) match {
|
||||
case Left(exception) ⇒ fail(exception)
|
||||
case Right(bytes) ⇒ bytes
|
||||
}
|
||||
val in = deserialize(b, classOf[ProtobufProtocol.MyMessage], None) match {
|
||||
case Left(exception) ⇒ fail(exception)
|
||||
case Right(i) ⇒ i
|
||||
}
|
||||
val m = in.asInstanceOf[ProtobufProtocol.MyMessage]
|
||||
MyMessage(m.getId, m.getName, m.getStatus) should equal(msg)
|
||||
}
|
||||
}
|
||||
|
||||
describe("serialize actor that accepts protobuf message") {
|
||||
it("should serialize") {
|
||||
|
||||
val actor1 = actorOf(Props[MyActorWithProtobufMessagesInMailbox].withLocalOnly(true)).asInstanceOf[LocalActorRef]
|
||||
val msg = MyMessage(123, "debasish ghosh", true)
|
||||
val b = ProtobufProtocol.MyMessage.newBuilder.setId(msg.id).setName(msg.name).setStatus(msg.status).build
|
||||
for (i ← 1 to 10) actor1 ! b
|
||||
actor1.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
val actor2 = fromBinary(toBinary(actor1)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor2.getDispatcher.mailboxSize(actor1) should be > (0)
|
||||
(actor2 ? "hello-reply").get should equal("world")
|
||||
|
||||
val actor3 = fromBinary(toBinary(actor1, false)).asInstanceOf[LocalActorRef]
|
||||
Thread.sleep(1000)
|
||||
actor3.getDispatcher.mailboxSize(actor1) should equal(0)
|
||||
(actor3 ? "hello-reply").get should equal("world")
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
class MyJavaSerializableActor extends Actor with scala.Serializable {
|
||||
var count = 0
|
||||
self.receiveTimeout = Some(1000)
|
||||
|
||||
def receive = {
|
||||
case "hello" ⇒
|
||||
count = count + 1
|
||||
self.reply("world " + count)
|
||||
}
|
||||
}
|
||||
|
||||
class MyStatelessActorWithMessagesInMailbox extends Actor with scala.Serializable {
|
||||
def receive = {
|
||||
case "hello" ⇒
|
||||
Thread.sleep(500)
|
||||
case "hello-reply" ⇒ self.reply("world")
|
||||
}
|
||||
}
|
||||
|
||||
class MyActorWithProtobufMessagesInMailbox extends Actor with scala.Serializable {
|
||||
def receive = {
|
||||
case m: Message ⇒
|
||||
Thread.sleep(500)
|
||||
case "hello-reply" ⇒ self.reply("world")
|
||||
}
|
||||
}
|
||||
|
||||
class PersonActorWithMessagesInMailbox extends Actor with scala.Serializable {
|
||||
def receive = {
|
||||
case p: Person ⇒
|
||||
Thread.sleep(500)
|
||||
case "hello-reply" ⇒ self.reply("hello")
|
||||
}
|
||||
}
|
||||
|
|
@ -21,7 +21,7 @@ object AkkaBuild extends Build {
|
|||
Unidoc.unidocExclude := Seq(samples.id, tutorials.id),
|
||||
rstdocDirectory <<= baseDirectory / "akka-docs"
|
||||
),
|
||||
aggregate = Seq(actor, testkit, actorTests, stm, http, slf4j, camel, camelTyped, samples, tutorials)
|
||||
aggregate = Seq(actor, testkit, actorTests, stm, http, remote, slf4j, camel, camelTyped, samples, tutorials)
|
||||
//aggregate = Seq(actor, testkit, actorTests, stm, http, slf4j, cluster, mailboxes, camel, camelTyped, samples, tutorials)
|
||||
)
|
||||
|
||||
|
|
@ -65,6 +65,23 @@ object AkkaBuild extends Build {
|
|||
)
|
||||
)
|
||||
|
||||
lazy val remote = Project(
|
||||
id = "akka-remote",
|
||||
base = file("akka-remote"),
|
||||
dependencies = Seq(stm, actorTests % "test->test", testkit % "test"),
|
||||
settings = defaultSettings ++ multiJvmSettings ++ Seq(
|
||||
libraryDependencies ++= Dependencies.cluster,
|
||||
extraOptions in MultiJvm <<= (sourceDirectory in MultiJvm) { src =>
|
||||
(name: String) => (src ** (name + ".conf")).get.headOption.map("-Dakka.config=" + _.absolutePath).toSeq
|
||||
},
|
||||
scalatestOptions in MultiJvm := Seq("-r", "org.scalatest.akka.QuietReporter"),
|
||||
jvmOptions in MultiJvm := {
|
||||
if (getBoolean("sbt.log.noformat")) Seq("-Dakka.test.nocolor=true") else Nil
|
||||
},
|
||||
test in Test <<= (test in Test) dependsOn (test in MultiJvm)
|
||||
)
|
||||
) configs (MultiJvm)
|
||||
|
||||
// lazy val cluster = Project(
|
||||
// id = "akka-cluster",
|
||||
// base = file("akka-cluster"),
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue