#2469 - Switching to scala.util.Try instead of Either[Throwable, T] in the codebase
This commit is contained in:
parent
6e4b0dc3de
commit
4eee04cb60
21 changed files with 123 additions and 249 deletions
|
|
@ -260,7 +260,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
||||||
} yield b + "-" + c
|
} yield b + "-" + c
|
||||||
|
|
||||||
Await.result(future1, timeout.duration) must be("10-14")
|
Await.result(future1, timeout.duration) must be("10-14")
|
||||||
assert(checkType(future1, manifest[String]))
|
assert(checkType(future1, scala.reflect.classTag[String]))
|
||||||
intercept[ClassCastException] { Await.result(future2, timeout.duration) }
|
intercept[ClassCastException] { Await.result(future2, timeout.duration) }
|
||||||
system.stop(actor)
|
system.stop(actor)
|
||||||
}
|
}
|
||||||
|
|
@ -479,7 +479,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
||||||
}
|
}
|
||||||
}))
|
}))
|
||||||
|
|
||||||
val oddFutures = List.fill(100)(oddActor ? 'GetNext mapTo manifest[Int])
|
val oddFutures = List.fill(100)(oddActor ? 'GetNext mapTo scala.reflect.classTag[Int])
|
||||||
|
|
||||||
assert(Await.result(Future.sequence(oddFutures), timeout.duration).sum === 10000)
|
assert(Await.result(Future.sequence(oddFutures), timeout.duration).sum === 10000)
|
||||||
system.stop(oddActor)
|
system.stop(oddActor)
|
||||||
|
|
@ -939,9 +939,8 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
||||||
f((future, message) ⇒ {
|
f((future, message) ⇒ {
|
||||||
future.value must be('defined)
|
future.value must be('defined)
|
||||||
future.value.get must be('failure)
|
future.value.get must be('failure)
|
||||||
future.value.get match {
|
val Failure(f) = future.value.get
|
||||||
case Failure(f) ⇒ f.getMessage must be(message)
|
f.getMessage must be(message)
|
||||||
}
|
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
"throw exception with 'get'" in { f((future, message) ⇒ (evaluating { Await.result(future, timeout.duration) } must produce[java.lang.Exception]).getMessage must be(message)) }
|
"throw exception with 'get'" in { f((future, message) ⇒ (evaluating { Await.result(future, timeout.duration) } must produce[java.lang.Exception]).getMessage must be(message)) }
|
||||||
|
|
|
||||||
|
|
@ -87,50 +87,23 @@ class SerializeSpec extends AkkaSpec(SerializeSpec.config) {
|
||||||
}
|
}
|
||||||
|
|
||||||
"serialize Address" in {
|
"serialize Address" in {
|
||||||
val b = serialize(addr) match {
|
assert(deserialize(serialize(addr).get, classOf[Address]).get === addr)
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(bytes) ⇒ bytes
|
|
||||||
}
|
|
||||||
deserialize(b.asInstanceOf[Array[Byte]], classOf[Address]) match {
|
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(add) ⇒ assert(add === addr)
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
"serialize Person" in {
|
"serialize Person" in {
|
||||||
|
assert(deserialize(serialize(person).get, classOf[Person]).get === person)
|
||||||
val b = serialize(person) match {
|
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(bytes) ⇒ bytes
|
|
||||||
}
|
|
||||||
deserialize(b.asInstanceOf[Array[Byte]], classOf[Person]) match {
|
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(p) ⇒ assert(p === person)
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
"serialize record with default serializer" in {
|
"serialize record with default serializer" in {
|
||||||
|
|
||||||
val r = Record(100, person)
|
val r = Record(100, person)
|
||||||
val b = serialize(r) match {
|
assert(deserialize(serialize(r).get, classOf[Record]).get === r)
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(bytes) ⇒ bytes
|
|
||||||
}
|
|
||||||
deserialize(b.asInstanceOf[Array[Byte]], classOf[Record]) match {
|
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(p) ⇒ assert(p === r)
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
"not serialize ActorCell" in {
|
"not serialize ActorCell" in {
|
||||||
val a = system.actorOf(Props(new Actor {
|
val a = system.actorOf(Props(new Actor {
|
||||||
def receive = {
|
def receive = {
|
||||||
case o: ObjectOutputStream ⇒
|
case o: ObjectOutputStream ⇒
|
||||||
try {
|
try o.writeObject(this) catch { case _: NotSerializableException ⇒ testActor ! "pass" }
|
||||||
o.writeObject(this)
|
|
||||||
} catch {
|
|
||||||
case _: NotSerializableException ⇒ testActor ! "pass"
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
}))
|
}))
|
||||||
a ! new ObjectOutputStream(new ByteArrayOutputStream())
|
a ! new ObjectOutputStream(new ByteArrayOutputStream())
|
||||||
|
|
|
||||||
|
|
@ -413,7 +413,7 @@ class LocalActorRefProvider(
|
||||||
def registerExtraNames(_extras: Map[String, InternalActorRef]): Unit = extraNames ++= _extras
|
def registerExtraNames(_extras: Map[String, InternalActorRef]): Unit = extraNames ++= _extras
|
||||||
|
|
||||||
private def guardianSupervisorStrategyConfigurator =
|
private def guardianSupervisorStrategyConfigurator =
|
||||||
dynamicAccess.createInstanceFor[SupervisorStrategyConfigurator](settings.SupervisorStrategyClass, Seq()).fold(throw _, x ⇒ x)
|
dynamicAccess.createInstanceFor[SupervisorStrategyConfigurator](settings.SupervisorStrategyClass, Seq()).get
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Overridable supervision strategy to be used by the “/user” guardian.
|
* Overridable supervision strategy to be used by the “/user” guardian.
|
||||||
|
|
|
||||||
|
|
@ -19,6 +19,7 @@ import java.util.concurrent.{ ThreadFactory, CountDownLatch, TimeoutException, R
|
||||||
import java.util.concurrent.TimeUnit.MILLISECONDS
|
import java.util.concurrent.TimeUnit.MILLISECONDS
|
||||||
import akka.actor.cell.ChildrenContainer
|
import akka.actor.cell.ChildrenContainer
|
||||||
import scala.concurrent.util.FiniteDuration
|
import scala.concurrent.util.FiniteDuration
|
||||||
|
import util.{ Failure, Success }
|
||||||
|
|
||||||
object ActorSystem {
|
object ActorSystem {
|
||||||
|
|
||||||
|
|
@ -540,10 +541,7 @@ private[akka] class ActorSystemImpl(val name: String, applicationConfig: Config,
|
||||||
classOf[Scheduler] -> scheduler,
|
classOf[Scheduler] -> scheduler,
|
||||||
classOf[DynamicAccess] -> dynamicAccess)
|
classOf[DynamicAccess] -> dynamicAccess)
|
||||||
|
|
||||||
dynamicAccess.createInstanceFor[ActorRefProvider](ProviderClass, arguments) match {
|
dynamicAccess.createInstanceFor[ActorRefProvider](ProviderClass, arguments).get
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
case Right(p) ⇒ p
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
def deadLetters: ActorRef = provider.deadLetters
|
def deadLetters: ActorRef = provider.deadLetters
|
||||||
|
|
@ -678,13 +676,12 @@ private[akka] class ActorSystemImpl(val name: String, applicationConfig: Config,
|
||||||
private def loadExtensions() {
|
private def loadExtensions() {
|
||||||
import scala.collection.JavaConversions._
|
import scala.collection.JavaConversions._
|
||||||
settings.config.getStringList("akka.extensions") foreach { fqcn ⇒
|
settings.config.getStringList("akka.extensions") foreach { fqcn ⇒
|
||||||
dynamicAccess.getObjectFor[AnyRef](fqcn).fold(_ ⇒ dynamicAccess.createInstanceFor[AnyRef](fqcn, Seq()), Right(_)) match {
|
dynamicAccess.getObjectFor[AnyRef](fqcn) recoverWith { case _ ⇒ dynamicAccess.createInstanceFor[AnyRef](fqcn, Seq()) } match {
|
||||||
case Right(p: ExtensionIdProvider) ⇒ registerExtension(p.lookup());
|
case Success(p: ExtensionIdProvider) ⇒ registerExtension(p.lookup())
|
||||||
case Right(p: ExtensionId[_]) ⇒ registerExtension(p);
|
case Success(p: ExtensionId[_]) ⇒ registerExtension(p)
|
||||||
case Right(other) ⇒ log.error("[{}] is not an 'ExtensionIdProvider' or 'ExtensionId', skipping...", fqcn)
|
case Success(other) ⇒ log.error("[{}] is not an 'ExtensionIdProvider' or 'ExtensionId', skipping...", fqcn)
|
||||||
case Left(problem) ⇒ log.error(problem, "While trying to load extension [{}], skipping...", fqcn)
|
case Failure(problem) ⇒ log.error(problem, "While trying to load extension [{}], skipping...", fqcn)
|
||||||
}
|
}
|
||||||
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
||||||
|
|
@ -156,15 +156,13 @@ private[akka] class Deployer(val settings: ActorSystem.Settings, val dynamicAcce
|
||||||
case "broadcast" ⇒ BroadcastRouter(nrOfInstances, routees, resizer)
|
case "broadcast" ⇒ BroadcastRouter(nrOfInstances, routees, resizer)
|
||||||
case fqn ⇒
|
case fqn ⇒
|
||||||
val args = Seq(classOf[Config] -> deployment)
|
val args = Seq(classOf[Config] -> deployment)
|
||||||
dynamicAccess.createInstanceFor[RouterConfig](fqn, args) match {
|
dynamicAccess.createInstanceFor[RouterConfig](fqn, args).recover({
|
||||||
case Right(router) ⇒ router
|
case exception ⇒ throw new IllegalArgumentException(
|
||||||
case Left(exception) ⇒
|
|
||||||
throw new IllegalArgumentException(
|
|
||||||
("Cannot instantiate router [%s], defined in [%s], " +
|
("Cannot instantiate router [%s], defined in [%s], " +
|
||||||
"make sure it extends [akka.routing.RouterConfig] and has constructor with " +
|
"make sure it extends [akka.routing.RouterConfig] and has constructor with " +
|
||||||
"[com.typesafe.config.Config] parameter")
|
"[com.typesafe.config.Config] parameter")
|
||||||
.format(fqn, key), exception)
|
.format(fqn, key), exception)
|
||||||
}
|
}).get
|
||||||
}
|
}
|
||||||
|
|
||||||
Some(Deploy(key, deployment, router, NoScopeGiven))
|
Some(Deploy(key, deployment, router, NoScopeGiven))
|
||||||
|
|
|
||||||
|
|
@ -1,11 +1,12 @@
|
||||||
/**
|
/**
|
||||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
* CopySuccess (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||||
*/
|
*/
|
||||||
package akka.actor
|
package akka.actor
|
||||||
|
|
||||||
import scala.util.control.NonFatal
|
import scala.util.control.NonFatal
|
||||||
import java.lang.reflect.InvocationTargetException
|
import java.lang.reflect.InvocationTargetException
|
||||||
import scala.reflect.ClassTag
|
import scala.reflect.ClassTag
|
||||||
|
import scala.util.Try
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* The DynamicAccess implementation is the class which is used for
|
* The DynamicAccess implementation is the class which is used for
|
||||||
|
|
@ -16,7 +17,6 @@ import scala.reflect.ClassTag
|
||||||
* unless they are extending Akka in ways which go beyond simple Extensions.
|
* unless they are extending Akka in ways which go beyond simple Extensions.
|
||||||
*/
|
*/
|
||||||
abstract class DynamicAccess {
|
abstract class DynamicAccess {
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Convenience method which given a `Class[_]` object and a constructor description
|
* Convenience method which given a `Class[_]` object and a constructor description
|
||||||
* will create a new instance of that class.
|
* will create a new instance of that class.
|
||||||
|
|
@ -25,23 +25,13 @@ abstract class DynamicAccess {
|
||||||
* val obj = DynamicAccess.createInstanceFor(clazz, Seq(classOf[Config] -> config, classOf[String] -> name))
|
* val obj = DynamicAccess.createInstanceFor(clazz, Seq(classOf[Config] -> config, classOf[String] -> name))
|
||||||
* }}}
|
* }}}
|
||||||
*/
|
*/
|
||||||
def createInstanceFor[T: ClassTag](clazz: Class[_], args: Seq[(Class[_], AnyRef)]): Either[Throwable, T] = {
|
def createInstanceFor[T: ClassTag](clazz: Class[_], args: Seq[(Class[_], AnyRef)]): Try[T]
|
||||||
val types = args.map(_._1).toArray
|
|
||||||
val values = args.map(_._2).toArray
|
|
||||||
withErrorHandling {
|
|
||||||
val constructor = clazz.getDeclaredConstructor(types: _*)
|
|
||||||
constructor.setAccessible(true)
|
|
||||||
val obj = constructor.newInstance(values: _*).asInstanceOf[T]
|
|
||||||
val t = implicitly[ClassTag[T]].runtimeClass
|
|
||||||
if (t.isInstance(obj)) Right(obj) else Left(new ClassCastException(clazz + " is not a subtype of " + t))
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Obtain a `Class[_]` object loaded with the right class loader (i.e. the one
|
* Obtain a `Class[_]` object loaded with the Success class loader (i.e. the one
|
||||||
* returned by `classLoader`).
|
* returned by `classLoader`).
|
||||||
*/
|
*/
|
||||||
def getClassFor[T: ClassTag](fqcn: String): Either[Throwable, Class[_ <: T]]
|
def getClassFor[T: ClassTag](fqcn: String): Try[Class[_ <: T]]
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Obtain an object conforming to the type T, which is expected to be
|
* Obtain an object conforming to the type T, which is expected to be
|
||||||
|
|
@ -50,35 +40,18 @@ abstract class DynamicAccess {
|
||||||
* `args` argument. The exact usage of args depends on which type is requested,
|
* `args` argument. The exact usage of args depends on which type is requested,
|
||||||
* see the relevant requesting code for details.
|
* see the relevant requesting code for details.
|
||||||
*/
|
*/
|
||||||
def createInstanceFor[T: ClassTag](fqcn: String, args: Seq[(Class[_], AnyRef)]): Either[Throwable, T]
|
def createInstanceFor[T: ClassTag](fqcn: String, args: Seq[(Class[_], AnyRef)]): Try[T]
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Obtain the Scala “object” instance for the given fully-qualified class name, if there is one.
|
* Obtain the Scala “object” instance for the given fully-qualified class name, if there is one.
|
||||||
*/
|
*/
|
||||||
def getObjectFor[T: ClassTag](fqcn: String): Either[Throwable, T]
|
def getObjectFor[T: ClassTag](fqcn: String): Try[T]
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* This is the class loader to be used in those special cases where the
|
* This is the class loader to be used in those special cases where the
|
||||||
* other factory method are not applicable (e.g. when constructing a ClassLoaderBinaryInputStream).
|
* other factory method are not applicable (e.g. when constructing a ClassLoaderBinaryInputStream).
|
||||||
*/
|
*/
|
||||||
def classLoader: ClassLoader
|
def classLoader: ClassLoader
|
||||||
|
|
||||||
/**
|
|
||||||
* Caught exception is returned as Left(exception).
|
|
||||||
* Unwraps `InvocationTargetException` if its getTargetException is an `Exception`.
|
|
||||||
* Other `Throwable`, such as `Error` is thrown.
|
|
||||||
*/
|
|
||||||
@inline
|
|
||||||
final def withErrorHandling[T](body: ⇒ Either[Throwable, T]): Either[Throwable, T] =
|
|
||||||
try body catch {
|
|
||||||
case e: InvocationTargetException ⇒
|
|
||||||
e.getTargetException match {
|
|
||||||
case NonFatal(t) ⇒ Left(t)
|
|
||||||
case t ⇒ throw t
|
|
||||||
}
|
|
||||||
case NonFatal(e) ⇒ Left(e)
|
|
||||||
}
|
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
@ -89,42 +62,41 @@ abstract class DynamicAccess {
|
||||||
* by default.
|
* by default.
|
||||||
*/
|
*/
|
||||||
class ReflectiveDynamicAccess(val classLoader: ClassLoader) extends DynamicAccess {
|
class ReflectiveDynamicAccess(val classLoader: ClassLoader) extends DynamicAccess {
|
||||||
//FIXME switch to Scala Reflection for 2.10
|
|
||||||
override def getClassFor[T: ClassTag](fqcn: String): Either[Throwable, Class[_ <: T]] =
|
override def getClassFor[T: ClassTag](fqcn: String): Try[Class[_ <: T]] =
|
||||||
try {
|
Try[Class[_ <: T]]({
|
||||||
val c = classLoader.loadClass(fqcn).asInstanceOf[Class[_ <: T]]
|
val c = classLoader.loadClass(fqcn).asInstanceOf[Class[_ <: T]]
|
||||||
val t = implicitly[ClassTag[T]].runtimeClass
|
val t = implicitly[ClassTag[T]].runtimeClass
|
||||||
if (t.isAssignableFrom(c)) Right(c) else Left(new ClassCastException(t + " is not assignable from " + c))
|
if (t.isAssignableFrom(c)) c else throw new ClassCastException(t + " is not assignable from " + c)
|
||||||
} catch {
|
})
|
||||||
case NonFatal(e) ⇒ Left(e)
|
|
||||||
}
|
|
||||||
|
|
||||||
override def createInstanceFor[T: ClassTag](fqcn: String, args: Seq[(Class[_], AnyRef)]): Either[Throwable, T] =
|
override def createInstanceFor[T: ClassTag](clazz: Class[_], args: Seq[(Class[_], AnyRef)]): Try[T] =
|
||||||
getClassFor(fqcn).fold(Left(_), { c ⇒
|
Try {
|
||||||
val types = args.map(_._1).toArray
|
val types = args.map(_._1).toArray
|
||||||
val values = args.map(_._2).toArray
|
val values = args.map(_._2).toArray
|
||||||
withErrorHandling {
|
val constructor = clazz.getDeclaredConstructor(types: _*)
|
||||||
val constructor = c.getDeclaredConstructor(types: _*)
|
|
||||||
constructor.setAccessible(true)
|
constructor.setAccessible(true)
|
||||||
val obj = constructor.newInstance(values: _*)
|
val obj = constructor.newInstance(values: _*)
|
||||||
val t = implicitly[ClassTag[T]].runtimeClass
|
val t = implicitly[ClassTag[T]].runtimeClass
|
||||||
if (t.isInstance(obj)) Right(obj) else Left(new ClassCastException(fqcn + " is not a subtype of " + t))
|
if (t.isInstance(obj)) obj.asInstanceOf[T] else throw new ClassCastException(clazz.getName + " is not a subtype of " + t)
|
||||||
}
|
} recover { case i: InvocationTargetException if i.getTargetException ne null ⇒ throw i.getTargetException }
|
||||||
})
|
|
||||||
|
|
||||||
override def getObjectFor[T: ClassTag](fqcn: String): Either[Throwable, T] = {
|
override def createInstanceFor[T: ClassTag](fqcn: String, args: Seq[(Class[_], AnyRef)]): Try[T] =
|
||||||
getClassFor(fqcn).fold(Left(_), { c ⇒
|
getClassFor(fqcn) flatMap { c ⇒ createInstanceFor(c, args) }
|
||||||
withErrorHandling {
|
|
||||||
|
override def getObjectFor[T: ClassTag](fqcn: String): Try[T] = {
|
||||||
|
getClassFor(fqcn) flatMap { c ⇒
|
||||||
|
Try {
|
||||||
val module = c.getDeclaredField("MODULE$")
|
val module = c.getDeclaredField("MODULE$")
|
||||||
module.setAccessible(true)
|
module.setAccessible(true)
|
||||||
val t = implicitly[ClassTag[T]].runtimeClass
|
val t = implicitly[ClassTag[T]].runtimeClass
|
||||||
module.get(null) match {
|
module.get(null) match {
|
||||||
case null ⇒ Left(new NullPointerException)
|
case null ⇒ throw new NullPointerException
|
||||||
case x if !t.isInstance(x) ⇒ Left(new ClassCastException(fqcn + " is not a subtype of " + t))
|
case x if !t.isInstance(x) ⇒ throw new ClassCastException(fqcn + " is not a subtype of " + t)
|
||||||
case x ⇒ Right(x.asInstanceOf[T])
|
case x: T ⇒ x
|
||||||
}
|
}
|
||||||
|
} recover { case i: InvocationTargetException if i.getTargetException ne null ⇒ throw i.getTargetException }
|
||||||
}
|
}
|
||||||
})
|
|
||||||
}
|
}
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
@ -98,9 +98,5 @@ abstract class ExtensionKey[T <: Extension](implicit m: ClassTag[T]) extends Ext
|
||||||
def this(clazz: Class[T]) = this()(ClassTag(clazz))
|
def this(clazz: Class[T]) = this()(ClassTag(clazz))
|
||||||
|
|
||||||
override def lookup(): ExtensionId[T] = this
|
override def lookup(): ExtensionId[T] = this
|
||||||
def createExtension(system: ExtendedActorSystem): T =
|
def createExtension(system: ExtendedActorSystem): T = system.dynamicAccess.createInstanceFor[T](m.runtimeClass, Seq(classOf[ExtendedActorSystem] -> system)).get
|
||||||
system.dynamicAccess.createInstanceFor[T](m.runtimeClass, Seq(classOf[ExtendedActorSystem] -> system)) match {
|
|
||||||
case Left(ex) ⇒ throw ex
|
|
||||||
case Right(r) ⇒ r
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -170,13 +170,7 @@ private[akka] trait Children { this: ActorCell ⇒
|
||||||
private def makeChild(cell: ActorCell, props: Props, name: String, async: Boolean, systemService: Boolean): ActorRef = {
|
private def makeChild(cell: ActorCell, props: Props, name: String, async: Boolean, systemService: Boolean): ActorRef = {
|
||||||
if (cell.system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) {
|
if (cell.system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) {
|
||||||
val ser = SerializationExtension(cell.system)
|
val ser = SerializationExtension(cell.system)
|
||||||
ser.serialize(props.creator) match {
|
ser.deserialize(ser.serialize(props.creator).get, props.creator.getClass).get
|
||||||
case Left(t) ⇒ throw t
|
|
||||||
case Right(bytes) ⇒ ser.deserialize(bytes, props.creator.getClass) match {
|
|
||||||
case Left(t) ⇒ throw t
|
|
||||||
case _ ⇒ //All good
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
/*
|
/*
|
||||||
* in case we are currently terminating, fail external attachChild requests
|
* in case we are currently terminating, fail external attachChild requests
|
||||||
|
|
|
||||||
|
|
@ -25,13 +25,7 @@ object Envelope {
|
||||||
if (msg eq null) throw new InvalidMessageException("Message is null")
|
if (msg eq null) throw new InvalidMessageException("Message is null")
|
||||||
if (system.settings.SerializeAllMessages && !msg.isInstanceOf[NoSerializationVerificationNeeded]) {
|
if (system.settings.SerializeAllMessages && !msg.isInstanceOf[NoSerializationVerificationNeeded]) {
|
||||||
val ser = SerializationExtension(system)
|
val ser = SerializationExtension(system)
|
||||||
ser.serialize(msg) match { //Verify serializability
|
ser.deserialize(ser.serialize(msg).get, msg.getClass).get
|
||||||
case Left(t) ⇒ throw t
|
|
||||||
case Right(bytes) ⇒ ser.deserialize(bytes, msg.getClass) match { //Verify deserializability
|
|
||||||
case Left(t) ⇒ throw t
|
|
||||||
case _ ⇒ //All good
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
new Envelope(message, sender)
|
new Envelope(message, sender)
|
||||||
}
|
}
|
||||||
|
|
@ -426,14 +420,13 @@ abstract class MessageDispatcherConfigurator(val config: Config, val prerequisit
|
||||||
case "bounded" ⇒ new BoundedMailbox(prerequisites.settings, config)
|
case "bounded" ⇒ new BoundedMailbox(prerequisites.settings, config)
|
||||||
case fqcn ⇒
|
case fqcn ⇒
|
||||||
val args = Seq(classOf[ActorSystem.Settings] -> prerequisites.settings, classOf[Config] -> config)
|
val args = Seq(classOf[ActorSystem.Settings] -> prerequisites.settings, classOf[Config] -> config)
|
||||||
prerequisites.dynamicAccess.createInstanceFor[MailboxType](fqcn, args) match {
|
prerequisites.dynamicAccess.createInstanceFor[MailboxType](fqcn, args).recover({
|
||||||
case Right(instance) ⇒ instance
|
case exception ⇒
|
||||||
case Left(exception) ⇒
|
|
||||||
throw new IllegalArgumentException(
|
throw new IllegalArgumentException(
|
||||||
("Cannot instantiate MailboxType [%s], defined in [%s], " +
|
("Cannot instantiate MailboxType [%s], defined in [%s], " +
|
||||||
"make sure it has constructor with [akka.actor.ActorSystem.Settings, com.typesafe.config.Config] parameters")
|
"make sure it has constructor with [akka.actor.ActorSystem.Settings, com.typesafe.config.Config] parameters")
|
||||||
.format(fqcn, config.getString("id")), exception)
|
.format(fqcn, config.getString("id")), exception)
|
||||||
}
|
}).get
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
@ -445,13 +438,12 @@ abstract class MessageDispatcherConfigurator(val config: Config, val prerequisit
|
||||||
val args = Seq(
|
val args = Seq(
|
||||||
classOf[Config] -> config,
|
classOf[Config] -> config,
|
||||||
classOf[DispatcherPrerequisites] -> prerequisites)
|
classOf[DispatcherPrerequisites] -> prerequisites)
|
||||||
prerequisites.dynamicAccess.createInstanceFor[ExecutorServiceConfigurator](fqcn, args) match {
|
prerequisites.dynamicAccess.createInstanceFor[ExecutorServiceConfigurator](fqcn, args).recover({
|
||||||
case Right(instance) ⇒ instance
|
case exception ⇒ throw new IllegalArgumentException(
|
||||||
case Left(exception) ⇒ throw new IllegalArgumentException(
|
|
||||||
("""Cannot instantiate ExecutorServiceConfigurator ("executor = [%s]"), defined in [%s],
|
("""Cannot instantiate ExecutorServiceConfigurator ("executor = [%s]"), defined in [%s],
|
||||||
make sure it has an accessible constructor with a [%s,%s] signature""")
|
make sure it has an accessible constructor with a [%s,%s] signature""")
|
||||||
.format(fqcn, config.getString("id"), classOf[Config], classOf[DispatcherPrerequisites]), exception)
|
.format(fqcn, config.getString("id"), classOf[Config], classOf[DispatcherPrerequisites]), exception)
|
||||||
}
|
}).get
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -148,15 +148,14 @@ class Dispatchers(val settings: ActorSystem.Settings, val prerequisites: Dispatc
|
||||||
case "PinnedDispatcher" ⇒ new PinnedDispatcherConfigurator(cfg, prerequisites)
|
case "PinnedDispatcher" ⇒ new PinnedDispatcherConfigurator(cfg, prerequisites)
|
||||||
case fqn ⇒
|
case fqn ⇒
|
||||||
val args = Seq(classOf[Config] -> cfg, classOf[DispatcherPrerequisites] -> prerequisites)
|
val args = Seq(classOf[Config] -> cfg, classOf[DispatcherPrerequisites] -> prerequisites)
|
||||||
prerequisites.dynamicAccess.createInstanceFor[MessageDispatcherConfigurator](fqn, args) match {
|
prerequisites.dynamicAccess.createInstanceFor[MessageDispatcherConfigurator](fqn, args).recover({
|
||||||
case Right(configurator) ⇒ configurator
|
case exception ⇒
|
||||||
case Left(exception) ⇒
|
|
||||||
throw new IllegalArgumentException(
|
throw new IllegalArgumentException(
|
||||||
("Cannot instantiate MessageDispatcherConfigurator type [%s], defined in [%s], " +
|
("Cannot instantiate MessageDispatcherConfigurator type [%s], defined in [%s], " +
|
||||||
"make sure it has constructor with [com.typesafe.config.Config] and " +
|
"make sure it has constructor with [com.typesafe.config.Config] and " +
|
||||||
"[akka.dispatch.DispatcherPrerequisites] parameters")
|
"[akka.dispatch.DispatcherPrerequisites] parameters")
|
||||||
.format(fqn, cfg.getString("id")), exception)
|
.format(fqn, cfg.getString("id")), exception)
|
||||||
}
|
}).get
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -101,17 +101,13 @@ trait LoggingBus extends ActorEventBus {
|
||||||
loggerName ← defaultLoggers
|
loggerName ← defaultLoggers
|
||||||
if loggerName != StandardOutLogger.getClass.getName
|
if loggerName != StandardOutLogger.getClass.getName
|
||||||
} yield {
|
} yield {
|
||||||
try {
|
system.dynamicAccess.getClassFor[Actor](loggerName).map({
|
||||||
system.dynamicAccess.getClassFor[Actor](loggerName) match {
|
case actorClass ⇒ addLogger(system, actorClass, level, logName)
|
||||||
case Right(actorClass) ⇒ addLogger(system, actorClass, level, logName)
|
}).recover({
|
||||||
case Left(exception) ⇒ throw exception
|
case e ⇒ throw new ConfigurationException(
|
||||||
}
|
|
||||||
} catch {
|
|
||||||
case e: Exception ⇒
|
|
||||||
throw new ConfigurationException(
|
|
||||||
"Event Handler specified in config can't be loaded [" + loggerName +
|
"Event Handler specified in config can't be loaded [" + loggerName +
|
||||||
"] due to [" + e.toString + "]", e)
|
"] due to [" + e.toString + "]", e)
|
||||||
}
|
}).get
|
||||||
}
|
}
|
||||||
guard.withGuard {
|
guard.withGuard {
|
||||||
loggers = myloggers
|
loggers = myloggers
|
||||||
|
|
|
||||||
|
|
@ -5,7 +5,6 @@
|
||||||
package akka.serialization
|
package akka.serialization
|
||||||
|
|
||||||
import akka.AkkaException
|
import akka.AkkaException
|
||||||
import scala.util.DynamicVariable
|
|
||||||
import com.typesafe.config.Config
|
import com.typesafe.config.Config
|
||||||
import akka.actor.{ Extension, ExtendedActorSystem, Address, DynamicAccess }
|
import akka.actor.{ Extension, ExtendedActorSystem, Address, DynamicAccess }
|
||||||
import akka.event.Logging
|
import akka.event.Logging
|
||||||
|
|
@ -13,6 +12,7 @@ import java.util.concurrent.ConcurrentHashMap
|
||||||
import scala.util.control.NonFatal
|
import scala.util.control.NonFatal
|
||||||
import scala.collection.mutable.ArrayBuffer
|
import scala.collection.mutable.ArrayBuffer
|
||||||
import java.io.NotSerializableException
|
import java.io.NotSerializableException
|
||||||
|
import util.{ Try, DynamicVariable }
|
||||||
|
|
||||||
object Serialization {
|
object Serialization {
|
||||||
|
|
||||||
|
|
@ -56,9 +56,7 @@ class Serialization(val system: ExtendedActorSystem) extends Extension {
|
||||||
* Serializes the given AnyRef/java.lang.Object according to the Serialization configuration
|
* Serializes the given AnyRef/java.lang.Object according to the Serialization configuration
|
||||||
* to either an Array of Bytes or an Exception if one was thrown.
|
* to either an Array of Bytes or an Exception if one was thrown.
|
||||||
*/
|
*/
|
||||||
def serialize(o: AnyRef): Either[Throwable, Array[Byte]] =
|
def serialize(o: AnyRef): Try[Array[Byte]] = Try(findSerializerFor(o).toBinary(o))
|
||||||
try Right(findSerializerFor(o).toBinary(o))
|
|
||||||
catch { case NonFatal(e) ⇒ Left(e) }
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Deserializes the given array of bytes using the specified serializer id,
|
* Deserializes the given array of bytes using the specified serializer id,
|
||||||
|
|
@ -67,18 +65,14 @@ class Serialization(val system: ExtendedActorSystem) extends Extension {
|
||||||
*/
|
*/
|
||||||
def deserialize(bytes: Array[Byte],
|
def deserialize(bytes: Array[Byte],
|
||||||
serializerId: Int,
|
serializerId: Int,
|
||||||
clazz: Option[Class[_]]): Either[Throwable, AnyRef] =
|
clazz: Option[Class[_]]): Try[AnyRef] = Try(serializerByIdentity(serializerId).fromBinary(bytes, clazz))
|
||||||
try Right(serializerByIdentity(serializerId).fromBinary(bytes, clazz))
|
|
||||||
catch { case NonFatal(e) ⇒ Left(e) }
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Deserializes the given array of bytes using the specified type to look up what Serializer should be used.
|
* Deserializes the given array of bytes using the specified type to look up what Serializer should be used.
|
||||||
* You can specify an optional ClassLoader to load the object into.
|
* You can specify an optional ClassLoader to load the object into.
|
||||||
* Returns either the resulting object or an Exception if one was thrown.
|
* Returns either the resulting object or an Exception if one was thrown.
|
||||||
*/
|
*/
|
||||||
def deserialize(bytes: Array[Byte], clazz: Class[_]): Either[Throwable, AnyRef] =
|
def deserialize(bytes: Array[Byte], clazz: Class[_]): Try[AnyRef] = Try(serializerFor(clazz).fromBinary(bytes, Some(clazz)))
|
||||||
try Right(serializerFor(clazz).fromBinary(bytes, Some(clazz)))
|
|
||||||
catch { case NonFatal(e) ⇒ Left(e) }
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Returns the Serializer configured for the given object, returns the NullSerializer if it's null.
|
* Returns the Serializer configured for the given object, returns the NullSerializer if it's null.
|
||||||
|
|
@ -128,28 +122,24 @@ class Serialization(val system: ExtendedActorSystem) extends Extension {
|
||||||
* Tries to load the specified Serializer by the fully-qualified name; the actual
|
* Tries to load the specified Serializer by the fully-qualified name; the actual
|
||||||
* loading is performed by the system’s [[akka.actor.DynamicAccess]].
|
* loading is performed by the system’s [[akka.actor.DynamicAccess]].
|
||||||
*/
|
*/
|
||||||
def serializerOf(serializerFQN: String): Either[Throwable, Serializer] =
|
def serializerOf(serializerFQN: String): Try[Serializer] =
|
||||||
system.dynamicAccess.createInstanceFor[Serializer](serializerFQN, Seq(classOf[ExtendedActorSystem] -> system)).fold(_ ⇒
|
system.dynamicAccess.createInstanceFor[Serializer](serializerFQN, Seq(classOf[ExtendedActorSystem] -> system)) recoverWith {
|
||||||
system.dynamicAccess.createInstanceFor[Serializer](serializerFQN, Seq()), Right(_))
|
case _ ⇒ system.dynamicAccess.createInstanceFor[Serializer](serializerFQN, Seq())
|
||||||
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* A Map of serializer from alias to implementation (class implementing akka.serialization.Serializer)
|
* A Map of serializer from alias to implementation (class implementing akka.serialization.Serializer)
|
||||||
* By default always contains the following mapping: "java" -> akka.serialization.JavaSerializer
|
* By default always contains the following mapping: "java" -> akka.serialization.JavaSerializer
|
||||||
*/
|
*/
|
||||||
private val serializers: Map[String, Serializer] =
|
private val serializers: Map[String, Serializer] =
|
||||||
for ((k: String, v: String) ← settings.Serializers) yield k -> serializerOf(v).fold(throw _, identity)
|
for ((k: String, v: String) ← settings.Serializers) yield k -> serializerOf(v).get
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* bindings is a Seq of tuple representing the mapping from Class to Serializer.
|
* bindings is a Seq of tuple representing the mapping from Class to Serializer.
|
||||||
* It is primarily ordered by the most specific classes first, and secondly in the configured order.
|
* It is primarily ordered by the most specific classes first, and secondly in the configured order.
|
||||||
*/
|
*/
|
||||||
private[akka] val bindings: Seq[ClassSerializer] = {
|
private[akka] val bindings: Seq[ClassSerializer] =
|
||||||
val configuredBindings = for ((k: String, v: String) ← settings.SerializationBindings if v != "none") yield {
|
sort(for ((k: String, v: String) ← settings.SerializationBindings if v != "none") yield (system.dynamicAccess.getClassFor[Any](k).get, serializers(v)))
|
||||||
val c = system.dynamicAccess.getClassFor[Any](k).fold(throw _, identity[Class[_]])
|
|
||||||
(c, serializers(v))
|
|
||||||
}
|
|
||||||
sort(configuredBindings)
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Sort so that subtypes always precede their supertypes, but without
|
* Sort so that subtypes always precede their supertypes, but without
|
||||||
|
|
|
||||||
|
|
@ -46,9 +46,9 @@ object Cluster extends ExtensionId[Cluster] with ExtensionIdProvider {
|
||||||
val failureDetector = {
|
val failureDetector = {
|
||||||
import clusterSettings.{ FailureDetectorImplementationClass ⇒ fqcn }
|
import clusterSettings.{ FailureDetectorImplementationClass ⇒ fqcn }
|
||||||
system.dynamicAccess.createInstanceFor[FailureDetector](
|
system.dynamicAccess.createInstanceFor[FailureDetector](
|
||||||
fqcn, Seq(classOf[ActorSystem] -> system, classOf[ClusterSettings] -> clusterSettings)).fold(
|
fqcn, Seq(classOf[ActorSystem] -> system, classOf[ClusterSettings] -> clusterSettings)).recover({
|
||||||
e ⇒ throw new ConfigurationException("Could not create custom failure detector [" + fqcn + "] due to:" + e.toString),
|
case e ⇒ throw new ConfigurationException("Could not create custom failure detector [" + fqcn + "] due to:" + e.toString)
|
||||||
identity)
|
}).get
|
||||||
}
|
}
|
||||||
|
|
||||||
new Cluster(system, failureDetector)
|
new Cluster(system, failureDetector)
|
||||||
|
|
|
||||||
|
|
@ -372,8 +372,8 @@ class ActorDocSpec extends AkkaSpec(Map("akka.loglevel" -> "INFO")) {
|
||||||
val f: Future[Result] =
|
val f: Future[Result] =
|
||||||
for {
|
for {
|
||||||
x ← ask(actorA, Request).mapTo[Int] // call pattern directly
|
x ← ask(actorA, Request).mapTo[Int] // call pattern directly
|
||||||
s ← actorB ask Request mapTo manifest[String] // call by implicit conversion
|
s ← (actorB ask Request).mapTo[String] // call by implicit conversion
|
||||||
d ← actorC ? Request mapTo manifest[Double] // call by symbolic name
|
d ← (actorC ? Request).mapTo[Double] // call by symbolic name
|
||||||
} yield Result(x, s, d)
|
} yield Result(x, s, d)
|
||||||
|
|
||||||
f pipeTo actorD // .. or ..
|
f pipeTo actorD // .. or ..
|
||||||
|
|
|
||||||
|
|
@ -49,12 +49,12 @@ object ZeromqDocSpec {
|
||||||
val timestamp = System.currentTimeMillis
|
val timestamp = System.currentTimeMillis
|
||||||
|
|
||||||
// use akka SerializationExtension to convert to bytes
|
// use akka SerializationExtension to convert to bytes
|
||||||
val heapPayload = ser.serialize(Heap(timestamp, currentHeap.getUsed, currentHeap.getMax)).fold(throw _, identity)
|
val heapPayload = ser.serialize(Heap(timestamp, currentHeap.getUsed, currentHeap.getMax)).get
|
||||||
// the first frame is the topic, second is the message
|
// the first frame is the topic, second is the message
|
||||||
pubSocket ! ZMQMessage(Seq(Frame("health.heap"), Frame(heapPayload)))
|
pubSocket ! ZMQMessage(Seq(Frame("health.heap"), Frame(heapPayload)))
|
||||||
|
|
||||||
// use akka SerializationExtension to convert to bytes
|
// use akka SerializationExtension to convert to bytes
|
||||||
val loadPayload = ser.serialize(Load(timestamp, os.getSystemLoadAverage)).fold(throw _, identity)
|
val loadPayload = ser.serialize(Load(timestamp, os.getSystemLoadAverage)).get
|
||||||
// the first frame is the topic, second is the message
|
// the first frame is the topic, second is the message
|
||||||
pubSocket ! ZMQMessage(Seq(Frame("health.load"), Frame(loadPayload)))
|
pubSocket ! ZMQMessage(Seq(Frame("health.load"), Frame(loadPayload)))
|
||||||
}
|
}
|
||||||
|
|
@ -71,18 +71,12 @@ object ZeromqDocSpec {
|
||||||
def receive = {
|
def receive = {
|
||||||
// the first frame is the topic, second is the message
|
// the first frame is the topic, second is the message
|
||||||
case m: ZMQMessage if m.firstFrameAsString == "health.heap" ⇒
|
case m: ZMQMessage if m.firstFrameAsString == "health.heap" ⇒
|
||||||
ser.deserialize(m.payload(1), classOf[Heap]) match {
|
val Heap(timestamp, used, max) = ser.deserialize(m.payload(1), classOf[Heap]).get
|
||||||
case Right(Heap(timestamp, used, max)) ⇒
|
|
||||||
log.info("Used heap {} bytes, at {}", used, timestampFormat.format(new Date(timestamp)))
|
log.info("Used heap {} bytes, at {}", used, timestampFormat.format(new Date(timestamp)))
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
}
|
|
||||||
|
|
||||||
case m: ZMQMessage if m.firstFrameAsString == "health.load" ⇒
|
case m: ZMQMessage if m.firstFrameAsString == "health.load" ⇒
|
||||||
ser.deserialize(m.payload(1), classOf[Load]) match {
|
val Load(timestamp, loadAverage) = ser.deserialize(m.payload(1), classOf[Load]).get
|
||||||
case Right(Load(timestamp, loadAverage)) ⇒
|
|
||||||
log.info("Load average {}, at {}", loadAverage, timestampFormat.format(new Date(timestamp)))
|
log.info("Load average {}, at {}", loadAverage, timestampFormat.format(new Date(timestamp)))
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
//#logger
|
//#logger
|
||||||
|
|
@ -97,13 +91,10 @@ object ZeromqDocSpec {
|
||||||
def receive = {
|
def receive = {
|
||||||
// the first frame is the topic, second is the message
|
// the first frame is the topic, second is the message
|
||||||
case m: ZMQMessage if m.firstFrameAsString == "health.heap" ⇒
|
case m: ZMQMessage if m.firstFrameAsString == "health.heap" ⇒
|
||||||
ser.deserialize(m.payload(1), classOf[Heap]) match {
|
val Heap(timestamp, used, max) = ser.deserialize(m.payload(1), classOf[Heap]).get
|
||||||
case Right(Heap(timestamp, used, max)) ⇒
|
|
||||||
if ((used.toDouble / max) > 0.9) count += 1
|
if ((used.toDouble / max) > 0.9) count += 1
|
||||||
else count = 0
|
else count = 0
|
||||||
if (count > 10) log.warning("Need more memory, using {} %", (100.0 * used / max))
|
if (count > 10) log.warning("Need more memory, using {} %", (100.0 * used / max))
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
//#alerter
|
//#alerter
|
||||||
|
|
|
||||||
|
|
@ -20,16 +20,10 @@ private[akka] object MessageSerializer {
|
||||||
* Uses Akka Serialization for the specified ActorSystem to transform the given MessageProtocol to a message
|
* Uses Akka Serialization for the specified ActorSystem to transform the given MessageProtocol to a message
|
||||||
*/
|
*/
|
||||||
def deserialize(system: ExtendedActorSystem, messageProtocol: MessageProtocol): AnyRef = {
|
def deserialize(system: ExtendedActorSystem, messageProtocol: MessageProtocol): AnyRef = {
|
||||||
val clazz =
|
SerializationExtension(system).deserialize(
|
||||||
if (messageProtocol.hasMessageManifest) {
|
messageProtocol.getMessage.toByteArray,
|
||||||
system.dynamicAccess.getClassFor[AnyRef](messageProtocol.getMessageManifest.toStringUtf8)
|
messageProtocol.getSerializerId,
|
||||||
.fold(throw _, Some(_))
|
if (messageProtocol.hasMessageManifest) Some(system.dynamicAccess.getClassFor[AnyRef](messageProtocol.getMessageManifest.toStringUtf8).get) else None).get
|
||||||
} else None
|
|
||||||
SerializationExtension(system)
|
|
||||||
.deserialize(messageProtocol.getMessage.toByteArray, messageProtocol.getSerializerId, clazz) match {
|
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
case Right(r) ⇒ r
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
|
||||||
|
|
@ -72,10 +72,9 @@ class RemoteActorRefProvider(
|
||||||
classOf[ExtendedActorSystem] -> system,
|
classOf[ExtendedActorSystem] -> system,
|
||||||
classOf[RemoteActorRefProvider] -> this)
|
classOf[RemoteActorRefProvider] -> this)
|
||||||
|
|
||||||
system.dynamicAccess.createInstanceFor[RemoteTransport](fqn, args) match {
|
system.dynamicAccess.createInstanceFor[RemoteTransport](fqn, args).recover({
|
||||||
case Left(problem) ⇒ throw new RemoteTransportException("Could not load remote transport layer " + fqn, problem)
|
case problem ⇒ throw new RemoteTransportException("Could not load remote transport layer " + fqn, problem)
|
||||||
case Right(remote) ⇒ remote
|
}).get
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
_log = Logging(eventStream, "RemoteActorRefProvider(" + transport.address + ")")
|
_log = Logging(eventStream, "RemoteActorRefProvider(" + transport.address + ")")
|
||||||
|
|
|
||||||
|
|
@ -13,6 +13,7 @@ import akka.remote.RemoteProtocol.{ DaemonMsgCreateProtocol, DeployProtocol, Pro
|
||||||
import akka.routing.{ NoRouter, RouterConfig }
|
import akka.routing.{ NoRouter, RouterConfig }
|
||||||
import akka.actor.FromClassCreator
|
import akka.actor.FromClassCreator
|
||||||
import scala.reflect.ClassTag
|
import scala.reflect.ClassTag
|
||||||
|
import util.{ Failure, Success }
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Serializes akka's internal DaemonMsgCreate using protobuf
|
* Serializes akka's internal DaemonMsgCreate using protobuf
|
||||||
|
|
@ -88,14 +89,10 @@ private[akka] class DaemonMsgCreateSerializer(val system: ExtendedActorSystem) e
|
||||||
|
|
||||||
def props = {
|
def props = {
|
||||||
val creator =
|
val creator =
|
||||||
if (proto.getProps.hasFromClassCreator) {
|
if (proto.getProps.hasFromClassCreator)
|
||||||
system.dynamicAccess.getClassFor[Actor](proto.getProps.getFromClassCreator) match {
|
FromClassCreator(system.dynamicAccess.getClassFor[Actor](proto.getProps.getFromClassCreator).get)
|
||||||
case Right(clazz) ⇒ FromClassCreator(clazz)
|
else
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
}
|
|
||||||
} else {
|
|
||||||
deserialize(proto.getProps.getCreator, classOf[() ⇒ Actor])
|
deserialize(proto.getProps.getCreator, classOf[() ⇒ Actor])
|
||||||
}
|
|
||||||
|
|
||||||
val routerConfig =
|
val routerConfig =
|
||||||
if (proto.getProps.hasRouterConfig) deserialize(proto.getProps.getRouterConfig, classOf[RouterConfig])
|
if (proto.getProps.hasRouterConfig) deserialize(proto.getProps.getRouterConfig, classOf[RouterConfig])
|
||||||
|
|
@ -115,25 +112,21 @@ private[akka] class DaemonMsgCreateSerializer(val system: ExtendedActorSystem) e
|
||||||
supervisor = deserializeActorRef(system, proto.getSupervisor))
|
supervisor = deserializeActorRef(system, proto.getSupervisor))
|
||||||
}
|
}
|
||||||
|
|
||||||
protected def serialize(any: AnyRef): ByteString =
|
protected def serialize(any: AnyRef): ByteString = ByteString.copyFrom(serialization.serialize(any).get)
|
||||||
serialization.serialize(any) match {
|
|
||||||
case Right(bytes) ⇒ ByteString.copyFrom(bytes)
|
|
||||||
case Left(e) ⇒ throw e
|
|
||||||
}
|
|
||||||
|
|
||||||
protected def deserialize[T: ClassTag](data: ByteString, clazz: Class[T]): T = {
|
protected def deserialize[T: ClassTag](data: ByteString, clazz: Class[T]): T = {
|
||||||
val bytes = data.toByteArray
|
val bytes = data.toByteArray
|
||||||
serialization.deserialize(bytes, clazz) match {
|
serialization.deserialize(bytes, clazz) match {
|
||||||
case Right(x: T) ⇒ x
|
case Success(x: T) ⇒ x
|
||||||
case Right(other) ⇒ throw new IllegalArgumentException("Can't deserialize to [%s], got [%s]".format(clazz.getName, other))
|
case Success(other) ⇒ throw new IllegalArgumentException("Can't deserialize to [%s], got [%s]".format(clazz.getName, other))
|
||||||
case Left(e) ⇒
|
case Failure(e) ⇒
|
||||||
// Fallback to the java serializer, because some interfaces don't implement java.io.Serializable,
|
// Fallback to the java serializer, because some interfaces don't implement java.io.Serializable,
|
||||||
// but the impl instance does. This could be optimized by adding java serializers in reference.conf:
|
// but the impl instance does. This could be optimized by adding java serializers in reference.conf:
|
||||||
// com.typesafe.config.Config
|
// com.typesafe.config.Config
|
||||||
// akka.routing.RouterConfig
|
// akka.routing.RouterConfig
|
||||||
// akka.actor.Scope
|
// akka.actor.Scope
|
||||||
serialization.deserialize(bytes, classOf[java.io.Serializable]) match {
|
serialization.deserialize(bytes, classOf[java.io.Serializable]) match {
|
||||||
case Right(x: T) ⇒ x
|
case Success(x: T) ⇒ x
|
||||||
case _ ⇒ throw e // the first exception
|
case _ ⇒ throw e // the first exception
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -79,15 +79,7 @@ class DaemonMsgCreateSerializerSpec extends AkkaSpec {
|
||||||
}
|
}
|
||||||
|
|
||||||
def verifySerialization(msg: DaemonMsgCreate): Unit = {
|
def verifySerialization(msg: DaemonMsgCreate): Unit = {
|
||||||
val bytes = ser.serialize(msg) match {
|
assertDaemonMsgCreate(msg, ser.deserialize(ser.serialize(msg).get, classOf[DaemonMsgCreate]).get.asInstanceOf[DaemonMsgCreate])
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(bytes) ⇒ bytes
|
|
||||||
}
|
|
||||||
ser.deserialize(bytes.asInstanceOf[Array[Byte]], classOf[DaemonMsgCreate]) match {
|
|
||||||
case Left(exception) ⇒ fail(exception)
|
|
||||||
case Right(m: DaemonMsgCreate) ⇒ assertDaemonMsgCreate(msg, m)
|
|
||||||
case other ⇒ throw new MatchError(other)
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
def assertDaemonMsgCreate(expected: DaemonMsgCreate, got: DaemonMsgCreate): Unit = {
|
def assertDaemonMsgCreate(expected: DaemonMsgCreate, got: DaemonMsgCreate): Unit = {
|
||||||
|
|
|
||||||
|
|
@ -136,14 +136,13 @@ object TestActorRef {
|
||||||
def apply[T <: Actor](implicit t: ClassTag[T], system: ActorSystem): TestActorRef[T] = apply[T](randomName)
|
def apply[T <: Actor](implicit t: ClassTag[T], system: ActorSystem): TestActorRef[T] = apply[T](randomName)
|
||||||
|
|
||||||
def apply[T <: Actor](name: String)(implicit t: ClassTag[T], system: ActorSystem): TestActorRef[T] = apply[T](Props({
|
def apply[T <: Actor](name: String)(implicit t: ClassTag[T], system: ActorSystem): TestActorRef[T] = apply[T](Props({
|
||||||
system.asInstanceOf[ExtendedActorSystem].dynamicAccess.createInstanceFor[T](t.runtimeClass, Seq()) match {
|
system.asInstanceOf[ExtendedActorSystem].dynamicAccess.createInstanceFor[T](t.runtimeClass, Seq()).recover({
|
||||||
case Right(value) ⇒ value
|
case exception ⇒ throw ActorInitializationException(null,
|
||||||
case Left(exception) ⇒ throw ActorInitializationException(null,
|
|
||||||
"Could not instantiate Actor" +
|
"Could not instantiate Actor" +
|
||||||
"\nMake sure Actor is NOT defined inside a class/trait," +
|
"\nMake sure Actor is NOT defined inside a class/trait," +
|
||||||
"\nif so put it outside the class/trait, f.e. in a companion object," +
|
"\nif so put it outside the class/trait, f.e. in a companion object," +
|
||||||
"\nOR try to change: 'actorOf(Props[MyActor]' to 'actorOf(Props(new MyActor)'.", exception)
|
"\nOR try to change: 'actorOf(Props[MyActor]' to 'actorOf(Props(new MyActor)'.", exception)
|
||||||
}
|
}).get
|
||||||
}), name)
|
}), name)
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
|
||||||
Loading…
Add table
Add a link
Reference in a new issue