Almost there... ActorRefSpec still has a failing test

This commit is contained in:
Viktor Klang 2011-09-20 18:34:21 +02:00
parent 48deb311fc
commit 9007b6e847
8 changed files with 189 additions and 101 deletions

View file

@ -11,11 +11,11 @@ import akka.testkit._
import akka.util.duration._ import akka.util.duration._
import akka.testkit.Testing.sleepFor import akka.testkit.Testing.sleepFor
import akka.config.Supervision.{ OneForOnePermanentStrategy } import akka.config.Supervision.{ OneForOnePermanentStrategy }
import akka.dispatch.Future
import java.util.concurrent.{ TimeUnit, CountDownLatch } import java.util.concurrent.{ TimeUnit, CountDownLatch }
import java.lang.IllegalStateException import java.lang.IllegalStateException
import akka.util.ReflectiveAccess import akka.util.ReflectiveAccess
import akka.actor.Actor.actorOf import akka.actor.Actor.actorOf
import akka.dispatch.{ DefaultPromise, Promise, Future }
object ActorRefSpec { object ActorRefSpec {
@ -115,6 +115,23 @@ object ActorRefSpec {
class ActorRefSpec extends WordSpec with MustMatchers { class ActorRefSpec extends WordSpec with MustMatchers {
import akka.actor.ActorRefSpec._ import akka.actor.ActorRefSpec._
def promiseIntercept(f: Actor)(to: Promise[Actor]): Actor = try {
val r = f
to.completeWithResult(r)
r
} catch {
case e
to.completeWithException(e)
throw e
}
def wrap[T](f: Promise[Actor] T): T = {
val result = new DefaultPromise[Actor](10 * 60 * 1000)
val r = f(result)
result.get
r
}
"An ActorRef" must { "An ActorRef" must {
"not allow Actors to be created outside of an actorOf" in { "not allow Actors to be created outside of an actorOf" in {
@ -123,10 +140,11 @@ class ActorRefSpec extends WordSpec with MustMatchers {
} }
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
wrap(result
actorOf(new Actor { actorOf(new Actor {
val nested = new Actor { def receive = { case _ } } val nested = promiseIntercept(new Actor { def receive = { case _ } })(result)
def receive = { case _ } def receive = { case _ }
}) }))
} }
def contextStackMustBeEmpty = ActorCell.contextStack.get.headOption must be === None def contextStackMustBeEmpty = ActorCell.contextStack.get.headOption must be === None
@ -134,69 +152,80 @@ class ActorRefSpec extends WordSpec with MustMatchers {
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingOuterActor(actorOf(new InnerActor))) wrap(result
actorOf(promiseIntercept(new FailingOuterActor(actorOf(new InnerActor)))(result)))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new OuterActor(actorOf(new FailingInnerActor))) wrap(result
actorOf(new OuterActor(actorOf(promiseIntercept(new FailingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingInheritingOuterActor(actorOf(new InnerActor))) wrap(result
actorOf(promiseIntercept(new FailingInheritingOuterActor(actorOf(new InnerActor)))(result)))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingOuterActor(actorOf(new FailingInheritingInnerActor))) wrap(result
actorOf(new FailingOuterActor(actorOf(promiseIntercept(new FailingInheritingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingInheritingOuterActor(actorOf(new FailingInheritingInnerActor))) wrap(result
actorOf(new FailingInheritingOuterActor(actorOf(promiseIntercept(new FailingInheritingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingInheritingOuterActor(actorOf(new FailingInnerActor))) wrap(result
actorOf(new FailingInheritingOuterActor(actorOf(promiseIntercept(new FailingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
wrap(result
actorOf(new OuterActor(actorOf(new InnerActor { actorOf(new OuterActor(actorOf(new InnerActor {
val a = new InnerActor val a = promiseIntercept(new InnerActor)(result)
}))) }))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new FailingOuterActor(actorOf(new FailingInheritingInnerActor))) wrap(result
actorOf(new FailingOuterActor(actorOf(promiseIntercept(new FailingInheritingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new OuterActor(actorOf(new FailingInheritingInnerActor))) wrap(result
actorOf(new OuterActor(actorOf(promiseIntercept(new FailingInheritingInnerActor)(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
intercept[akka.actor.ActorInitializationException] { intercept[akka.actor.ActorInitializationException] {
actorOf(new OuterActor(actorOf({ new InnerActor; new InnerActor }))) wrap(result
actorOf(new OuterActor(actorOf(promiseIntercept({ new InnerActor; new InnerActor })(result)))))
} }
contextStackMustBeEmpty contextStackMustBeEmpty
(intercept[java.lang.IllegalStateException] { (intercept[java.lang.IllegalStateException] {
actorOf(new OuterActor(actorOf({ throw new IllegalStateException("Ur state be b0rked"); new InnerActor }))) wrap(result
actorOf(new OuterActor(actorOf(promiseIntercept({ throw new IllegalStateException("Ur state be b0rked"); new InnerActor })(result)))))
}).getMessage must be === "Ur state be b0rked" }).getMessage must be === "Ur state be b0rked"
contextStackMustBeEmpty contextStackMustBeEmpty

View file

@ -56,9 +56,8 @@ case class HotSwap(code: ActorRef ⇒ Actor.Receive, discardOld: Boolean = true)
def this(code: akka.japi.Function[ActorRef, Procedure[Any]]) = this(code, true) def this(code: akka.japi.Function[ActorRef, Procedure[Any]]) = this(code, true)
} }
case object Init extends AutoReceivedMessage with LifeCycleMessage
case class Death(deceased: ActorRef, cause: Throwable, recoverable: Boolean) extends AutoReceivedMessage with LifeCycleMessage case class Death(deceased: ActorRef, cause: Throwable, recoverable: Boolean) extends AutoReceivedMessage with LifeCycleMessage
case class Restart(reason: Throwable) extends AutoReceivedMessage with LifeCycleMessage case class Crash(reason: Throwable) extends AutoReceivedMessage
case object RevertHotSwap extends AutoReceivedMessage case object RevertHotSwap extends AutoReceivedMessage
@ -657,14 +656,13 @@ trait Actor {
*/ */
msg match { msg match {
case Init reply(()); false //All gud nao FIXME remove reply when we can have fully async init
case HotSwap(code, discardOld) become(code(self), discardOld); false case HotSwap(code, discardOld) become(code(self), discardOld); false
case RevertHotSwap unbecome(); false case RevertHotSwap unbecome(); false
case d: Death context.handleDeath(d); false case d: Death context.handleDeath(d); false
case Link(child) self.link(child); false case Link(child) self.link(child); false
case Unlink(child) self.unlink(child); false case Unlink(child) self.unlink(child); false
case UnlinkAndStop(child) self.unlink(child); child.stop(); false case UnlinkAndStop(child) self.unlink(child); child.stop(); false
case Restart(reason) throw reason case Crash(reason) throw reason
case Kill throw new ActorKilledException("Kill") case Kill throw new ActorKilledException("Kill")
case PoisonPill case PoisonPill
val ch = channel val ch = channel

View file

@ -49,12 +49,6 @@ private[akka] trait ActorContext {
} }
private[akka] object ActorCell { private[akka] object ActorCell {
sealed trait Status
object Status {
object Running extends Status
object Shutdown extends Status
}
val contextStack = new ThreadLocal[Stack[ActorContext]] { val contextStack = new ThreadLocal[Stack[ActorContext]] {
override def initialValue = Stack[ActorContext]() override def initialValue = Stack[ActorContext]()
} }
@ -72,7 +66,7 @@ private[akka] class ActorCell(
val guard = new ReentrantGuard // TODO: remove this last synchronization point val guard = new ReentrantGuard // TODO: remove this last synchronization point
@volatile @volatile
var status: Status = Status.Running var terminated = false
@volatile @volatile
var mailbox: AnyRef = _ var mailbox: AnyRef = _
@ -111,14 +105,13 @@ private[akka] class ActorCell(
def dispatcher: MessageDispatcher = props.dispatcher def dispatcher: MessageDispatcher = props.dispatcher
def isRunning: Boolean = status == Status.Running def isRunning: Boolean = !terminated
def isShutdown: Boolean = status == Status.Shutdown def isShutdown: Boolean = terminated
def start(): Unit = { def start(): Unit = {
if (isShutdown) throw new ActorStartException("Can't start an actor that has been stopped")
if (props.supervisor.isDefined) props.supervisor.get.link(self) if (props.supervisor.isDefined) props.supervisor.get.link(self)
dispatcher.attach(this) dispatcher.attach(this)
Actor.registry.register(self) dispatcher.systemDispatch(SystemMessageInvocation(this, Create, NullChannel))
} }
def newActor(restart: Boolean): Actor = { def newActor(restart: Boolean): Actor = {
@ -153,33 +146,8 @@ private[akka] class ActorCell(
def resume(): Unit = dispatcher.resume(this) def resume(): Unit = dispatcher.resume(this)
private[akka] def stop(): Unit = guard.withGuard { private[akka] def stop(): Unit =
if (isRunning) { if (!terminated) dispatcher.systemDispatch(SystemMessageInvocation(this, Terminate, NullChannel))
receiveTimeout = None
cancelReceiveTimeout
Actor.registry.unregister(self)
status = Status.Shutdown
dispatcher.detach(this)
try {
val a = actor.get
if (Actor.debugLifecycle) EventHandler.debug(a, "stopping")
if (a ne null) a.postStop()
{ //Stop supervised actors
val i = _linkedActors.values.iterator
while (i.hasNext) {
i.next.stop()
i.remove()
}
}
} finally {
//if (supervisor.isDefined) supervisor.get ! Death(self, new ActorKilledException("Stopped"), false)
currentMessage = null
clearActorContext()
}
}
}
def link(actorRef: ActorRef): ActorRef = { def link(actorRef: ActorRef): ActorRef = {
guard.withGuard { guard.withGuard {
@ -252,7 +220,79 @@ private[akka] class ActorCell(
case msg msg.channel case msg msg.channel
} }
def systemInvoke(envelope: SystemMessageInvocation): Unit = {
var isTerminated = terminated
def create(recreation: Boolean): Unit = try {
actor.get() match {
case null
val created = newActor(restart = false)
actor.set(created)
created.preStart()
Actor.registry.register(self)
case instance if recreation
restart(new Exception("Restart commanded"), None, None)
case _
}
} catch {
case e
e.printStackTrace()
envelope.channel.sendException(e)
if (supervisor.isDefined) supervisor.get ! Death(self, e, false) else throw e
}
def suspend(): Unit = dispatcher suspend this
def resume(): Unit = dispatcher resume this
def terminate(): Unit = {
receiveTimeout = None
cancelReceiveTimeout
Actor.registry.unregister(self)
isTerminated = true
dispatcher.detach(this)
try {
val a = actor.get
if (Actor.debugLifecycle) EventHandler.debug(a, "stopping")
if (a ne null) a.postStop()
{ //Stop supervised actors
val i = _linkedActors.values.iterator
while (i.hasNext) {
i.next.stop()
i.remove()
}
}
} finally {
if (supervisor.isDefined) supervisor.get ! Death(self, new ActorKilledException("Stopped"), false)
currentMessage = null
clearActorContext()
}
}
guard.lock.lock()
try {
if (!isTerminated) {
envelope.message match {
case Create create(recreation = false)
case Recreate create(recreation = true)
case Suspend suspend()
case Resume resume()
case Terminate terminate()
}
}
} catch {
case e //Should we really catch everything here?
EventHandler.error(e, actor.get(), e.getMessage)
throw e
} finally {
terminated = isTerminated
guard.lock.unlock()
}
}
def invoke(messageHandle: MessageInvocation): Unit = { def invoke(messageHandle: MessageInvocation): Unit = {
var isTerminated = terminated
guard.lock.lock() guard.lock.lock()
try { try {
if (!isShutdown) { if (!isShutdown) {
@ -261,17 +301,7 @@ private[akka] class ActorCell(
try { try {
cancelReceiveTimeout() // FIXME: leave this here? cancelReceiveTimeout() // FIXME: leave this here?
val a = actor.get() match { actor.get().apply(messageHandle.message)
case null
val created = newActor(restart = false)
actor.set(created)
if (Actor.debugLifecycle) EventHandler.debug(created, "started")
created.preStart()
created
case instance instance
}
a.apply(messageHandle.message)
currentMessage = null // reset current message after successful invocation currentMessage = null // reset current message after successful invocation
} catch { } catch {
case e case e
@ -294,9 +324,11 @@ private[akka] class ActorCell(
throw e throw e
} }
} else { } else {
messageHandle.channel sendException new ActorKilledException("Actor has been stopped")
// throwing away message if actor is shut down, no use throwing an exception in receiving actor's thread, isShutdown is enforced on caller side // throwing away message if actor is shut down, no use throwing an exception in receiving actor's thread, isShutdown is enforced on caller side
} }
} finally { } finally {
terminated = isTerminated
guard.lock.unlock() guard.lock.unlock()
} }
} }

View file

@ -92,9 +92,19 @@ class Dispatcher(
protected[akka] def dispatch(invocation: MessageInvocation) = { protected[akka] def dispatch(invocation: MessageInvocation) = {
val mbox = getMailbox(invocation.receiver) val mbox = getMailbox(invocation.receiver)
if (mbox ne null) {
mbox enqueue invocation mbox enqueue invocation
registerForExecution(mbox) registerForExecution(mbox)
} }
}
protected[akka] def systemDispatch(invocation: SystemMessageInvocation) = {
val mbox = getMailbox(invocation.receiver)
if (mbox ne null) {
mbox systemEnqueue invocation
registerForExecution(mbox)
}
}
protected[akka] def executeTask(invocation: TaskInvocation): Unit = if (active.isOn) { protected[akka] def executeTask(invocation: TaskInvocation): Unit = if (active.isOn) {
try executorService.get() execute invocation try executorService.get() execute invocation
@ -142,7 +152,7 @@ class Dispatcher(
protected[akka] def registerForExecution(mbox: MessageQueue with ExecutableMailbox): Unit = { protected[akka] def registerForExecution(mbox: MessageQueue with ExecutableMailbox): Unit = {
if (mbox.dispatcherLock.tryLock()) { if (mbox.dispatcherLock.tryLock()) {
if (active.isOn && !mbox.suspended.locked) { //If the dispatcher is active and the actor not suspended if (active.isOn && (!mbox.suspended.locked || !mbox.systemMessages.isEmpty)) { //If the dispatcher is active and the actor not suspended
try { try {
executorService.get() execute mbox executorService.get() execute mbox
} catch { } catch {
@ -196,7 +206,7 @@ trait ExecutableMailbox extends Runnable { self: MessageQueue ⇒
case ie: InterruptedException Thread.currentThread().interrupt() //Restore interrupt case ie: InterruptedException Thread.currentThread().interrupt() //Restore interrupt
} finally { } finally {
dispatcherLock.unlock() dispatcherLock.unlock()
if (!self.isEmpty) if (!self.isEmpty || !self.systemMessages.isEmpty)
dispatcher.reRegisterForExecution(this) dispatcher.reRegisterForExecution(this)
} }
} }
@ -207,6 +217,7 @@ trait ExecutableMailbox extends Runnable { self: MessageQueue ⇒
* @return true if the processing finished before the mailbox was empty, due to the throughput constraint * @return true if the processing finished before the mailbox was empty, due to the throughput constraint
*/ */
final def processMailbox() { final def processMailbox() {
processAllSystemMessages()
if (!self.suspended.locked) { if (!self.suspended.locked) {
var nextMessage = self.dequeue var nextMessage = self.dequeue
if (nextMessage ne null) { //If we have a message if (nextMessage ne null) { //If we have a message
@ -219,6 +230,7 @@ trait ExecutableMailbox extends Runnable { self: MessageQueue ⇒
else 0 else 0
do { do {
nextMessage.invoke nextMessage.invoke
processAllSystemMessages()
nextMessage = nextMessage =
if (self.suspended.locked) { if (self.suspended.locked) {
null // If we are suspended, abort null // If we are suspended, abort

View file

@ -16,12 +16,24 @@ class MessageQueueAppendFailedException(message: String, cause: Throwable = null
* @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a> * @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a>
*/ */
trait MessageQueue { trait MessageQueue {
val dispatcherLock = new SimpleLock val dispatcherLock = new SimpleLock(startLocked = false)
val suspended = new SimpleLock val suspended = new SimpleLock(startLocked = false) //(startLocked = true)
val systemMessages = new ConcurrentLinkedQueue[SystemMessageInvocation]()
def enqueue(handle: MessageInvocation) def enqueue(handle: MessageInvocation)
def dequeue(): MessageInvocation def dequeue(): MessageInvocation
def systemEnqueue(handle: SystemMessageInvocation): Unit = systemMessages.offer(handle)
def systemDequeue(): SystemMessageInvocation = systemMessages.poll()
def size: Int def size: Int
def isEmpty: Boolean def isEmpty: Boolean
def processAllSystemMessages(): Unit = {
var nextMessage = systemDequeue()
while (nextMessage ne null) {
nextMessage.invoke()
nextMessage = systemDequeue()
}
}
} }
/** /**

View file

@ -16,14 +16,22 @@ import akka.actor._
/** /**
* @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a> * @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a>
*/ */
final case class MessageInvocation(val receiver: ActorCell, final case class MessageInvocation(val receiver: ActorCell, val message: Any, val channel: UntypedChannel) {
val message: Any,
val channel: UntypedChannel) {
if (receiver eq null) throw new IllegalArgumentException("Receiver can't be null") if (receiver eq null) throw new IllegalArgumentException("Receiver can't be null")
final def invoke() { final def invoke() { receiver invoke this }
receiver invoke this }
}
sealed trait SystemMessage
case object Create extends SystemMessage
case object Recreate extends SystemMessage
case object Suspend extends SystemMessage
case object Resume extends SystemMessage
case object Terminate extends SystemMessage
final case class SystemMessageInvocation(val receiver: ActorCell, val message: SystemMessage, val channel: UntypedChannel) {
if (receiver eq null) throw new IllegalArgumentException("Receiver can't be null")
final def invoke() { receiver systemInvoke this }
} }
final case class TaskInvocation(function: () Unit, cleanup: () Unit) extends Runnable { final case class TaskInvocation(function: () Unit, cleanup: () Unit) extends Runnable {
@ -73,15 +81,12 @@ abstract class MessageDispatcher extends Serializable {
* Attaches the specified actor instance to this dispatcher * Attaches the specified actor instance to this dispatcher
*/ */
final def attach(actor: ActorCell): Unit = { final def attach(actor: ActorCell): Unit = {
val promise = new ActorPromise(Timeout.never)(this)
guard.lock.lock() guard.lock.lock()
try { try {
register(actor) register(actor)
dispatchMessage(new MessageInvocation(actor, Init, promise))
} finally { } finally {
guard.lock.unlock() guard.lock.unlock()
} }
promise.get
} }
/** /**
@ -226,6 +231,11 @@ abstract class MessageDispatcher extends Serializable {
*/ */
protected[akka] def dispatch(invocation: MessageInvocation) protected[akka] def dispatch(invocation: MessageInvocation)
/**
* Will be called when the dispatcher is to queue an invocation for execution
*/
protected[akka] def systemDispatch(invocation: SystemMessageInvocation)
protected[akka] def executeTask(invocation: TaskInvocation) protected[akka] def executeTask(invocation: TaskInvocation)
/** /**

View file

@ -55,9 +55,7 @@ class ReadWriteGuard {
* A very simple lock that uses CCAS (Compare Compare-And-Swap) * A very simple lock that uses CCAS (Compare Compare-And-Swap)
* Does not keep track of the owner and isn't Reentrant, so don't nest and try to stick to the if*-methods * Does not keep track of the owner and isn't Reentrant, so don't nest and try to stick to the if*-methods
*/ */
class SimpleLock { class SimpleLock(startLocked: Boolean = false) extends AtomicBoolean(startLocked) {
val acquired = new AtomicBoolean(false)
def ifPossible(perform: () Unit): Boolean = { def ifPossible(perform: () Unit): Boolean = {
if (tryLock()) { if (tryLock()) {
try { try {
@ -89,20 +87,13 @@ class SimpleLock {
} else None } else None
} }
def tryLock() = { def tryLock() = compareAndSet(false, true)
if (acquired.get) false
else acquired.compareAndSet(false, true)
}
def tryUnlock() = { def tryUnlock() = compareAndSet(true, false)
acquired.compareAndSet(true, false)
}
def locked = acquired.get def locked = get
def unlock() { def unlock(): Unit = set(false)
acquired.set(false)
}
} }
/** /**

View file

@ -4,7 +4,6 @@
package akka.testkit package akka.testkit
import akka.event.EventHandler import akka.event.EventHandler
import akka.dispatch.{ MessageDispatcher, MessageInvocation, TaskInvocation, Promise, ActorPromise }
import java.util.concurrent.locks.ReentrantLock import java.util.concurrent.locks.ReentrantLock
import java.util.LinkedList import java.util.LinkedList
import java.util.concurrent.RejectedExecutionException import java.util.concurrent.RejectedExecutionException
@ -12,6 +11,7 @@ import akka.util.Switch
import java.lang.ref.WeakReference import java.lang.ref.WeakReference
import scala.annotation.tailrec import scala.annotation.tailrec
import akka.actor.ActorCell import akka.actor.ActorCell
import akka.dispatch._
/* /*
* Locking rules: * Locking rules:
@ -137,6 +137,10 @@ class CallingThreadDispatcher(val name: String = "calling-thread", val warnings:
override def mailboxIsEmpty(actor: ActorCell): Boolean = getMailbox(actor).queue.isEmpty override def mailboxIsEmpty(actor: ActorCell): Boolean = getMailbox(actor).queue.isEmpty
protected[akka] override def systemDispatch(handle: SystemMessageInvocation) {
handle.invoke() //Roland, look at me
}
protected[akka] override def dispatch(handle: MessageInvocation) { protected[akka] override def dispatch(handle: MessageInvocation) {
val mbox = getMailbox(handle.receiver) val mbox = getMailbox(handle.receiver)
val queue = mbox.queue val queue = mbox.queue