From 2e459f5f1d6b88472118b8c479f3eded3740926a Mon Sep 17 00:00:00 2001 From: Roland Date: Thu, 14 Jun 2012 15:52:30 +0200 Subject: [PATCH 01/14] make suspend/resume act recursively, as always intended, see #2212 --- .../akka/actor/SupervisorHierarchySpec.scala | 53 ++++++++++++++++++- .../src/main/scala/akka/actor/ActorCell.scala | 10 +++- 2 files changed, 60 insertions(+), 3 deletions(-) diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala index 62752d8052..f9e9ab42b1 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala @@ -29,10 +29,19 @@ object SupervisorHierarchySpec { countDown.countDown() } } + + class Resumer extends Actor { + override def supervisorStrategy = OneForOneStrategy() { case _ ⇒ SupervisorStrategy.Resume } + def receive = { + case "spawn" ⇒ sender ! context.actorOf(Props[Resumer]) + case "fail" ⇒ throw new Exception("expected") + case "ping" ⇒ sender ! "pong" + } + } } @org.junit.runner.RunWith(classOf[org.scalatest.junit.JUnitRunner]) -class SupervisorHierarchySpec extends AkkaSpec with DefaultTimeout { +class SupervisorHierarchySpec extends AkkaSpec with DefaultTimeout with ImplicitSender { import SupervisorHierarchySpec._ "A Supervisor Hierarchy" must { @@ -81,6 +90,48 @@ class SupervisorHierarchySpec extends AkkaSpec with DefaultTimeout { assert(countDownMax.await(2, TimeUnit.SECONDS)) } } + + "resume children after Resume" in { + val boss = system.actorOf(Props[Resumer], "resumer") + boss ! "spawn" + val middle = expectMsgType[ActorRef] + middle ! "spawn" + val worker = expectMsgType[ActorRef] + worker ! "ping" + expectMsg("pong") + EventFilter[Exception]("expected", occurrences = 1) intercept { + middle ! "fail" + } + middle ! "ping" + expectMsg("pong") + worker ! "ping" + expectMsg("pong") + } + + "suspend children while failing" in { + val latch = TestLatch() + val slowResumer = system.actorOf(Props(new Actor { + override def supervisorStrategy = OneForOneStrategy() { case _ ⇒ Await.ready(latch, 4.seconds.dilated); SupervisorStrategy.Resume } + def receive = { + case "spawn" ⇒ sender ! context.actorOf(Props[Resumer]) + } + }), "slowResumer") + slowResumer ! "spawn" + val boss = expectMsgType[ActorRef] + boss ! "spawn" + val middle = expectMsgType[ActorRef] + middle ! "spawn" + val worker = expectMsgType[ActorRef] + worker ! "ping" + expectMsg("pong") + EventFilter[Exception]("expected", occurrences = 1) intercept { + boss ! "fail" + } + worker ! "ping" + expectNoMsg(2 seconds) + latch.countDown() + expectMsg("pong") + } } } diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 9dbe610195..c5d2afb3fa 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -581,9 +581,15 @@ private[akka] class ActorCell( } } - def suspend(): Unit = if (isNormal) dispatcher suspend this + def suspend(): Unit = if (isNormal) { + dispatcher suspend this + children foreach (_.asInstanceOf[InternalActorRef].suspend()) + } - def resume(): Unit = if (isNormal) dispatcher resume this + def resume(): Unit = if (isNormal) { + dispatcher resume this + children foreach (_.asInstanceOf[InternalActorRef].resume()) + } def addWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { val watcheeSelf = watchee == self From 1148e20dbbf81ecac09b95433b25376d803910d9 Mon Sep 17 00:00:00 2001 From: Roland Date: Tue, 19 Jun 2012 11:02:06 +0200 Subject: [PATCH 02/14] add stress test, keep count of suspend/resume, and fix resulting bugs --- .../akka/actor/SupervisorHierarchySpec.scala | 371 +++++++++++++++++- .../scala/akka/actor/SupervisorSpec.scala | 6 +- .../src/main/scala/akka/actor/ActorCell.scala | 5 + .../main/scala/akka/dispatch/Mailbox.scala | 42 +- 4 files changed, 401 insertions(+), 23 deletions(-) diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala index f9e9ab42b1..33387443bb 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala @@ -4,10 +4,16 @@ package akka.actor -import akka.testkit._ +import java.lang.Exception import java.util.concurrent.{ TimeUnit, CountDownLatch } -import akka.dispatch.Await +import scala.util.Random +import scala.util.control.NoStackTrace +import com.typesafe.config.{ ConfigFactory, Config } +import SupervisorStrategy.{ Resume, Restart, Directive } +import akka.dispatch.{ MessageDispatcher, DispatcherPrerequisites, DispatcherConfigurator, Dispatcher, Await } import akka.pattern.ask +import akka.testkit._ +import akka.testkit.TestEvent.Mute import akka.util.Duration import akka.util.duration._ @@ -38,10 +44,350 @@ object SupervisorHierarchySpec { case "ping" ⇒ sender ! "pong" } } + + case class Event(msg: Any) { val time: Long = System.nanoTime } + case class ErrorLog(msg: String, log: Vector[Event]) + case class Failure(directive: Directive, log: Vector[Event]) extends RuntimeException with NoStackTrace { + override def toString = "Failure(" + directive + ")" + } + val strategy = OneForOneStrategy() { case Failure(directive, _) ⇒ directive } + + val config = ConfigFactory.parseString(""" + hierarchy { + type = "akka.actor.SupervisorHierarchySpec$MyDispatcherConfigurator" + } + akka.loglevel = INFO + akka.actor.debug.fsm = on + """) + + class MyDispatcherConfigurator(config: Config, prerequisites: DispatcherPrerequisites) + extends DispatcherConfigurator(config, prerequisites) { + + private val instance: MessageDispatcher = + new Dispatcher(prerequisites, + config.getString("id"), + config.getInt("throughput"), + Duration(config.getNanoseconds("throughput-deadline-time"), TimeUnit.NANOSECONDS), + mailboxType, + configureExecutor(), + Duration(config.getMilliseconds("shutdown-timeout"), TimeUnit.MILLISECONDS)) { + + override def suspend(cell: ActorCell): Unit = { + val a = cell.actor.asInstanceOf[Hierarchy] + a.log :+= Event("suspended") + super.suspend(cell) + } + + override def resume(cell: ActorCell): Unit = { + val a = cell.actor.asInstanceOf[Hierarchy] + a.log :+= Event("resumed") + super.resume(cell) + } + + } + + override def dispatcher(): MessageDispatcher = instance + } + + class Hierarchy(depth: Int, breadth: Int, listener: ActorRef) extends Actor { + + override def preStart { + if (depth > 1) + for (_ ← 1 to breadth) + context.watch(context.actorOf(Props(new Hierarchy(depth - 1, breadth, listener)).withDispatcher("hierarchy"))) + listener ! self + } + override def postRestart(cause: Throwable) { + cause match { + case Failure(_, l) ⇒ log = l + } + log :+= Event("restarted") + } + + override def supervisorStrategy = strategy + override def preRestart(cause: Throwable, msg: Option[Any]): Unit = { + // do not scrap children + } + + override def postStop { + if (failed || suspended) { + listener ! ErrorLog("not resumed (" + failed + ", " + suspended + ")", log) + } + } + + var failed = false + var suspended = false + var log = Vector.empty[Event] + def check(msg: Any) = { + suspended = false + log :+= Event(msg) + if (failed) { + listener ! ErrorLog("processing message while failed", log) + failed = false + context stop self + } + } + + def receive = new Receive { + val handler: Receive = { + case f @ Failure(Resume, _) ⇒ suspended = true; throw f.copy(log = log) + case f: Failure ⇒ failed = true; throw f.copy(log = log) + case "ping" ⇒ Thread.sleep((Random.nextFloat * 1.03).toLong); sender ! "pong" + case Terminated(_) ⇒ listener ! ErrorLog("terminating", log); context stop self + } + override def isDefinedAt(msg: Any) = handler.isDefinedAt(msg) + override def apply(msg: Any) = { check(msg); handler(msg) } + } + } + + case class Work(n: Int) + + sealed trait Action + case class Ping(ref: ActorRef) extends Action + case class Fail(ref: ActorRef, directive: Directive) extends Action + + sealed trait State + case object Idle extends State + case object Init extends State + case object Stress extends State + case object Finishing extends State + case object LastPing extends State + case object Stopping extends State + case object Failed extends State + + /* + * This stress test will construct a supervision hierarchy of configurable + * depth and breadth and then randomly fail and check its actors. The actors + * perform certain checks internally (verifying that they do not run when + * suspended, for example), and they are checked for health by the test + * procedure. + * + * Execution happens in phases (which is the reason for FSM): + * + * Idle: + * - upon reception of Init message, construct hierary and go to Init state + * + * Init: + * - receive refs of all contained actors + * + * Stress: + * - deal out actions (Fail or "ping"), keeping the hierarchy busy + * - whenever all actors are in the "pinged" list (i.e. have not yet + * answered with a "pong"), delay processing of the next Work() by + * 100 millis + * - when receiving a Work() while all actors are "pinged", stop the + * hierarchy and go to the Stopping state + * + * Finishing: + * - after dealing out the last action, wait for the outstanding "pong" + * messages + * - when last "pong" is received, goto LastPing state + * - upon state timeout, stop the hierarchy and go to the Failed state + * + * LastPing: + * - upon entering this state, send a "ping" to all actors + * - when last "pong" is received, goto Stopping state + * - upon state timeout, stop the hierarchy and go to the Failed state + * + * Stopping: + * - upon entering this state, stop the hierarchy + * - upon termination of the hierarchy send back successful result + * + * Whenever an ErrorLog is received, goto Failed state + * + * Failed: + * - accumulate ErrorLog messages + * - upon termination of the hierarchy send back failed result and print + * the logs, merged and in chronological order. + * + * TODO RK: also test Stop directive, and keep a complete list of all + * actors ever created, then verify after stop()ping the hierarchy that + * all are terminated, transfer them to a WeakHashMap and verify that + * they are indeed GCed + * + * TODO RK: make hierarchy construction stochastic so that it includes + * different breadth (including the degenerate breadth-1 case). + * + * TODO RK: also test Escalate by adding an exception with a `var depth` + * which gets decremented within the supervisor and gets handled when zero + * is reached (Restart resolution) + * + * TODO RK: also test exceptions during recreate + * + * TODO RK: also test recreate including terminating children + */ + + class StressTest(testActor: ActorRef, depth: Int, breadth: Int) extends Actor with LoggingFSM[State, Null] { + import context.system + + override def supervisorStrategy = strategy + + var children = Vector.empty[ActorRef] + var idleChildren = Vector.empty[ActorRef] + var pingChildren = Set.empty[ActorRef] + + val nextJob = Iterator.continually(Random.nextFloat match { + case x if x > 0.5 ⇒ + // ping one child + val pick = ((x - 0.5) * 2 * idleChildren.size).toInt + val ref = idleChildren(pick) + idleChildren = idleChildren.take(pick) ++ idleChildren.drop(pick + 1) + pingChildren += ref + Ping(ref) + case x ⇒ + // fail one child + val pick = ((if (x > 0.25) x - 0.25 else x) * 4 * children.size).toInt + Fail(children(pick), if (x > 0.25) Restart else Resume) + }) + + val familySize = ((1 - BigInt(breadth).pow(depth)) / (1 - breadth)).toInt + var hierarchy: ActorRef = _ + + override def preRestart(cause: Throwable, msg: Option[Any]) { + throw new ActorKilledException("I want to DIE") + } + + override def postRestart(cause: Throwable) { + throw new ActorKilledException("I said I wanted to DIE, dammit!") + } + + override def postStop { + testActor ! "stressTestStopped" + } + + startWith(Idle, null) + + when(Idle) { + case Event(Init, _) ⇒ + hierarchy = context.watch(context.actorOf(Props(new Hierarchy(depth, breadth, self)).withDispatcher("hierarchy"))) + setTimer("phase", StateTimeout, 5 seconds, false) + goto(Init) + } + + when(Init) { + case Event(ref: ActorRef, _) ⇒ + if (idleChildren.nonEmpty || pingChildren.nonEmpty) + throw new IllegalStateException("received unexpected child " + children.size) + children :+= ref + if (children.size == familySize) { + idleChildren = children + goto(Stress) + } else stay + case Event(StateTimeout, _) ⇒ + testActor ! "only got %d out of %d refs".format(children.size, familySize) + stop() + } + + onTransition { + case Init -> Stress ⇒ + self ! Work(familySize * 1000) + // set timeout for completion of the whole test (i.e. including Finishing and Stopping) + setTimer("phase", StateTimeout, 60 seconds, false) + } + + val workSchedule = 250.millis + + when(Stress) { + case Event(w: Work, _) if idleChildren.isEmpty ⇒ + context stop hierarchy + goto(Failed) + case Event(Work(x), _) if x > 0 ⇒ + nextJob.next match { + case Ping(ref) ⇒ ref ! "ping" + case Fail(ref, dir) ⇒ ref ! Failure(dir, Vector.empty) + } + if (idleChildren.nonEmpty) self ! Work(x - 1) + else context.system.scheduler.scheduleOnce(workSchedule, self, Work(x - 1)) + stay + case Event(Work(_), _) ⇒ if (pingChildren.isEmpty) goto(LastPing) else goto(Finishing) + case Event("pong", _) ⇒ + pingChildren -= sender + idleChildren :+= sender + stay + } + + when(Finishing) { + case Event("pong", _) ⇒ + pingChildren -= sender + idleChildren :+= sender + if (pingChildren.isEmpty) goto(LastPing) else stay + } + + onTransition { + case _ -> LastPing ⇒ + idleChildren foreach (_ ! "ping") + pingChildren ++= idleChildren + idleChildren = Vector.empty + } + + when(LastPing) { + case Event("pong", _) ⇒ + pingChildren -= sender + idleChildren :+= sender + if (pingChildren.isEmpty) goto(Stopping) else stay + } + + onTransition { + case _ -> Stopping ⇒ context stop hierarchy + } + + when(Stopping, stateTimeout = 5 seconds) { + case Event(Terminated(r), _) if r == hierarchy ⇒ + testActor ! "stressTestSuccessful" + stop + case Event(StateTimeout, _) ⇒ + testActor ! "timeout in Stopping" + stop + } + + var errors = Vector.empty[(ActorRef, ErrorLog)] + + when(Failed, stateTimeout = 5 seconds) { + case Event(e: ErrorLog, _) ⇒ + errors :+= sender -> e + stay + case Event(Terminated(r), _) if r == hierarchy ⇒ + printErrors() + testActor ! "stressTestFailed" + stop + case Event(StateTimeout, _) ⇒ + printErrors() + testActor ! "timeout in Failed" + stop + case Event("pong", _) ⇒ stay // don’t care? + } + + def printErrors(): Unit = { + val merged = errors flatMap { + case (ref, ErrorLog(msg, log)) ⇒ + println(ref + " " + msg) + log map (l ⇒ (l.time, ref, l.msg.toString)) + } + merged.sorted foreach println + } + + whenUnhandled { + case Event(e: ErrorLog, _) ⇒ + errors :+= sender -> e + // don’t stop the hierarchy, that is going to happen all by itself and in the right order + goto(Failed) + case Event(StateTimeout, _) ⇒ + println("pingChildren:\n" + pingChildren.mkString("\n")) + context stop hierarchy + goto(Failed) + case Event(msg, _) ⇒ + testActor ! ("received unexpected msg: " + msg) + stop + } + + initialize + + } + } @org.junit.runner.RunWith(classOf[org.scalatest.junit.JUnitRunner]) -class SupervisorHierarchySpec extends AkkaSpec with DefaultTimeout with ImplicitSender { +class SupervisorHierarchySpec extends AkkaSpec(SupervisorHierarchySpec.config) with DefaultTimeout with ImplicitSender { import SupervisorHierarchySpec._ "A Supervisor Hierarchy" must { @@ -132,6 +478,25 @@ class SupervisorHierarchySpec extends AkkaSpec with DefaultTimeout with Implicit latch.countDown() expectMsg("pong") } + + "survive being stressed" in { + system.eventStream.publish(Mute(EventFilter[Failure]())) + system.eventStream.publish(Mute(EventFilter.warning(start = "received dead letter"))) + + val fsm = system.actorOf(Props(new StressTest(testActor, 6, 3)), "stressTest") + + fsm ! FSM.SubscribeTransitionCallBack(system.actorOf(Props(new Actor { + def receive = { + case s: FSM.CurrentState[_] ⇒ log.info("{}", s) + case t: FSM.Transition[_] ⇒ log.info("{}", t) + } + }))) + + fsm ! Init + + expectMsg(70 seconds, "stressTestSuccessful") + expectMsg("stressTestStopped") + } } } diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala index 3db5b5b5dc..ba0314714e 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala @@ -349,7 +349,8 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende throw e } }) - val dyingActor = Await.result((supervisor ? dyingProps).mapTo[ActorRef], timeout.duration) + supervisor ! dyingProps + val dyingActor = expectMsgType[ActorRef] filterEvents(EventFilter[RuntimeException]("Expected", occurrences = 1), EventFilter[IllegalStateException]("error while creating actor", occurrences = 1)) { @@ -358,7 +359,8 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende } } - Await.result(dyingActor.?(Ping)(DilatedTimeout), DilatedTimeout) must be === PongMessage + dyingActor ! Ping + expectMsg(PongMessage) inits.get must be(3) diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index c5d2afb3fa..7931e5428e 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -682,6 +682,11 @@ private[akka] class ActorCell( // prevent any further messages to be processed until the actor has been restarted dispatcher.suspend(this) if (actor ne null) actor.supervisorStrategy.handleSupervisorFailing(self, children) + // now we may just have suspended the poor guy which made us fail by way of Escalate, so adjust the score + currentMessage match { + case Envelope(Failed(`t`), child) ⇒ child.asInstanceOf[InternalActorRef].resume() + case _ ⇒ + } } finally { t match { // Wrap InterruptedExceptions and rethrow case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t diff --git a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala index 25fc0250af..d66b16cc27 100644 --- a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala +++ b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala @@ -23,12 +23,16 @@ private[akka] object Mailbox { * the following assigned numbers CANNOT be changed without looking at the code which uses them! */ - // primary status: only first three + // primary status final val Open = 0 // _status is not initialized in AbstractMailbox, so default must be zero! Deliberately without type ascription to make it a compile-time constant - final val Suspended = 1 // Deliberately without type ascription to make it a compile-time constant - final val Closed = 2 // Deliberately without type ascription to make it a compile-time constant + final val Closed = 1 // Deliberately without type ascription to make it a compile-time constant // secondary status: Scheduled bit may be added to Open/Suspended - final val Scheduled = 4 // Deliberately without type ascription to make it a compile-time constant + final val Scheduled = 2 // Deliberately without type ascription to make it a compile-time constant + // shifted by 2: the suspend count! + final val shouldScheduleMask = 3 + final val shouldProcessMask = ~2 + final val suspendMask = ~3 + final val suspendUnit = 4 // mailbox debugging helper using println (see below) // since this is a compile-time constant, scalac will elide code behind if (Mailbox.debug) (RK checked with 2.9.1) @@ -78,10 +82,10 @@ private[akka] abstract class Mailbox(val actor: ActorCell, val messageQueue: Mes final def status: Mailbox.Status = Unsafe.instance.getIntVolatile(this, AbstractMailbox.mailboxStatusOffset) @inline - final def shouldProcessMessage: Boolean = (status & 3) == Open + final def shouldProcessMessage: Boolean = (status & shouldProcessMask) == 0 @inline - final def isSuspended: Boolean = (status & 3) == Suspended + final def isSuspended: Boolean = (status & suspendMask) != 0 @inline final def isClosed: Boolean = status == Closed @@ -100,21 +104,30 @@ private[akka] abstract class Mailbox(val actor: ActorCell, val messageQueue: Mes /** * set new primary status Open. Caller does not need to worry about whether * status was Scheduled or not. + * + * @returns true if the suspend count reached zero */ @tailrec final def becomeOpen(): Boolean = status match { case Closed ⇒ setStatus(Closed); false - case s ⇒ updateStatus(s, Open | s & Scheduled) || becomeOpen() + case s ⇒ + val next = if (s < suspendUnit) s else s - suspendUnit + if (updateStatus(s, next)) next < suspendUnit + else becomeOpen() } /** * set new primary status Suspended. Caller does not need to worry about whether * status was Scheduled or not. + * + * @returns true if the previous suspend count was zero */ @tailrec final def becomeSuspended(): Boolean = status match { case Closed ⇒ setStatus(Closed); false - case s ⇒ updateStatus(s, Suspended | s & Scheduled) || becomeSuspended() + case s ⇒ + if (updateStatus(s, s + suspendUnit)) s < suspendUnit + else becomeSuspended() } /** @@ -135,11 +148,10 @@ private[akka] abstract class Mailbox(val actor: ActorCell, val messageQueue: Mes val s = status /* * only try to add Scheduled bit if pure Open/Suspended, not Closed or with - * Scheduled bit already set (this is one of the reasons why the numbers - * cannot be changed in object Mailbox above) + * Scheduled bit already set */ - if (s <= Suspended) updateStatus(s, s | Scheduled) || setAsScheduled() - else false + if ((s & shouldScheduleMask) != Open) false + else updateStatus(s, s | Scheduled) || setAsScheduled() } /** @@ -148,12 +160,6 @@ private[akka] abstract class Mailbox(val actor: ActorCell, val messageQueue: Mes @tailrec final def setAsIdle(): Boolean = { val s = status - /* - * only try to remove Scheduled bit if currently Scheduled, not Closed or - * without Scheduled bit set (this is one of the reasons why the numbers - * cannot be changed in object Mailbox above) - */ - updateStatus(s, s & ~Scheduled) || setAsIdle() } From 78a39198f1c2b3d5fa0adea0eea41d4a5c12e402 Mon Sep 17 00:00:00 2001 From: Roland Date: Wed, 4 Jul 2012 09:20:17 +0200 Subject: [PATCH 03/14] another round of fixes due to suspend counting, see #2212 MIME-Version: 1.0 Content-Type: text/plain; charset=UTF-8 Content-Transfer-Encoding: 8bit - always suspend/resume for Suspend/Resume/Recreate, no matter which state the actor is in, to keep the counter balanced - preRestart failures are logged but otherwise ignored; there’s nothing else (apart from terminating the actor) which we could do at that point - preRestart/postRestart exceptions have their own distinguishable subtype of ActorKilledException now - fix some race conditions in tests to make them produce fewer false failures - remove cruft from SupervisorStrategy and add methods which can actually be used to implement your own (with proper warning signs) --- .../test/scala/akka/actor/ActorRefSpec.scala | 2 +- .../test/scala/akka/actor/FSMTimingSpec.scala | 2 +- .../akka/actor/SupervisorHierarchySpec.scala | 2 + .../scala/akka/actor/SupervisorSpec.scala | 25 +- .../akka/actor/dispatch/ActorModelSpec.scala | 2 +- .../dispatch/PriorityDispatcherSpec.scala | 2 +- .../src/main/scala/akka/actor/Actor.scala | 28 ++- .../src/main/scala/akka/actor/ActorCell.scala | 238 ++++++++++-------- .../src/main/scala/akka/actor/ActorRef.scala | 6 +- .../main/scala/akka/actor/FaultHandling.scala | 58 +++-- .../akka/dispatch/AbstractDispatcher.scala | 2 +- .../main/scala/akka/dispatch/Mailbox.scala | 4 +- .../src/main/scala/akka/agent/Agent.scala | 2 +- .../akka/remote/RemoteActorRefProvider.scala | 2 +- .../testkit/CallingThreadDispatcher.scala | 5 +- 15 files changed, 243 insertions(+), 137 deletions(-) diff --git a/akka-actor-tests/src/test/scala/akka/actor/ActorRefSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/ActorRefSpec.scala index bec066d97a..3d3fecdf01 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/ActorRefSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/ActorRefSpec.scala @@ -227,7 +227,7 @@ class ActorRefSpec extends AkkaSpec with DefaultTimeout { contextStackMustBeEmpty } - filterException[java.lang.IllegalStateException] { + EventFilter[ActorInitializationException](occurrences = 1) intercept { (intercept[java.lang.IllegalStateException] { wrap(result ⇒ actorOf(Props(new OuterActor(actorOf(Props(promiseIntercept({ throw new IllegalStateException("Ur state be b0rked"); new InnerActor })(result))))))) diff --git a/akka-actor-tests/src/test/scala/akka/actor/FSMTimingSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/FSMTimingSpec.scala index df47c801bb..79c4e33a9d 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/FSMTimingSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/FSMTimingSpec.scala @@ -145,7 +145,7 @@ object FSMTimingSpec { } def resume(actorRef: ActorRef): Unit = actorRef match { - case l: LocalActorRef ⇒ l.resume() + case l: LocalActorRef ⇒ l.resume(inResponseToFailure = false) case _ ⇒ } diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala index 33387443bb..42eccf2a81 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala @@ -215,6 +215,8 @@ object SupervisorHierarchySpec { * TODO RK: also test exceptions during recreate * * TODO RK: also test recreate including terminating children + * + * TODO RK: also verify that preRestart is not called more than once per instance */ class StressTest(testActor: ActorRef, depth: Int, breadth: Int) extends Actor with LoggingFSM[State, Null] { diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala index ba0314714e..521f8d9f00 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorSpec.scala @@ -339,7 +339,12 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende OneForOneStrategy(maxNrOfRetries = 3, withinTimeRange = 10 seconds)(classOf[Exception] :: Nil)))) val dyingProps = Props(new Actor { - if (inits.incrementAndGet % 2 == 0) throw new IllegalStateException("Don't wanna!") + val init = inits.getAndIncrement() + if (init % 3 == 1) throw new IllegalStateException("Don't wanna!") + + override def preRestart(cause: Throwable, msg: Option[Any]) { + if (init % 3 == 0) throw new IllegalStateException("Don't wanna!") + } def receive = { case Ping ⇒ sender ! PongMessage @@ -352,8 +357,10 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende supervisor ! dyingProps val dyingActor = expectMsgType[ActorRef] - filterEvents(EventFilter[RuntimeException]("Expected", occurrences = 1), - EventFilter[IllegalStateException]("error while creating actor", occurrences = 1)) { + filterEvents( + EventFilter[RuntimeException]("Expected", occurrences = 1), + EventFilter[PreRestartException]("Don't wanna!", occurrences = 1), + EventFilter[PostRestartException]("Don't wanna!", occurrences = 1)) { intercept[RuntimeException] { Await.result(dyingActor.?(DieReply)(DilatedTimeout), DilatedTimeout) } @@ -376,8 +383,8 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende val child = context.watch(context.actorOf(Props(new Actor { override def postRestart(reason: Throwable): Unit = testActor ! "child restarted" def receive = { - case "die" ⇒ throw new IllegalStateException("OHNOES") - case "test" ⇒ sender ! "child green" + case l: TestLatch ⇒ Await.ready(l, 5 seconds); throw new IllegalStateException("OHNOES") + case "test" ⇒ sender ! "child green" } }), "child")) @@ -385,14 +392,18 @@ class SupervisorSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitSende def receive = { case t @ Terminated(`child`) ⇒ testActor ! "child terminated" - case "die" ⇒ child ! "die" + case l: TestLatch ⇒ child ! l case "test" ⇒ sender ! "green" case "testchild" ⇒ child forward "test" } })) - parent ! "die" + val latch = TestLatch() + parent ! latch parent ! "testchild" + EventFilter[IllegalStateException]("OHNOES", occurrences = 2) intercept { + latch.countDown() + } expectMsg("parent restarted") expectMsg("child terminated") parent ! "test" diff --git a/akka-actor-tests/src/test/scala/akka/actor/dispatch/ActorModelSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/dispatch/ActorModelSpec.scala index 4d83c85b82..0a85b0158f 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/dispatch/ActorModelSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/dispatch/ActorModelSpec.scala @@ -335,7 +335,7 @@ abstract class ActorModelSpec(config: String) extends AkkaSpec(config) with Defa assertNoCountDown(done, 1000, "Should not process messages while suspended") assertRefDefaultZero(a)(registers = 1, msgsReceived = 1, suspensions = 1) - a.resume + a.resume(inResponseToFailure = false) assertCountDown(done, 3.seconds.dilated.toMillis, "Should resume processing of messages when resumed") assertRefDefaultZero(a)(registers = 1, msgsReceived = 1, msgsProcessed = 1, suspensions = 1, resumes = 1) diff --git a/akka-actor-tests/src/test/scala/akka/dispatch/PriorityDispatcherSpec.scala b/akka-actor-tests/src/test/scala/akka/dispatch/PriorityDispatcherSpec.scala index a9855fef7d..5d052330dc 100644 --- a/akka-actor-tests/src/test/scala/akka/dispatch/PriorityDispatcherSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/dispatch/PriorityDispatcherSpec.scala @@ -61,7 +61,7 @@ class PriorityDispatcherSpec extends AkkaSpec(PriorityDispatcherSpec.config) wit val msgs = (1 to 100).toList for (m ← msgs) actor ! m - actor.resume //Signal the actor to start treating it's message backlog + actor.resume(inResponseToFailure = false) //Signal the actor to start treating it's message backlog Await.result(actor.?('Result).mapTo[List[Int]], timeout.duration) must be === msgs.reverse } diff --git a/akka-actor/src/main/scala/akka/actor/Actor.scala b/akka-actor/src/main/scala/akka/actor/Actor.scala index 2721ccffa0..327743d5d4 100644 --- a/akka-actor/src/main/scala/akka/actor/Actor.scala +++ b/akka-actor/src/main/scala/akka/actor/Actor.scala @@ -123,12 +123,36 @@ class InvalidActorNameException(message: String) extends AkkaException(message) /** * An ActorInitializationException is thrown when the the initialization logic for an Actor fails. */ -class ActorInitializationException private[akka] (actor: ActorRef, message: String, cause: Throwable) - extends AkkaException(message, cause) /*with NoStackTrace*/ { +class ActorInitializationException private[akka] (val actor: ActorRef, message: String, cause: Throwable) + extends AkkaException(message, cause) { def this(msg: String) = this(null, msg, null) def this(actor: ActorRef, msg: String) = this(actor, msg, null) } +/** + * A PreRestartException is thrown when the preRestart() method failed. + * + * @param actor is the actor whose preRestart() hook failed + * @param cause is the exception thrown by that actor within preRestart() + * @param origCause is the exception which caused the restart in the first place + * @param msg is the message which was optionally passed into preRestart() + */ +class PreRestartException private[akka] (actor: ActorRef, cause: Throwable, val origCause: Throwable, val msg: Option[Any]) + extends ActorInitializationException(actor, "exception in preRestart(" + origCause + ", " + msg + ")", cause) { +} + +/** + * A PostRestartException is thrown when constructor or postRestart() method + * fails during a restart attempt. + * + * @param actor is the actor whose constructor or postRestart() hook failed + * @param cause is the exception thrown by that actor within preRestart() + * @param origCause is the exception which caused the restart in the first place + */ +class PostRestartException private[akka] (actor: ActorRef, cause: Throwable, val origCause: Throwable) + extends ActorInitializationException(actor, "exception post restart (" + origCause + ")", cause) { +} + /** * InvalidMessageException is thrown when an invalid message is sent to an Actor. * Technically it's only "null" which is an InvalidMessageException but who knows, diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 7931e5428e..928944ca44 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -12,7 +12,7 @@ import akka.event.Logging.{ Debug, Warning, Error } import akka.japi.Procedure import java.io.{ NotSerializableException, ObjectOutputStream } import akka.serialization.SerializationExtension -import akka.event.Logging.LogEventException +import akka.event.Logging.{ LogEventException, LogEvent } import collection.immutable.{ TreeSet, Stack, TreeMap } import akka.util.{ Unsafe, Duration, Helpers, NonFatal } @@ -362,6 +362,7 @@ private[akka] class ActorCell( } private def isNormal = childrenRefs match { case TerminatingChildrenContainer(_, _, Termination | _: Recreation) ⇒ false + case TerminatedChildrenContainer ⇒ false case _ ⇒ true } @@ -414,6 +415,17 @@ private[akka] class ActorCell( var watching: Set[ActorRef] = emptyActorRefSet var watchedBy: Set[ActorRef] = emptyActorRefSet + /* + * have we told our supervisor that we Failed() and have not yet heard back? + * (actually: we might have heard back but not yet acted upon it, in case of + * a restart with dying children) + * might well be replaced by ref to a Cancellable in the future (see #2299) + */ + private var _failed = false + def currentlyFailed: Boolean = _failed + def setFailed(): Unit = _failed = true + def setNotFailed(): Unit = _failed = false + //Not thread safe, so should only be used inside the actor that inhabits this ActorCell final protected def randomName(): String = { val n = nextNameSequence @@ -469,7 +481,7 @@ private[akka] class ActorCell( final def suspend(): Unit = dispatcher.systemDispatch(this, Suspend()) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def resume(): Unit = dispatcher.systemDispatch(this, Resume()) + final def resume(inResponseToFailure: Boolean): Unit = dispatcher.systemDispatch(this, Resume(inResponseToFailure)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ final def stop(): Unit = dispatcher.systemDispatch(this, Terminate()) @@ -492,6 +504,23 @@ private[akka] class ActorCell( a } + /* ================= + * T H E R U L E S + * ================= + * + * Actors can be suspended for two reasons: + * - they fail + * - their supervisor gets suspended + * + * In particular they are not suspended multiple times because of cascading + * own failures, i.e. while currentlyFailed() they do not fail again. In case + * of a restart, failures in constructor/preStart count as new failures. + */ + + private def suspendNonRecursive(): Unit = dispatcher suspend this + + private def resumeNonRecursive(): Unit = dispatcher resume this + final def children: Iterable[ActorRef] = childrenRefs.children /** @@ -542,7 +571,7 @@ private[akka] class ActorCell( actor = created created.preStart() checkReceiveTimeout - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(created), "started (" + created + ")")) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(created), "started (" + created + ")")) } catch { case NonFatal(i: InstantiationException) ⇒ throw new ActorInitializationException(self, @@ -554,41 +583,46 @@ private[akka] class ActorCell( } } - def recreate(cause: Throwable): Unit = if (isNormal) { - try { + def recreate(cause: Throwable): Unit = + if (isNormal) { val failedActor = actor - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(failedActor), "restarting")) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(failedActor), "restarting")) if (failedActor ne null) { - val c = currentMessage //One read only plz try { - if (failedActor.context ne null) failedActor.preRestart(cause, if (c ne null) Some(c.message) else None) + // if the actor fails in preRestart, we can do nothing but log it: it’s best-effort + if (failedActor.context ne null) failedActor.preRestart(cause, Option(currentMessage)) + } catch { + case NonFatal(e) ⇒ + val ex = new PreRestartException(self, e, cause, Option(currentMessage)) + publish(Error(ex, self.path.toString, clazz(failedActor), e.getMessage)) } finally { clearActorFields(failedActor) } } + assert(mailbox.isSuspended, "mailbox must be suspended during restart, status=" + mailbox.status) childrenRefs match { case ct: TerminatingChildrenContainer ⇒ childrenRefs = ct.copy(reason = Recreation(cause)) - dispatcher suspend this case _ ⇒ doRecreate(cause, failedActor) } - } catch { - case NonFatal(e) ⇒ throw new ActorInitializationException(self, "exception during creation", e match { - case i: InstantiationException ⇒ i.getCause - case other ⇒ other - }) + } else { + // need to keep that suspend counter balanced + doResume(inResponseToFailure = false) } - } - def suspend(): Unit = if (isNormal) { - dispatcher suspend this + def doSuspend(): Unit = { + // done always to keep that suspend counter balanced + suspendNonRecursive() children foreach (_.asInstanceOf[InternalActorRef].suspend()) } - def resume(): Unit = if (isNormal) { - dispatcher resume this - children foreach (_.asInstanceOf[InternalActorRef].resume()) + def doResume(inResponseToFailure: Boolean): Unit = { + // done always to keep that suspend counter balanced + // must happen “atomically” + try resumeNonRecursive() + finally if (inResponseToFailure) setNotFailed() + children foreach (_.asInstanceOf[InternalActorRef].resume(inResponseToFailure = false)) } def addWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { @@ -598,12 +632,12 @@ private[akka] class ActorCell( if (watcheeSelf && !watcherSelf) { if (!watchedBy.contains(watcher)) { watchedBy += watcher - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(actor), "now monitoring " + watcher)) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now monitoring " + watcher)) } } else if (!watcheeSelf && watcherSelf) { watch(watchee) } else { - system.eventStream.publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Watch(%s,%s) for %s".format(watchee, watcher, self))) + publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Watch(%s,%s) for %s".format(watchee, watcher, self))) } } @@ -614,12 +648,12 @@ private[akka] class ActorCell( if (watcheeSelf && !watcherSelf) { if (watchedBy.contains(watcher)) { watchedBy -= watcher - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(actor), "stopped monitoring " + watcher)) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopped monitoring " + watcher)) } } else if (!watcheeSelf && watcherSelf) { unwatch(watchee) } else { - system.eventStream.publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Unwatch(%s,%s) for %s".format(watchee, watcher, self))) + publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Unwatch(%s,%s) for %s".format(watchee, watcher, self))) } } @@ -634,15 +668,15 @@ private[akka] class ActorCell( case ct: TerminatingChildrenContainer ⇒ childrenRefs = ct.copy(reason = Termination) // do not process normal messages while waiting for all children to terminate - dispatcher suspend this - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(actor), "stopping")) + suspendNonRecursive() + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) case _ ⇒ doTerminate() } } def supervise(child: ActorRef): Unit = if (!isTerminating) { if (childrenRefs.getByRef(child).isEmpty) childrenRefs = childrenRefs.add(child) - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(actor), "now supervising " + child)) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now supervising " + child)) } try { @@ -651,11 +685,12 @@ private[akka] class ActorCell( case Recreate(cause) ⇒ recreate(cause) case Watch(watchee, watcher) ⇒ addWatcher(watchee, watcher) case Unwatch(watchee, watcher) ⇒ remWatcher(watchee, watcher) - case Suspend() ⇒ suspend() - case Resume() ⇒ resume() + case Suspend() ⇒ doSuspend() + case Resume(inRespToFailure) ⇒ doResume(inRespToFailure) case Terminate() ⇒ terminate() case Supervise(child) ⇒ supervise(child) case ChildTerminated(child) ⇒ handleChildTerminated(child) + case NoMessage ⇒ // to shut up the exhaustiveness warning } } catch { case e @ (_: InterruptedException | NonFatal(_)) ⇒ handleInvokeFailure(e, "error while processing " + message) @@ -677,20 +712,30 @@ private[akka] class ActorCell( checkReceiveTimeout // Reschedule receive timeout } - final def handleInvokeFailure(t: Throwable, message: String): Unit = try { - dispatcher.reportFailure(new LogEventException(Error(t, self.path.toString, clazz(actor), message), t)) - // prevent any further messages to be processed until the actor has been restarted - dispatcher.suspend(this) - if (actor ne null) actor.supervisorStrategy.handleSupervisorFailing(self, children) - // now we may just have suspended the poor guy which made us fail by way of Escalate, so adjust the score - currentMessage match { - case Envelope(Failed(`t`), child) ⇒ child.asInstanceOf[InternalActorRef].resume() - case _ ⇒ + final def handleInvokeFailure(t: Throwable, message: String): Unit = { + try { + dispatcher.reportFailure(new LogEventException(Error(t, self.path.toString, clazz(actor), message), t)) + } catch { + case NonFatal(_) ⇒ // no sense logging if logging does not work } - } finally { - t match { // Wrap InterruptedExceptions and rethrow - case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t - case _ ⇒ parent.tell(Failed(t), self) + // prevent any further messages to be processed until the actor has been restarted + if (!currentlyFailed) { + // suspend self; these two must happen “atomically” + try suspendNonRecursive() + finally setFailed() + // suspend children + val skip: Set[ActorRef] = currentMessage match { + case Envelope(Failed(`t`), child) ⇒ Set(child) + case _ ⇒ Set.empty + } + childrenRefs.stats collect { + case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child + } foreach (_.asInstanceOf[InternalActorRef].suspend()) + // tell supervisor + t match { // Wrap InterruptedExceptions and rethrow + case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t + case _ ⇒ parent.tell(Failed(t), self) + } } } @@ -718,7 +763,7 @@ private[akka] class ActorCell( def autoReceiveMessage(msg: Envelope): Unit = { if (system.settings.DebugAutoReceive) - system.eventStream.publish(Debug(self.path.toString, clazz(actor), "received AutoReceiveMessage " + msg)) + publish(Debug(self.path.toString, clazz(actor), "received AutoReceiveMessage " + msg)) msg.message match { case Failed(cause) ⇒ handleFailure(sender, cause) @@ -738,73 +783,69 @@ private[akka] class ActorCell( private def doTerminate() { val a = actor - try { - try { - if (a ne null) a.postStop() - } finally { - dispatcher.detach(this) - } - } finally { - try { - parent.sendSystemMessage(ChildTerminated(self)) - - if (!watchedBy.isEmpty) { - val terminated = Terminated(self)(existenceConfirmed = true) - try { - watchedBy foreach { - watcher ⇒ - try watcher.tell(terminated, self) catch { - case NonFatal(t) ⇒ system.eventStream.publish(Error(t, self.path.toString, clazz(a), "deathwatch")) - } - } - } finally watchedBy = emptyActorRefSet - } - - if (!watching.isEmpty) { - try { - watching foreach { // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - case watchee: InternalActorRef ⇒ try watchee.sendSystemMessage(Unwatch(watchee, self)) catch { - case NonFatal(t) ⇒ system.eventStream.publish(Error(t, self.path.toString, clazz(a), "deathwatch")) + try if (a ne null) a.postStop() + finally try dispatcher.detach(this) + finally try parent.sendSystemMessage(ChildTerminated(self)) + finally try + if (!watchedBy.isEmpty) { + val terminated = Terminated(self)(existenceConfirmed = true) + try { + watchedBy foreach { + watcher ⇒ + try watcher.tell(terminated, self) catch { + case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(a), "deathwatch")) } - } - } finally watching = emptyActorRefSet - } - if (system.settings.DebugLifecycle) - system.eventStream.publish(Debug(self.path.toString, clazz(a), "stopped")) - } finally { - behaviorStack = behaviorStackPlaceHolder - clearActorFields(a) - actor = null + } + } finally watchedBy = emptyActorRefSet } + finally try + if (!watching.isEmpty) { + try { + watching foreach { // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + case watchee: InternalActorRef ⇒ try watchee.sendSystemMessage(Unwatch(watchee, self)) catch { + case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(a), "deathwatch")) + } + } + } finally watching = emptyActorRefSet + } + finally { + if (system.settings.DebugLifecycle) + publish(Debug(self.path.toString, clazz(a), "stopped")) + behaviorStack = behaviorStackPlaceHolder + clearActorFields(a) + actor = null } } private def doRecreate(cause: Throwable, failedActor: Actor): Unit = try { - // after all killed children have terminated, recreate the rest, then go on to start the new instance - actor.supervisorStrategy.handleSupervisorRestarted(cause, self, children) + // must happen “atomically” + try resumeNonRecursive() + finally setNotFailed() + + val survivors = children + val freshActor = newActor() actor = freshActor // this must happen before postRestart has a chance to fail if (freshActor eq failedActor) setActorFields(freshActor, this, self) // If the creator returns the same instance, we need to restore our nulled out fields. freshActor.postRestart(cause) - if (system.settings.DebugLifecycle) system.eventStream.publish(Debug(self.path.toString, clazz(freshActor), "restarted")) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(freshActor), "restarted")) - dispatcher.resume(this) + // only after parent is up and running again do restart the children which were not stopped + survivors foreach (child ⇒ + try child.asInstanceOf[InternalActorRef].restart(cause) + catch { + case NonFatal(e) ⇒ publish(Error(e, self.path.toString, clazz(freshActor), "restarting " + child)) + }) } catch { - case NonFatal(e) ⇒ try { - dispatcher.reportFailure(new LogEventException(Error(e, self.path.toString, clazz(actor), "error while creating actor"), e)) - // prevent any further messages to be processed until the actor has been restarted - dispatcher.suspend(this) - actor.supervisorStrategy.handleSupervisorFailing(self, children) // FIXME Should this be called on actor or failedActor? - clearActorFields(actor) // If this fails, we need to ensure that preRestart isn't called. - } finally { - parent.tell(Failed(new ActorInitializationException(self, "exception during re-creation", e)), self) - } + case NonFatal(e) ⇒ + clearActorFields(actor) // in order to prevent preRestart() from happening again + handleInvokeFailure(new PostRestartException(self, e, cause), e.getMessage) } final def handleFailure(child: ActorRef, cause: Throwable): Unit = childrenRefs.getByRef(child) match { case Some(stats) ⇒ if (!actor.supervisorStrategy.handleFailure(this, child, cause, stats, childrenRefs.stats)) throw cause - case None ⇒ system.eventStream.publish(Warning(self.path.toString, clazz(actor), "dropping Failed(" + cause + ") from unknown child " + child)) + case None ⇒ publish(Warning(self.path.toString, clazz(actor), "dropping Failed(" + cause + ") from unknown child " + child)) } final def handleChildTerminated(child: ActorRef): Unit = try { @@ -823,13 +864,7 @@ private[akka] class ActorCell( actor.supervisorStrategy.handleChildTerminated(this, child, children) } } catch { - case NonFatal(e) ⇒ - try { - dispatcher suspend this - actor.supervisorStrategy.handleSupervisorFailing(self, children) - } finally { - parent.tell(Failed(e), self) - } + case NonFatal(e) ⇒ handleInvokeFailure(e, "handleChildTerminated failed") } // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ @@ -881,6 +916,9 @@ private[akka] class ActorCell( } } + // logging is not the main purpose, and if it fails there’s nothing we can do + private final def publish(e: LogEvent): Unit = try system.eventStream.publish(e) catch { case NonFatal(_) ⇒ } + private final def clazz(o: AnyRef): Class[_] = if (o eq null) this.getClass else o.getClass } diff --git a/akka-actor/src/main/scala/akka/actor/ActorRef.scala b/akka-actor/src/main/scala/akka/actor/ActorRef.scala index 0620a73a28..5a6e101ea8 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorRef.scala @@ -177,7 +177,7 @@ private[akka] abstract class InternalActorRef extends ActorRef with ScalaActorRe /* * Actor life-cycle management, invoked only internally (in response to user requests via ActorContext). */ - def resume(): Unit + def resume(inResponseToFailure: Boolean): Unit def suspend(): Unit def restart(cause: Throwable): Unit def stop(): Unit @@ -267,7 +267,7 @@ private[akka] class LocalActorRef private[akka] ( /** * Resumes a suspended actor. */ - override def resume(): Unit = actorCell.resume() + override def resume(inResponseToFailure: Boolean): Unit = actorCell.resume(inResponseToFailure) /** * Shuts down the actor and its message queue @@ -367,7 +367,7 @@ private[akka] trait MinimalActorRef extends InternalActorRef with LocalRef { override def getChild(names: Iterator[String]): InternalActorRef = if (names.forall(_.isEmpty)) this else Nobody override def suspend(): Unit = () - override def resume(): Unit = () + override def resume(inResponseToFailure: Boolean): Unit = () override def stop(): Unit = () override def isTerminated = false diff --git a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala index 27a9f346db..b3737d5b57 100644 --- a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala @@ -196,19 +196,30 @@ object SupervisorStrategy extends SupervisorStrategyLowPriorityImplicits { } /** - * An Akka SupervisorStrategy is the policy to apply for crashing children + * An Akka SupervisorStrategy is the policy to apply for crashing children. + * + * IMPORTANT: + * + * You should not normally need to create new subclasses, instead use the + * existing [[akka.actor.OneForOneStrategy]] or [[akka.actor.AllForOneStrategy]], + * but if you do, please read the docs of the methods below carefully, as + * incorrect implementations may lead to “blocked” actor systems (i.e. + * permanently suspended actors). */ abstract class SupervisorStrategy { import SupervisorStrategy._ /** - * Returns the Decider that is associated with this SupervisorStrategy + * Returns the Decider that is associated with this SupervisorStrategy. + * The Decider is invoked by the default implementation of `handleFailure` + * to obtain the Directive to be applied. */ def decider: Decider /** * This method is called after the child has been removed from the set of children. + * It does not need to do anything special. */ def handleChildTerminated(context: ActorContext, child: ActorRef, children: Iterable[ActorRef]): Unit @@ -217,27 +228,46 @@ abstract class SupervisorStrategy { */ def processFailure(context: ActorContext, restart: Boolean, child: ActorRef, cause: Throwable, stats: ChildRestartStats, children: Iterable[ChildRestartStats]): Unit - //FIXME docs - def handleSupervisorFailing(supervisor: ActorRef, children: Iterable[ActorRef]): Unit = - if (children.nonEmpty) children.foreach(_.asInstanceOf[InternalActorRef].suspend()) - - //FIXME docs - def handleSupervisorRestarted(cause: Throwable, supervisor: ActorRef, children: Iterable[ActorRef]): Unit = - if (children.nonEmpty) children.foreach(_.asInstanceOf[InternalActorRef].restart(cause)) - /** - * Returns whether it processed the failure or not + * This is the main entry point: in case of a child’s failure, this method + * must try to handle the failure by resuming, restarting or stopping the + * child (and returning `true`), or it returns `false` to escalate the + * failure, which will lead to this actor re-throwing the exception which + * caused the failure. The exception will not be wrapped. */ def handleFailure(context: ActorContext, child: ActorRef, cause: Throwable, stats: ChildRestartStats, children: Iterable[ChildRestartStats]): Boolean = { val directive = if (decider.isDefinedAt(cause)) decider(cause) else Escalate //FIXME applyOrElse in Scala 2.10 directive match { - case Resume ⇒ child.asInstanceOf[InternalActorRef].resume(); true + case Resume ⇒ resumeChild(child); true case Restart ⇒ processFailure(context, true, child, cause, stats, children); true case Stop ⇒ processFailure(context, false, child, cause, stats, children); true case Escalate ⇒ false } } + /** + * Resume the previously failed child: do never apply this to a child which + * is not the currently failing child. Suspend/resume needs to be done in + * matching pairs, otherwise actors will wake up too soon or never at all. + */ + final def resumeChild(child: ActorRef): Unit = child.asInstanceOf[InternalActorRef].resume(inResponseToFailure = true) + + /** + * Restart the given child, possibly suspending it first. + * + * IMPORTANT: + * + * If the child is the currently failing one, it will already have been + * suspended, hence `suspendFirst` is false. If the child is not the + * currently failing one, then it did not request this treatment and is + * therefore not prepared to be resumed without prior suspend. + */ + final def restartChild(child: ActorRef, cause: Throwable, suspendFirst: Boolean): Unit = { + val c = child.asInstanceOf[InternalActorRef] + if (suspendFirst) c.suspend() + c.restart(cause) + } + } /** @@ -276,7 +306,7 @@ case class AllForOneStrategy(maxNrOfRetries: Int = -1, withinTimeRange: Duration def processFailure(context: ActorContext, restart: Boolean, child: ActorRef, cause: Throwable, stats: ChildRestartStats, children: Iterable[ChildRestartStats]): Unit = { if (children.nonEmpty) { if (restart && children.forall(_.requestRestartPermission(retriesWindow))) - children.foreach(_.child.asInstanceOf[InternalActorRef].restart(cause)) + children foreach (crs ⇒ restartChild(crs.child, cause, suspendFirst = (crs.child != child))) else for (c ← children) context.stop(c.child) } @@ -318,7 +348,7 @@ case class OneForOneStrategy(maxNrOfRetries: Int = -1, withinTimeRange: Duration def processFailure(context: ActorContext, restart: Boolean, child: ActorRef, cause: Throwable, stats: ChildRestartStats, children: Iterable[ChildRestartStats]): Unit = { if (restart && stats.requestRestartPermission(retriesWindow)) - child.asInstanceOf[InternalActorRef].restart(cause) + restartChild(child, cause, suspendFirst = false) else context.stop(child) //TODO optimization to drop child here already? } diff --git a/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala b/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala index 12eea14ffc..fa3ccc8674 100644 --- a/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala +++ b/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala @@ -86,7 +86,7 @@ private[akka] case class Suspend() extends SystemMessage // sent to self from Ac /** * INTERNAL API */ -private[akka] case class Resume() extends SystemMessage // sent to self from ActorCell.resume +private[akka] case class Resume(inResponseToFailure: Boolean) extends SystemMessage // sent to self from ActorCell.resume /** * INTERNAL API */ diff --git a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala index d66b16cc27..58f9998485 100644 --- a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala +++ b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala @@ -30,7 +30,7 @@ private[akka] object Mailbox { final val Scheduled = 2 // Deliberately without type ascription to make it a compile-time constant // shifted by 2: the suspend count! final val shouldScheduleMask = 3 - final val shouldProcessMask = ~2 + final val shouldNotProcessMask = ~2 final val suspendMask = ~3 final val suspendUnit = 4 @@ -82,7 +82,7 @@ private[akka] abstract class Mailbox(val actor: ActorCell, val messageQueue: Mes final def status: Mailbox.Status = Unsafe.instance.getIntVolatile(this, AbstractMailbox.mailboxStatusOffset) @inline - final def shouldProcessMessage: Boolean = (status & shouldProcessMask) == 0 + final def shouldProcessMessage: Boolean = (status & shouldNotProcessMask) == 0 @inline final def isSuspended: Boolean = (status & suspendMask) != 0 diff --git a/akka-agent/src/main/scala/akka/agent/Agent.scala b/akka-agent/src/main/scala/akka/agent/Agent.scala index 64834178a8..2b4c4b0f00 100644 --- a/akka-agent/src/main/scala/akka/agent/Agent.scala +++ b/akka-agent/src/main/scala/akka/agent/Agent.scala @@ -214,7 +214,7 @@ class Agent[T](initialValue: T, system: ActorSystem) { /** * Resumes processing of `send` actions for the agent. */ - def resume(): Unit = updater.resume() + def resume(): Unit = updater.resume(inResponseToFailure = false) /** * Closes the agents and makes it eligible for garbage collection. diff --git a/akka-remote/src/main/scala/akka/remote/RemoteActorRefProvider.scala b/akka-remote/src/main/scala/akka/remote/RemoteActorRefProvider.scala index eaecf67792..fa989fb235 100644 --- a/akka-remote/src/main/scala/akka/remote/RemoteActorRefProvider.scala +++ b/akka-remote/src/main/scala/akka/remote/RemoteActorRefProvider.scala @@ -236,7 +236,7 @@ private[akka] class RemoteActorRef private[akka] ( def suspend(): Unit = sendSystemMessage(Suspend()) - def resume(): Unit = sendSystemMessage(Resume()) + def resume(inResponseToFailure: Boolean): Unit = sendSystemMessage(Resume(inResponseToFailure)) def stop(): Unit = sendSystemMessage(Terminate()) diff --git a/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala b/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala index 2fe664d7b6..1eba0ac23a 100644 --- a/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala +++ b/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala @@ -151,7 +151,7 @@ class CallingThreadDispatcher( override def suspend(actor: ActorCell) { actor.mailbox match { - case m: CallingThreadMailbox ⇒ m.suspendSwitch.switchOn + case m: CallingThreadMailbox ⇒ m.suspendSwitch.switchOn; m.becomeSuspended() case m ⇒ m.systemEnqueue(actor.self, Suspend()) } } @@ -163,11 +163,12 @@ class CallingThreadDispatcher( val wasActive = queue.isActive val switched = mbox.suspendSwitch.switchOff { CallingThreadDispatcherQueues(actor.system).gatherFromAllOtherQueues(mbox, queue) + mbox.becomeOpen() } if (switched && !wasActive) { runQueue(mbox, queue) } - case m ⇒ m.systemEnqueue(actor.self, Resume()) + case m ⇒ m.systemEnqueue(actor.self, Resume(false)) } } From c0dd08fa94b5e532f20e1c611e4b0095b74f79d9 Mon Sep 17 00:00:00 2001 From: Roland Date: Thu, 5 Jul 2012 13:43:08 +0200 Subject: [PATCH 04/14] move child-related things from ActorCell to ChildrenContainer --- .../src/main/scala/akka/actor/ActorCell.scala | 324 ++++++------------ .../scala/akka/actor/ChildrenContainer.scala | 107 +++++- 2 files changed, 212 insertions(+), 219 deletions(-) diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 90ebb89ab3..266aeff4b2 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -16,6 +16,7 @@ import akka.event.Logging.{ LogEventException, LogEvent } import collection.immutable.{ TreeSet, TreeMap } import akka.util.{ Unsafe, Duration, Helpers, NonFatal } import java.util.concurrent.atomic.AtomicLong +import scala.collection.JavaConverters.asJavaIterableConverter //TODO: everything here for current compatibility - could be limited more @@ -260,7 +261,6 @@ private[akka] object ActorCell { final val emptyBehaviorStack: List[Actor.Receive] = Nil final val emptyActorRefSet: Set[ActorRef] = TreeSet.empty - } //ACTORCELL IS 64bytes and should stay that way unless very good reason not to (machine sympathy, cache line fit) @@ -286,6 +286,12 @@ private[akka] class ActorCell( final def provider = system.provider + /* + * RECEIVE TIMEOUT + */ + + var receiveTimeoutData: (Duration, Cancellable) = emptyReceiveTimeoutData + override final def receiveTimeout: Option[Duration] = receiveTimeoutData._1 match { case Duration.Undefined ⇒ None case duration ⇒ Some(duration) @@ -300,142 +306,22 @@ private[akka] class ActorCell( final override def resetReceiveTimeout(): Unit = setReceiveTimeout(None) - /** - * In milliseconds + /* + * CHILDREN */ - var receiveTimeoutData: (Duration, Cancellable) = emptyReceiveTimeoutData @volatile private var _childrenRefsDoNotCallMeDirectly: ChildrenContainer = EmptyChildrenContainer def childrenRefs: ChildrenContainer = Unsafe.instance.getObjectVolatile(this, childrenOffset).asInstanceOf[ChildrenContainer] - private def swapChildrenRefs(oldChildren: ChildrenContainer, newChildren: ChildrenContainer): Boolean = - Unsafe.instance.compareAndSwapObject(this, childrenOffset, oldChildren, newChildren) + final def children: Iterable[ActorRef] = childrenRefs.children + final def getChildren(): java.lang.Iterable[ActorRef] = asJavaIterableConverter(children).asJava - @tailrec private def reserveChild(name: String): Boolean = { - val c = childrenRefs - swapChildrenRefs(c, c.reserve(name)) || reserveChild(name) - } - - @tailrec private def unreserveChild(name: String): Boolean = { - val c = childrenRefs - swapChildrenRefs(c, c.unreserve(name)) || unreserveChild(name) - } - - @tailrec private def addChild(ref: ActorRef): Boolean = { - val c = childrenRefs - swapChildrenRefs(c, c.add(ref)) || addChild(ref) - } - - @tailrec private def shallDie(ref: ActorRef): Boolean = { - val c = childrenRefs - swapChildrenRefs(c, c.shallDie(ref)) || shallDie(ref) - } - - @tailrec private def removeChild(ref: ActorRef): ChildrenContainer = { - val c = childrenRefs - val n = c.remove(ref) - if (swapChildrenRefs(c, n)) n - else removeChild(ref) - } - - @tailrec private def setChildrenTerminationReason(reason: SuspendReason): Boolean = { - childrenRefs match { - case c: TerminatingChildrenContainer ⇒ swapChildrenRefs(c, c.copy(reason = reason)) || setChildrenTerminationReason(reason) - case _ ⇒ false - } - } - - private def isTerminating = childrenRefs match { - case TerminatingChildrenContainer(_, _, Termination) ⇒ true - case TerminatedChildrenContainer ⇒ true - case _ ⇒ false - } - private def isNormal = childrenRefs match { - case TerminatingChildrenContainer(_, _, Termination | _: Recreation) ⇒ false - case TerminatedChildrenContainer ⇒ false - case _ ⇒ true - } - - private def _actorOf(props: Props, name: String, async: Boolean): ActorRef = { - if (system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) { - val ser = SerializationExtension(system) - ser.serialize(props.creator) match { - 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 - * (internal calls cannot happen anyway because we are suspended) - */ - if (isTerminating) throw new IllegalStateException("cannot create children while terminating or terminated") - else { - reserveChild(name) - // this name will either be unreserved or overwritten with a real child below - val actor = - try { - provider.actorOf(systemImpl, props, self, self.path / name, - systemService = false, deploy = None, lookupDeploy = true, async = async) - } catch { - case NonFatal(e) ⇒ - unreserveChild(name) - throw e - } - addChild(actor) - actor - } - } - - def actorOf(props: Props): ActorRef = _actorOf(props, randomName(), async = false) - - def actorOf(props: Props, name: String): ActorRef = _actorOf(props, checkName(name), async = false) - - private def checkName(name: String): String = { - import ActorPath.ElementRegex - name match { - case null ⇒ throw new InvalidActorNameException("actor name must not be null") - case "" ⇒ throw new InvalidActorNameException("actor name must not be empty") - case ElementRegex() ⇒ name - case _ ⇒ throw new InvalidActorNameException("illegal actor name '" + name + "', must conform to " + ElementRegex) - } - } - - private[akka] def attachChild(props: Props, name: String): ActorRef = - _actorOf(props, checkName(name), async = true) - - private[akka] def attachChild(props: Props): ActorRef = - _actorOf(props, randomName(), async = true) - - final def stop(actor: ActorRef): Unit = { - val started = actor match { - case r: RepointableRef ⇒ r.isStarted - case _ ⇒ true - } - if (childrenRefs.getByRef(actor).isDefined && started) shallDie(actor) - actor.asInstanceOf[InternalActorRef].stop() - } - - var currentMessage: Envelope = _ - var actor: Actor = _ - private var behaviorStack: List[Actor.Receive] = emptyBehaviorStack - var watching: Set[ActorRef] = emptyActorRefSet - var watchedBy: Set[ActorRef] = emptyActorRefSet - - /* - * have we told our supervisor that we Failed() and have not yet heard back? - * (actually: we might have heard back but not yet acted upon it, in case of - * a restart with dying children) - * might well be replaced by ref to a Cancellable in the future (see #2299) - */ - private var _failed = false - def currentlyFailed: Boolean = _failed - def setFailed(): Unit = _failed = true - def setNotFailed(): Unit = _failed = false + def actorOf(props: Props): ActorRef = makeChild(this, props, randomName(), async = false) + def actorOf(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = false) + private[akka] def attachChild(props: Props): ActorRef = makeChild(this, props, randomName(), async = true) + private[akka] def attachChild(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = true) @volatile private var _nextNameDoNotCallMeDirectly = 0L final protected def randomName(): String = { @@ -447,20 +333,51 @@ private[akka] class ActorCell( Helpers.base64(inc()) } + final def stop(actor: ActorRef): Unit = { + val started = actor match { + case r: RepointableRef ⇒ r.isStarted + case _ ⇒ true + } + if (childrenRefs.getByRef(actor).isDefined && started) shallDie(this, actor) + actor.asInstanceOf[InternalActorRef].stop() + } + + /* + * ACTOR STATE + */ + + var currentMessage: Envelope = _ + var actor: Actor = _ + private var behaviorStack: List[Actor.Receive] = emptyBehaviorStack + var watching: Set[ActorRef] = emptyActorRefSet + var watchedBy: Set[ActorRef] = emptyActorRefSet + + override final def watch(subject: ActorRef): ActorRef = subject match { + case a: InternalActorRef ⇒ + if (a != self && !watching.contains(a)) { + a.sendSystemMessage(Watch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + watching += a + } + a + } + + override final def unwatch(subject: ActorRef): ActorRef = subject match { + case a: InternalActorRef ⇒ + if (a != self && watching.contains(a)) { + a.sendSystemMessage(Unwatch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + watching -= a + } + a + } + + /* + * MAILBOX and DISPATCHER + */ + @volatile private var _mailboxDoNotCallMeDirectly: Mailbox = _ //This must be volatile since it isn't protected by the mailbox status - /** - * INTERNAL API - * - * Returns a reference to the current mailbox - */ @inline final def mailbox: Mailbox = Unsafe.instance.getObjectVolatile(this, mailboxOffset).asInstanceOf[Mailbox] - /** - * INTERNAL API - * - * replaces the current mailbox using getAndSet semantics - */ @tailrec final def swapMailbox(newMailbox: Mailbox): Mailbox = { val oldMailbox = mailbox if (!Unsafe.instance.compareAndSwapObject(this, mailboxOffset, oldMailbox, newMailbox)) swapMailbox(newMailbox) @@ -504,27 +421,46 @@ private[akka] class ActorCell( // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ final def resume(inResponseToFailure: Boolean): Unit = dispatcher.systemDispatch(this, Resume(inResponseToFailure)) + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + final def restart(cause: Throwable): Unit = dispatcher.systemDispatch(this, Recreate(cause)) + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ final def stop(): Unit = dispatcher.systemDispatch(this, Terminate()) - override final def watch(subject: ActorRef): ActorRef = subject match { - case a: InternalActorRef ⇒ - if (a != self && !watching.contains(a)) { - a.sendSystemMessage(Watch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - watching += a - } - a + def tell(message: Any, sender: ActorRef): Unit = + dispatcher.dispatch(this, Envelope(message, if (sender eq null) system.deadLetters else sender, system)) + + override def sendSystemMessage(message: SystemMessage): Unit = dispatcher.systemDispatch(this, message) + + /* + * ACTOR CONTEXT IMPL + */ + + final def sender: ActorRef = currentMessage match { + case null ⇒ system.deadLetters + case msg if msg.sender ne null ⇒ msg.sender + case _ ⇒ system.deadLetters } - override final def unwatch(subject: ActorRef): ActorRef = subject match { - case a: InternalActorRef ⇒ - if (a != self && watching.contains(a)) { - a.sendSystemMessage(Unwatch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - watching -= a - } - a + def become(behavior: Actor.Receive, discardOld: Boolean = true): Unit = + behaviorStack = behavior :: (if (discardOld && behaviorStack.nonEmpty) behaviorStack.tail else behaviorStack) + + def become(behavior: Procedure[Any]): Unit = become(behavior, false) + + def become(behavior: Procedure[Any], discardOld: Boolean): Unit = + become({ case msg ⇒ behavior.apply(msg) }: Actor.Receive, discardOld) + + def unbecome(): Unit = { + val original = behaviorStack + behaviorStack = + if (original.isEmpty || original.tail.isEmpty) actor.receive :: emptyBehaviorStack + else original.tail } + /* + * FAILURE HANDLING + */ + /* ================= * T H E R U L E S * ================= @@ -542,24 +478,16 @@ private[akka] class ActorCell( private def resumeNonRecursive(): Unit = dispatcher resume this - final def children: Iterable[ActorRef] = childrenRefs.children - - /** - * Impl UntypedActorContext + /* + * have we told our supervisor that we Failed() and have not yet heard back? + * (actually: we might have heard back but not yet acted upon it, in case of + * a restart with dying children) + * might well be replaced by ref to a Cancellable in the future (see #2299) */ - final def getChildren(): java.lang.Iterable[ActorRef] = - scala.collection.JavaConverters.asJavaIterableConverter(children).asJava - - def tell(message: Any, sender: ActorRef): Unit = - dispatcher.dispatch(this, Envelope(message, if (sender eq null) system.deadLetters else sender, system)) - - override def sendSystemMessage(message: SystemMessage): Unit = dispatcher.systemDispatch(this, message) - - final def sender: ActorRef = currentMessage match { - case null ⇒ system.deadLetters - case msg if msg.sender ne null ⇒ msg.sender - case _ ⇒ system.deadLetters - } + private var _failed = false + def currentlyFailed: Boolean = _failed + def setFailed(): Unit = _failed = true + def setNotFailed(): Unit = _failed = false //This method is in charge of setting up the contextStack and create a new instance of the Actor protected def newActor(): Actor = { @@ -584,7 +512,7 @@ private[akka] class ActorCell( //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status final def systemInvoke(message: SystemMessage) { - def create(): Unit = if (isNormal) { + def create(): Unit = if (childrenRefs.isNormal) { try { val created = newActor() actor = created @@ -603,7 +531,7 @@ private[akka] class ActorCell( } def recreate(cause: Throwable): Unit = - if (isNormal) { + if (childrenRefs.isNormal) { val failedActor = actor if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(failedActor), "restarting")) if (failedActor ne null) { @@ -621,7 +549,7 @@ private[akka] class ActorCell( assert(mailbox.isSuspended, "mailbox must be suspended during restart, status=" + mailbox.status) childrenRefs match { case ct: TerminatingChildrenContainer ⇒ - setChildrenTerminationReason(Recreation(cause)) + setChildrenTerminationReason(this, Recreation(cause)) case _ ⇒ doRecreate(cause, failedActor) } @@ -633,7 +561,7 @@ private[akka] class ActorCell( def doSuspend(): Unit = { // done always to keep that suspend counter balanced suspendNonRecursive() - children foreach (_.asInstanceOf[InternalActorRef].suspend()) + childrenRefs.suspendChildren() } def doResume(inResponseToFailure: Boolean): Unit = { @@ -641,7 +569,7 @@ private[akka] class ActorCell( // must happen “atomically” try resumeNonRecursive() finally if (inResponseToFailure) setNotFailed() - children foreach (_.asInstanceOf[InternalActorRef].resume(inResponseToFailure = false)) + childrenRefs.resumeChildren() } def addWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { @@ -685,7 +613,7 @@ private[akka] class ActorCell( childrenRefs match { case ct: TerminatingChildrenContainer ⇒ - setChildrenTerminationReason(Termination) + setChildrenTerminationReason(this, Termination) // do not process normal messages while waiting for all children to terminate suspendNonRecursive() if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) @@ -693,8 +621,8 @@ private[akka] class ActorCell( } } - def supervise(child: ActorRef): Unit = if (!isTerminating) { - if (childrenRefs.getByRef(child).isEmpty) addChild(child) + def supervise(child: ActorRef): Unit = if (!childrenRefs.isTerminating) { + if (childrenRefs.getByRef(child).isEmpty) addChild(this, child) handleSupervise(child) if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now supervising " + child)) } @@ -733,11 +661,7 @@ private[akka] class ActorCell( } final def handleInvokeFailure(t: Throwable, message: String): Unit = { - try { - dispatcher.reportFailure(new LogEventException(Error(t, self.path.toString, clazz(actor), message), t)) - } catch { - case NonFatal(_) ⇒ // no sense logging if logging does not work - } + publish(Error(t, self.path.toString, clazz(actor), message)) // prevent any further messages to be processed until the actor has been restarted if (!currentlyFailed) { // suspend self; these two must happen “atomically” @@ -748,9 +672,7 @@ private[akka] class ActorCell( case Envelope(Failed(`t`), child) ⇒ Set(child) case _ ⇒ Set.empty } - childrenRefs.stats collect { - case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child - } foreach (_.asInstanceOf[InternalActorRef].suspend()) + childrenRefs.suspendChildren(skip) // tell supervisor t match { // Wrap InterruptedExceptions and rethrow case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t @@ -759,27 +681,6 @@ private[akka] class ActorCell( } } - def become(behavior: Actor.Receive, discardOld: Boolean = true): Unit = - behaviorStack = behavior :: (if (discardOld && behaviorStack.nonEmpty) behaviorStack.tail else behaviorStack) - - /** - * UntypedActorContext impl - */ - def become(behavior: Procedure[Any]): Unit = become(behavior, false) - - /* - * UntypedActorContext impl - */ - def become(behavior: Procedure[Any], discardOld: Boolean): Unit = - become({ case msg ⇒ behavior.apply(msg) }: Actor.Receive, discardOld) - - def unbecome(): Unit = { - val original = behaviorStack - behaviorStack = - if (original.isEmpty || original.tail.isEmpty) actor.receive :: emptyBehaviorStack - else original.tail - } - def autoReceiveMessage(msg: Envelope): Unit = { if (system.settings.DebugAutoReceive) publish(Debug(self.path.toString, clazz(actor), "received AutoReceiveMessage " + msg)) @@ -870,8 +771,8 @@ private[akka] class ActorCell( final def handleChildTerminated(child: ActorRef): Unit = try { childrenRefs match { - case tc @ TerminatingChildrenContainer(_, _, reason) ⇒ - val n = removeChild(child) + case TerminatingChildrenContainer(_, _, reason) ⇒ + val n = removeChild(this, child) actor.supervisorStrategy.handleChildTerminated(this, child, children) if (!n.isInstanceOf[TerminatingChildrenContainer]) reason match { case Recreation(cause) ⇒ doRecreate(cause, actor) // doRecreate since this is the continuation of "recreate" @@ -879,7 +780,7 @@ private[akka] class ActorCell( case _ ⇒ } case _ ⇒ - removeChild(child) + removeChild(this, child) actor.supervisorStrategy.handleChildTerminated(this, child, children) } } catch { @@ -891,9 +792,6 @@ private[akka] class ActorCell( case _ ⇒ } - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def restart(cause: Throwable): Unit = dispatcher.systemDispatch(this, Recreate(cause)) - final def checkReceiveTimeout() { val recvtimeout = receiveTimeoutData if (Duration.Undefined != recvtimeout._1 && !mailbox.hasMessages) { diff --git a/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala b/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala index bd6608bf2b..692dd3ba07 100644 --- a/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala +++ b/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala @@ -5,28 +5,41 @@ package akka.actor import scala.collection.immutable.TreeMap +import scala.annotation.tailrec +import akka.util.Unsafe +import akka.serialization.SerializationExtension +import akka.util.NonFatal /** * INTERNAL API */ private[akka] trait ChildrenContainer { + def add(child: ActorRef): ChildrenContainer def remove(child: ActorRef): ChildrenContainer + def getByName(name: String): Option[ChildRestartStats] def getByRef(actor: ActorRef): Option[ChildRestartStats] + def children: Iterable[ActorRef] def stats: Iterable[ChildRestartStats] + def shallDie(actor: ActorRef): ChildrenContainer - /** - * reserve that name or throw an exception - */ + + // reserve that name or throw an exception def reserve(name: String): ChildrenContainer - /** - * cancel a reservation - */ + // cancel a reservation def unreserve(name: String): ChildrenContainer + def isTerminating: Boolean = false def isNormal: Boolean = true + + def suspendChildren(skip: Set[ActorRef] = Set.empty): Unit = + stats collect { + case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child + } foreach (_.asInstanceOf[InternalActorRef].suspend()) + + def resumeChildren(): Unit = stats foreach (_.child.asInstanceOf[InternalActorRef].suspend()) } /** @@ -37,6 +50,88 @@ private[akka] trait ChildrenContainer { */ private[akka] object ChildrenContainer { + // low level CAS helpers + @inline private def swapChildrenRefs(cell: ActorCell, oldChildren: ChildrenContainer, newChildren: ChildrenContainer): Boolean = + Unsafe.instance.compareAndSwapObject(cell, AbstractActorCell.childrenOffset, oldChildren, newChildren) + + @tailrec final def reserveChild(cell: ActorCell, name: String): Boolean = { + val c = cell.childrenRefs + swapChildrenRefs(cell, c, c.reserve(name)) || reserveChild(cell, name) + } + + @tailrec final def unreserveChild(cell: ActorCell, name: String): Boolean = { + val c = cell.childrenRefs + swapChildrenRefs(cell, c, c.unreserve(name)) || unreserveChild(cell, name) + } + + @tailrec final def addChild(cell: ActorCell, ref: ActorRef): Boolean = { + val c = cell.childrenRefs + swapChildrenRefs(cell, c, c.add(ref)) || addChild(cell, ref) + } + + @tailrec final def shallDie(cell: ActorCell, ref: ActorRef): Boolean = { + val c = cell.childrenRefs + swapChildrenRefs(cell, c, c.shallDie(ref)) || shallDie(cell, ref) + } + + @tailrec final def removeChild(cell: ActorCell, ref: ActorRef): ChildrenContainer = { + val c = cell.childrenRefs + val n = c.remove(ref) + if (swapChildrenRefs(cell, c, n)) n + else removeChild(cell, ref) + } + + @tailrec final def setChildrenTerminationReason(cell: ActorCell, reason: ChildrenContainer.SuspendReason): Boolean = { + cell.childrenRefs match { + case c: ChildrenContainer.TerminatingChildrenContainer ⇒ + swapChildrenRefs(cell, c, c.copy(reason = reason)) || setChildrenTerminationReason(cell, reason) + case _ ⇒ false + } + } + + def checkName(name: String): String = { + import ActorPath.ElementRegex + name match { + case null ⇒ throw new InvalidActorNameException("actor name must not be null") + case "" ⇒ throw new InvalidActorNameException("actor name must not be empty") + case ElementRegex() ⇒ name + case _ ⇒ throw new InvalidActorNameException("illegal actor name '" + name + "', must conform to " + ElementRegex) + } + } + + def makeChild(cell: ActorCell, props: Props, name: String, async: Boolean): ActorRef = { + if (cell.system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) { + val ser = SerializationExtension(cell.system) + ser.serialize(props.creator) match { + 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 + * (internal calls cannot happen anyway because we are suspended) + */ + if (cell.childrenRefs.isTerminating) throw new IllegalStateException("cannot create children while terminating or terminated") + else { + reserveChild(cell, name) + // this name will either be unreserved or overwritten with a real child below + val actor = + try { + cell.provider.actorOf(cell.systemImpl, props, cell.self, cell.self.path / name, + systemService = false, deploy = None, lookupDeploy = true, async = async) + } catch { + case NonFatal(e) ⇒ + unreserveChild(cell, name) + throw e + } + addChild(cell, actor) + actor + } + } + sealed trait SuspendReason case object UserRequest extends SuspendReason case class Recreation(cause: Throwable) extends SuspendReason From 36ac4d89de1ce91ba1f4e317026ef1c74394b658 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 8 Jul 2012 17:41:22 +0200 Subject: [PATCH 05/14] split up ActorCell functionality into multiple source files MIME-Version: 1.0 Content-Type: text/plain; charset=UTF-8 Content-Transfer-Encoding: 8bit - created package akka.actor.cell to hold the different traits from which the ActorCell cake is made - split up by topic, but leave the message processing itself within ActorCell - move ChildrenContainer into the akka.actor.cell package - move AbstractActorCell also - make members of the behavior traits private/protected to tighten their scope as much as possible => make it easier to see what’s going on --- .../akka/actor/SupervisorHierarchySpec.scala | 2 +- .../actor/{ => cell}/AbstractActorCell.java | 9 +- .../src/main/scala/akka/actor/ActorCell.scala | 563 +++--------------- .../src/main/scala/akka/actor/ActorRef.scala | 2 +- .../main/scala/akka/actor/ActorSystem.scala | 1 + .../main/scala/akka/actor/FaultHandling.scala | 3 +- .../akka/actor/RepointableActorRef.scala | 4 +- .../main/scala/akka/actor/cell/Children.scala | 181 ++++++ .../actor/{ => cell}/ChildrenContainer.scala | 102 +--- .../scala/akka/actor/cell/DeathWatch.scala | 93 +++ .../main/scala/akka/actor/cell/Dispatch.scala | 81 +++ .../scala/akka/actor/cell/FaultHandling.scala | 184 ++++++ .../akka/actor/cell/ReceiveTimeout.scala | 54 ++ 13 files changed, 714 insertions(+), 565 deletions(-) rename akka-actor/src/main/java/akka/actor/{ => cell}/AbstractActorCell.java (63%) create mode 100644 akka-actor/src/main/scala/akka/actor/cell/Children.scala rename akka-actor/src/main/scala/akka/actor/{ => cell}/ChildrenContainer.scala (67%) create mode 100644 akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala create mode 100644 akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala create mode 100644 akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala create mode 100644 akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala index 42eccf2a81..31b046486d 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorHierarchySpec.scala @@ -229,7 +229,7 @@ object SupervisorHierarchySpec { var pingChildren = Set.empty[ActorRef] val nextJob = Iterator.continually(Random.nextFloat match { - case x if x > 0.5 ⇒ + case x if x >= 0.5 ⇒ // ping one child val pick = ((x - 0.5) * 2 * idleChildren.size).toInt val ref = idleChildren(pick) diff --git a/akka-actor/src/main/java/akka/actor/AbstractActorCell.java b/akka-actor/src/main/java/akka/actor/cell/AbstractActorCell.java similarity index 63% rename from akka-actor/src/main/java/akka/actor/AbstractActorCell.java rename to akka-actor/src/main/java/akka/actor/cell/AbstractActorCell.java index 95fb7368bc..2d8c4fbc1e 100644 --- a/akka-actor/src/main/java/akka/actor/AbstractActorCell.java +++ b/akka-actor/src/main/java/akka/actor/cell/AbstractActorCell.java @@ -2,8 +2,9 @@ * Copyright (C) 2009-2012 Typesafe Inc. */ -package akka.actor; +package akka.actor.cell; +import akka.actor.ActorCell; import akka.util.Unsafe; final class AbstractActorCell { @@ -13,9 +14,9 @@ final class AbstractActorCell { static { try { - mailboxOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("_mailboxDoNotCallMeDirectly")); - childrenOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("_childrenRefsDoNotCallMeDirectly")); - nextNameOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("_nextNameDoNotCallMeDirectly")); + mailboxOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("akka$actor$cell$Dispatch$$_mailboxDoNotCallMeDirectly")); + childrenOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("akka$actor$cell$Children$$_childrenRefsDoNotCallMeDirectly")); + nextNameOffset = Unsafe.instance.objectFieldOffset(ActorCell.class.getDeclaredField("akka$actor$cell$Children$$_nextNameDoNotCallMeDirectly")); } catch(Throwable t){ throw new ExceptionInInitializerError(t); } diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 266aeff4b2..5aac54dce1 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -17,6 +17,7 @@ import collection.immutable.{ TreeSet, TreeMap } import akka.util.{ Unsafe, Duration, Helpers, NonFatal } import java.util.concurrent.atomic.AtomicLong import scala.collection.JavaConverters.asJavaIterableConverter +import akka.actor.cell.ChildrenContainer //TODO: everything here for current compatibility - could be limited more @@ -214,6 +215,10 @@ private[akka] trait Cell { * All children of this actor, including only reserved-names. */ def childrenRefs: ChildrenContainer + /** + * Get the stats for the named child, if that exists. + */ + def getChildByName(name: String): Option[ChildRestartStats] /** * Enqueue a message to be sent to the actor; may or may not actually * schedule the actor to run, depending on which type of cell it is. @@ -256,8 +261,6 @@ private[akka] object ActorCell { def cancel() {} } - final val emptyReceiveTimeoutData: (Duration, Cancellable) = (Duration.Undefined, emptyCancellable) - final val emptyBehaviorStack: List[Actor.Receive] = Nil final val emptyActorRefSet: Set[ActorRef] = TreeSet.empty @@ -266,174 +269,96 @@ private[akka] object ActorCell { //ACTORCELL IS 64bytes and should stay that way unless very good reason not to (machine sympathy, cache line fit) //vars don't need volatile since it's protected with the mailbox status //Make sure that they are not read/written outside of a message processing (systemInvoke/invoke) +/** + * Everything in here is completely Akka PRIVATE. You will not find any + * supported APIs in this place. This is not the API you were looking + * for! (waves hand) + */ private[akka] class ActorCell( val system: ActorSystemImpl, val self: InternalActorRef, val props: Props, - @volatile var parent: InternalActorRef) extends UntypedActorContext with Cell { + @volatile var parent: InternalActorRef) + extends UntypedActorContext with Cell + with cell.ReceiveTimeout + with cell.Children + with cell.Dispatch + with cell.DeathWatch + with cell.FaultHandling { - import AbstractActorCell.{ childrenOffset, mailboxOffset, nextNameOffset } import ActorCell._ - import ChildrenContainer._ final def isLocal = true final def systemImpl = system - protected final def guardian = self - protected final def lookupRoot = self - final def provider = system.provider - /* - * RECEIVE TIMEOUT - */ - - var receiveTimeoutData: (Duration, Cancellable) = emptyReceiveTimeoutData - - override final def receiveTimeout: Option[Duration] = receiveTimeoutData._1 match { - case Duration.Undefined ⇒ None - case duration ⇒ Some(duration) - } - - final def setReceiveTimeout(timeout: Option[Duration]): Unit = setReceiveTimeout(timeout.getOrElse(Duration.Undefined)) - - override final def setReceiveTimeout(timeout: Duration): Unit = - receiveTimeoutData = ( - if (Duration.Undefined == timeout || timeout.toMillis < 1) Duration.Undefined else timeout, - receiveTimeoutData._2) - - final override def resetReceiveTimeout(): Unit = setReceiveTimeout(None) - - /* - * CHILDREN - */ - - @volatile - private var _childrenRefsDoNotCallMeDirectly: ChildrenContainer = EmptyChildrenContainer - - def childrenRefs: ChildrenContainer = Unsafe.instance.getObjectVolatile(this, childrenOffset).asInstanceOf[ChildrenContainer] - - final def children: Iterable[ActorRef] = childrenRefs.children - final def getChildren(): java.lang.Iterable[ActorRef] = asJavaIterableConverter(children).asJava - - def actorOf(props: Props): ActorRef = makeChild(this, props, randomName(), async = false) - def actorOf(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = false) - private[akka] def attachChild(props: Props): ActorRef = makeChild(this, props, randomName(), async = true) - private[akka] def attachChild(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = true) - - @volatile private var _nextNameDoNotCallMeDirectly = 0L - final protected def randomName(): String = { - @tailrec def inc(): Long = { - val current = Unsafe.instance.getLongVolatile(this, nextNameOffset) - if (Unsafe.instance.compareAndSwapLong(this, nextNameOffset, current, current + 1)) current - else inc() - } - Helpers.base64(inc()) - } - - final def stop(actor: ActorRef): Unit = { - val started = actor match { - case r: RepointableRef ⇒ r.isStarted - case _ ⇒ true - } - if (childrenRefs.getByRef(actor).isDefined && started) shallDie(this, actor) - actor.asInstanceOf[InternalActorRef].stop() - } - - /* - * ACTOR STATE - */ - - var currentMessage: Envelope = _ var actor: Actor = _ + var currentMessage: Envelope = _ private var behaviorStack: List[Actor.Receive] = emptyBehaviorStack - var watching: Set[ActorRef] = emptyActorRefSet - var watchedBy: Set[ActorRef] = emptyActorRefSet - override final def watch(subject: ActorRef): ActorRef = subject match { - case a: InternalActorRef ⇒ - if (a != self && !watching.contains(a)) { - a.sendSystemMessage(Watch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - watching += a - } - a + /* + * MESSAGE PROCESSING + */ + + //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status + final def systemInvoke(message: SystemMessage): Unit = try { + message match { + case Create() ⇒ create() + case Recreate(cause) ⇒ faultRecreate(cause) + case Watch(watchee, watcher) ⇒ addWatcher(watchee, watcher) + case Unwatch(watchee, watcher) ⇒ remWatcher(watchee, watcher) + case Suspend() ⇒ faultSuspend() + case Resume(inRespToFailure) ⇒ faultResume(inRespToFailure) + case Terminate() ⇒ terminate() + case Supervise(child) ⇒ supervise(child) + case ChildTerminated(child) ⇒ handleChildTerminated(child) + case NoMessage ⇒ // only here to suppress warning + } + } catch { + case e @ (_: InterruptedException | NonFatal(_)) ⇒ handleInvokeFailure(e, "error while processing " + message) } - override final def unwatch(subject: ActorRef): ActorRef = subject match { - case a: InternalActorRef ⇒ - if (a != self && watching.contains(a)) { - a.sendSystemMessage(Unwatch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - watching -= a - } - a + //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status + final def invoke(messageHandle: Envelope): Unit = try { + currentMessage = messageHandle + cancelReceiveTimeout() // FIXME: leave this here??? + messageHandle.message match { + case msg: AutoReceivedMessage ⇒ autoReceiveMessage(messageHandle) + case msg ⇒ receiveMessage(msg) + } + currentMessage = null // reset current message after successful invocation + } catch { + case e @ (_: InterruptedException | NonFatal(_)) ⇒ handleInvokeFailure(e, e.getMessage) + } finally { + checkReceiveTimeout // Reschedule receive timeout + } + + def autoReceiveMessage(msg: Envelope): Unit = { + if (system.settings.DebugAutoReceive) + publish(Debug(self.path.toString, clazz(actor), "received AutoReceiveMessage " + msg)) + + msg.message match { + case Failed(cause) ⇒ handleFailure(sender, cause) + case t: Terminated ⇒ watchedActorTerminated(t.actor); receiveMessage(t) + case Kill ⇒ throw new ActorKilledException("Kill") + case PoisonPill ⇒ self.stop() + case SelectParent(m) ⇒ parent.tell(m, msg.sender) + case SelectChildName(name, m) ⇒ for (c ← getChildByName(name)) c.child.tell(m, msg.sender) + case SelectChildPattern(p, m) ⇒ for (c ← children if p.matcher(c.path.name).matches) c.tell(m, msg.sender) + } + } + + final def receiveMessage(msg: Any): Unit = { + //FIXME replace with behaviorStack.head.applyOrElse(msg, unhandled) + "-optimize" + val head = behaviorStack.head + if (head.isDefinedAt(msg)) head.apply(msg) else actor.unhandled(msg) } /* - * MAILBOX and DISPATCHER - */ - - @volatile private var _mailboxDoNotCallMeDirectly: Mailbox = _ //This must be volatile since it isn't protected by the mailbox status - - @inline final def mailbox: Mailbox = Unsafe.instance.getObjectVolatile(this, mailboxOffset).asInstanceOf[Mailbox] - - @tailrec final def swapMailbox(newMailbox: Mailbox): Mailbox = { - val oldMailbox = mailbox - if (!Unsafe.instance.compareAndSwapObject(this, mailboxOffset, oldMailbox, newMailbox)) swapMailbox(newMailbox) - else oldMailbox - } - - final def hasMessages: Boolean = mailbox.hasMessages - - final def numberOfMessages: Int = mailbox.numberOfMessages - - val dispatcher: MessageDispatcher = system.dispatchers.lookup(props.dispatcher) - - /** - * UntypedActorContext impl - */ - final def getDispatcher(): MessageDispatcher = dispatcher - - final def isTerminated: Boolean = mailbox.isClosed - - final def start(): this.type = { - - /* - * Create the mailbox and enqueue the Create() message to ensure that - * this is processed before anything else. - */ - swapMailbox(dispatcher.createMailbox(this)) - mailbox.setActor(this) - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - mailbox.systemEnqueue(self, Create()) - - // This call is expected to start off the actor by scheduling its mailbox. - dispatcher.attach(this) - - this - } - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def suspend(): Unit = dispatcher.systemDispatch(this, Suspend()) - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def resume(inResponseToFailure: Boolean): Unit = dispatcher.systemDispatch(this, Resume(inResponseToFailure)) - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def restart(cause: Throwable): Unit = dispatcher.systemDispatch(this, Recreate(cause)) - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - final def stop(): Unit = dispatcher.systemDispatch(this, Terminate()) - - def tell(message: Any, sender: ActorRef): Unit = - dispatcher.dispatch(this, Envelope(message, if (sender eq null) system.deadLetters else sender, system)) - - override def sendSystemMessage(message: SystemMessage): Unit = dispatcher.systemDispatch(this, message) - - /* - * ACTOR CONTEXT IMPL + * ACTOR CONTEXT IMPLEMENTATION */ final def sender: ActorRef = currentMessage match { @@ -458,37 +383,9 @@ private[akka] class ActorCell( } /* - * FAILURE HANDLING + * ACTOR INSTANCE HANDLING */ - /* ================= - * T H E R U L E S - * ================= - * - * Actors can be suspended for two reasons: - * - they fail - * - their supervisor gets suspended - * - * In particular they are not suspended multiple times because of cascading - * own failures, i.e. while currentlyFailed() they do not fail again. In case - * of a restart, failures in constructor/preStart count as new failures. - */ - - private def suspendNonRecursive(): Unit = dispatcher suspend this - - private def resumeNonRecursive(): Unit = dispatcher resume this - - /* - * have we told our supervisor that we Failed() and have not yet heard back? - * (actually: we might have heard back but not yet acted upon it, in case of - * a restart with dying children) - * might well be replaced by ref to a Cancellable in the future (see #2299) - */ - private var _failed = false - def currentlyFailed: Boolean = _failed - def setFailed(): Unit = _failed = true - def setNotFailed(): Unit = _failed = false - //This method is in charge of setting up the contextStack and create a new instance of the Actor protected def newActor(): Actor = { contextStack.set(this :: contextStack.get) @@ -509,311 +406,43 @@ private[akka] class ActorCell( } } - //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status - final def systemInvoke(message: SystemMessage) { - - def create(): Unit = if (childrenRefs.isNormal) { - try { - val created = newActor() - actor = created - created.preStart() - checkReceiveTimeout - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(created), "started (" + created + ")")) - } catch { - case NonFatal(i: InstantiationException) ⇒ - throw new ActorInitializationException(self, - """exception during creation, this problem is likely to occur because the class of the Actor you tried to create is either, + private def create(): Unit = if (isNormal) { + try { + val created = newActor() + actor = created + created.preStart() + checkReceiveTimeout + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(created), "started (" + created + ")")) + } catch { + case NonFatal(i: InstantiationException) ⇒ + throw new ActorInitializationException(self, + """exception during creation, this problem is likely to occur because the class of the Actor you tried to create is either, a non-static inner class (in which case make it a static inner class or use Props(new ...) or Props( new UntypedActorFactory ... ) or is missing an appropriate, reachable no-args constructor. """, i.getCause) - case NonFatal(e) ⇒ throw new ActorInitializationException(self, "exception during creation", e) - } - } - - def recreate(cause: Throwable): Unit = - if (childrenRefs.isNormal) { - val failedActor = actor - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(failedActor), "restarting")) - if (failedActor ne null) { - try { - // if the actor fails in preRestart, we can do nothing but log it: it’s best-effort - if (failedActor.context ne null) failedActor.preRestart(cause, Option(currentMessage)) - } catch { - case NonFatal(e) ⇒ - val ex = new PreRestartException(self, e, cause, Option(currentMessage)) - publish(Error(ex, self.path.toString, clazz(failedActor), e.getMessage)) - } finally { - clearActorFields(failedActor) - } - } - assert(mailbox.isSuspended, "mailbox must be suspended during restart, status=" + mailbox.status) - childrenRefs match { - case ct: TerminatingChildrenContainer ⇒ - setChildrenTerminationReason(this, Recreation(cause)) - case _ ⇒ - doRecreate(cause, failedActor) - } - } else { - // need to keep that suspend counter balanced - doResume(inResponseToFailure = false) - } - - def doSuspend(): Unit = { - // done always to keep that suspend counter balanced - suspendNonRecursive() - childrenRefs.suspendChildren() - } - - def doResume(inResponseToFailure: Boolean): Unit = { - // done always to keep that suspend counter balanced - // must happen “atomically” - try resumeNonRecursive() - finally if (inResponseToFailure) setNotFailed() - childrenRefs.resumeChildren() - } - - def addWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { - val watcheeSelf = watchee == self - val watcherSelf = watcher == self - - if (watcheeSelf && !watcherSelf) { - if (!watchedBy.contains(watcher)) { - watchedBy += watcher - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now monitoring " + watcher)) - } - } else if (!watcheeSelf && watcherSelf) { - watch(watchee) - } else { - publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Watch(%s,%s) for %s".format(watchee, watcher, self))) - } - } - - def remWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { - val watcheeSelf = watchee == self - val watcherSelf = watcher == self - - if (watcheeSelf && !watcherSelf) { - if (watchedBy.contains(watcher)) { - watchedBy -= watcher - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopped monitoring " + watcher)) - } - } else if (!watcheeSelf && watcherSelf) { - unwatch(watchee) - } else { - publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Unwatch(%s,%s) for %s".format(watchee, watcher, self))) - } - } - - def terminate() { - setReceiveTimeout(None) - cancelReceiveTimeout - - // stop all children, which will turn childrenRefs into TerminatingChildrenContainer (if there are children) - children foreach stop - - childrenRefs match { - case ct: TerminatingChildrenContainer ⇒ - setChildrenTerminationReason(this, Termination) - // do not process normal messages while waiting for all children to terminate - suspendNonRecursive() - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) - case _ ⇒ doTerminate() - } - } - - def supervise(child: ActorRef): Unit = if (!childrenRefs.isTerminating) { - if (childrenRefs.getByRef(child).isEmpty) addChild(this, child) - handleSupervise(child) - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now supervising " + child)) - } - - try { - message match { - case Create() ⇒ create() - case Recreate(cause) ⇒ recreate(cause) - case Watch(watchee, watcher) ⇒ addWatcher(watchee, watcher) - case Unwatch(watchee, watcher) ⇒ remWatcher(watchee, watcher) - case Suspend() ⇒ doSuspend() - case Resume(inRespToFailure) ⇒ doResume(inRespToFailure) - case Terminate() ⇒ terminate() - case Supervise(child) ⇒ supervise(child) - case ChildTerminated(child) ⇒ handleChildTerminated(child) - case NoMessage ⇒ // only here to suppress warning - } - } catch { - case e @ (_: InterruptedException | NonFatal(_)) ⇒ handleInvokeFailure(e, "error while processing " + message) + case NonFatal(e) ⇒ throw new ActorInitializationException(self, "exception during creation", e) } } - //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status - final def invoke(messageHandle: Envelope): Unit = try { - currentMessage = messageHandle - cancelReceiveTimeout() // FIXME: leave this here??? - messageHandle.message match { - case msg: AutoReceivedMessage ⇒ autoReceiveMessage(messageHandle) - case msg ⇒ receiveMessage(msg) - } - currentMessage = null // reset current message after successful invocation - } catch { - case e @ (_: InterruptedException | NonFatal(_)) ⇒ handleInvokeFailure(e, e.getMessage) - } finally { - checkReceiveTimeout // Reschedule receive timeout - } - - final def handleInvokeFailure(t: Throwable, message: String): Unit = { - publish(Error(t, self.path.toString, clazz(actor), message)) - // prevent any further messages to be processed until the actor has been restarted - if (!currentlyFailed) { - // suspend self; these two must happen “atomically” - try suspendNonRecursive() - finally setFailed() - // suspend children - val skip: Set[ActorRef] = currentMessage match { - case Envelope(Failed(`t`), child) ⇒ Set(child) - case _ ⇒ Set.empty - } - childrenRefs.suspendChildren(skip) - // tell supervisor - t match { // Wrap InterruptedExceptions and rethrow - case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t - case _ ⇒ parent.tell(Failed(t), self) - } - } - } - - def autoReceiveMessage(msg: Envelope): Unit = { - if (system.settings.DebugAutoReceive) - publish(Debug(self.path.toString, clazz(actor), "received AutoReceiveMessage " + msg)) - - msg.message match { - case Failed(cause) ⇒ handleFailure(sender, cause) - case t: Terminated ⇒ watching -= t.actor; receiveMessage(t) - case Kill ⇒ throw new ActorKilledException("Kill") - case PoisonPill ⇒ self.stop() - case SelectParent(m) ⇒ parent.tell(m, msg.sender) - case SelectChildName(name, m) ⇒ for (c ← childrenRefs getByName name) c.child.tell(m, msg.sender) - case SelectChildPattern(p, m) ⇒ for (c ← children if p.matcher(c.path.name).matches) c.tell(m, msg.sender) - } - } - - final def receiveMessage(msg: Any): Unit = { - //FIXME replace with behaviorStack.head.applyOrElse(msg, unhandled) + "-optimize" - val head = behaviorStack.head - if (head.isDefinedAt(msg)) head.apply(msg) else actor.unhandled(msg) - } - - private def doTerminate() { - val a = actor - try if (a ne null) a.postStop() - finally try dispatcher.detach(this) - finally try parent.sendSystemMessage(ChildTerminated(self)) - finally try - if (!watchedBy.isEmpty) { - val terminated = Terminated(self)(existenceConfirmed = true) - try { - watchedBy foreach { - watcher ⇒ - try watcher.tell(terminated, self) catch { - case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(a), "deathwatch")) - } - } - } finally watchedBy = emptyActorRefSet - } - finally try - if (!watching.isEmpty) { - try { - watching foreach { // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - case watchee: InternalActorRef ⇒ try watchee.sendSystemMessage(Unwatch(watchee, self)) catch { - case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(a), "deathwatch")) - } - } - } finally watching = emptyActorRefSet - } - finally { - if (system.settings.DebugLifecycle) - publish(Debug(self.path.toString, clazz(a), "stopped")) - behaviorStack = emptyBehaviorStack - clearActorFields(a) - actor = null - } - } - - private def doRecreate(cause: Throwable, failedActor: Actor): Unit = try { - // must happen “atomically” - try resumeNonRecursive() - finally setNotFailed() - - val survivors = children - - val freshActor = newActor() - actor = freshActor // this must happen before postRestart has a chance to fail - if (freshActor eq failedActor) setActorFields(freshActor, this, self) // If the creator returns the same instance, we need to restore our nulled out fields. - - freshActor.postRestart(cause) - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(freshActor), "restarted")) - - // only after parent is up and running again do restart the children which were not stopped - survivors foreach (child ⇒ - try child.asInstanceOf[InternalActorRef].restart(cause) - catch { - case NonFatal(e) ⇒ publish(Error(e, self.path.toString, clazz(freshActor), "restarting " + child)) - }) - } catch { - case NonFatal(e) ⇒ - clearActorFields(actor) // in order to prevent preRestart() from happening again - handleInvokeFailure(new PostRestartException(self, e, cause), e.getMessage) - } - - final def handleFailure(child: ActorRef, cause: Throwable): Unit = childrenRefs.getByRef(child) match { - case Some(stats) ⇒ if (!actor.supervisorStrategy.handleFailure(this, child, cause, stats, childrenRefs.stats)) throw cause - case None ⇒ publish(Warning(self.path.toString, clazz(actor), "dropping Failed(" + cause + ") from unknown child " + child)) - } - - final def handleChildTerminated(child: ActorRef): Unit = try { - childrenRefs match { - case TerminatingChildrenContainer(_, _, reason) ⇒ - val n = removeChild(this, child) - actor.supervisorStrategy.handleChildTerminated(this, child, children) - if (!n.isInstanceOf[TerminatingChildrenContainer]) reason match { - case Recreation(cause) ⇒ doRecreate(cause, actor) // doRecreate since this is the continuation of "recreate" - case Termination ⇒ doTerminate() - case _ ⇒ - } - case _ ⇒ - removeChild(this, child) - actor.supervisorStrategy.handleChildTerminated(this, child, children) - } - } catch { - case NonFatal(e) ⇒ handleInvokeFailure(e, "handleChildTerminated failed") + private def supervise(child: ActorRef): Unit = if (!isTerminating) { + addChild(child) + handleSupervise(child) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now supervising " + child)) } + // future extension point protected def handleSupervise(child: ActorRef): Unit = child match { case r: RepointableActorRef ⇒ r.activate() case _ ⇒ } - final def checkReceiveTimeout() { - val recvtimeout = receiveTimeoutData - if (Duration.Undefined != recvtimeout._1 && !mailbox.hasMessages) { - recvtimeout._2.cancel() //Cancel any ongoing future - //Only reschedule if desired and there are currently no more messages to be processed - receiveTimeoutData = (recvtimeout._1, system.scheduler.scheduleOnce(recvtimeout._1, self, ReceiveTimeout)) - } else cancelReceiveTimeout() - - } - - final def cancelReceiveTimeout(): Unit = - if (receiveTimeoutData._2 ne emptyCancellable) { - receiveTimeoutData._2.cancel() - receiveTimeoutData = (receiveTimeoutData._1, emptyCancellable) - } - - final def clearActorFields(actorInstance: Actor): Unit = { + final protected def clearActorFields(actorInstance: Actor): Unit = { setActorFields(actorInstance, context = null, self = system.deadLetters) currentMessage = null + behaviorStack = emptyBehaviorStack } - final def setActorFields(actorInstance: Actor, context: ActorContext, self: ActorRef) { + final protected def setActorFields(actorInstance: Actor, context: ActorContext, self: ActorRef) { @tailrec def lookupAndSetField(clazz: Class[_], actor: Actor, name: String, value: Any): Boolean = { val success = try { @@ -839,8 +468,8 @@ private[akka] class ActorCell( } // logging is not the main purpose, and if it fails there’s nothing we can do - private final def publish(e: LogEvent): Unit = try system.eventStream.publish(e) catch { case NonFatal(_) ⇒ } + protected final def publish(e: LogEvent): Unit = try system.eventStream.publish(e) catch { case NonFatal(_) ⇒ } - private final def clazz(o: AnyRef): Class[_] = if (o eq null) this.getClass else o.getClass + protected final def clazz(o: AnyRef): Class[_] = if (o eq null) this.getClass else o.getClass } diff --git a/akka-actor/src/main/scala/akka/actor/ActorRef.scala b/akka-actor/src/main/scala/akka/actor/ActorRef.scala index 0d360f4559..6cbd821af4 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorRef.scala @@ -307,7 +307,7 @@ private[akka] class LocalActorRef private[akka] ( * to inject “synthetic” actor paths like “/temp”. */ protected def getSingleChild(name: String): InternalActorRef = - actorCell.childrenRefs.getByName(name) match { + actorCell.getChildByName(name) match { case Some(crs) ⇒ crs.child.asInstanceOf[InternalActorRef] case None ⇒ Nobody } diff --git a/akka-actor/src/main/scala/akka/actor/ActorSystem.scala b/akka-actor/src/main/scala/akka/actor/ActorSystem.scala index 4f5eeb41a7..c155f9d092 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorSystem.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorSystem.scala @@ -15,6 +15,7 @@ import akka.util._ import akka.util.internal.{ HashedWheelTimer, ConcurrentIdentityHashMap } import java.util.concurrent.{ ThreadFactory, CountDownLatch, TimeoutException, RejectedExecutionException } import java.util.concurrent.TimeUnit.MILLISECONDS +import akka.actor.cell.ChildrenContainer object ActorSystem { diff --git a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala index 0be24eee51..61a28f3ccb 100644 --- a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala @@ -230,7 +230,8 @@ abstract class SupervisorStrategy { /** * This method is called after the child has been removed from the set of children. - * It does not need to do anything special. + * It does not need to do anything special. Exceptions thrown from this method + * do NOT make the actor fail if this happens during termination. */ def handleChildTerminated(context: ActorContext, child: ActorRef, children: Iterable[ActorRef]): Unit diff --git a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala index d7493e42dc..b08c7adbee 100644 --- a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala @@ -16,6 +16,7 @@ import akka.dispatch.MessageDispatcher import java.util.concurrent.locks.ReentrantLock import akka.event.Logging.Warning import scala.collection.mutable.Queue +import akka.actor.cell.ChildrenContainer /** * This actor ref starts out with some dummy cell (by default just enqueuing @@ -102,7 +103,7 @@ private[akka] class RepointableActorRef( case ".." ⇒ getParent.getChild(name) case "" ⇒ getChild(name) case other ⇒ - underlying.childrenRefs.getByName(other) match { + underlying.getChildByName(other) match { case Some(crs) ⇒ crs.child.asInstanceOf[InternalActorRef].getChild(name) case None ⇒ Nobody } @@ -176,6 +177,7 @@ private[akka] class UnstartedCell(val systemImpl: ActorSystemImpl, val self: Rep def isTerminated: Boolean = false def parent: InternalActorRef = supervisor def childrenRefs: ChildrenContainer = ChildrenContainer.EmptyChildrenContainer + def getChildByName(name: String): Option[ChildRestartStats] = None def tell(message: Any, sender: ActorRef): Unit = { lock.lock() try { diff --git a/akka-actor/src/main/scala/akka/actor/cell/Children.scala b/akka-actor/src/main/scala/akka/actor/cell/Children.scala new file mode 100644 index 0000000000..07871ed6d2 --- /dev/null +++ b/akka-actor/src/main/scala/akka/actor/cell/Children.scala @@ -0,0 +1,181 @@ +/** + * Copyright (C) 2009-2012 Typesafe Inc. + */ + +package akka.actor.cell + +import scala.annotation.tailrec +import scala.collection.JavaConverters.asJavaIterableConverter +import akka.actor.{ RepointableRef, Props, InternalActorRef, ActorRef, ActorCell } +import akka.util.{ Unsafe, Helpers } +import akka.actor.InvalidActorNameException +import akka.serialization.SerializationExtension +import akka.util.NonFatal +import akka.actor.NoSerializationVerificationNeeded +import akka.actor.ActorPath +import akka.actor.ChildRestartStats +import akka.actor.ChildRestartStats +import akka.actor.ChildRestartStats +import akka.actor.ChildRestartStats + +trait Children { this: ActorCell ⇒ + + import ChildrenContainer._ + + @volatile + private var _childrenRefsDoNotCallMeDirectly: ChildrenContainer = EmptyChildrenContainer + + def childrenRefs: ChildrenContainer = + Unsafe.instance.getObjectVolatile(this, AbstractActorCell.childrenOffset).asInstanceOf[ChildrenContainer] + + final def children: Iterable[ActorRef] = childrenRefs.children + final def getChildren(): java.lang.Iterable[ActorRef] = children.asJava + + def actorOf(props: Props): ActorRef = makeChild(this, props, randomName(), async = false) + def actorOf(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = false) + private[akka] def attachChild(props: Props): ActorRef = makeChild(this, props, randomName(), async = true) + private[akka] def attachChild(props: Props, name: String): ActorRef = makeChild(this, props, checkName(name), async = true) + + @volatile private var _nextNameDoNotCallMeDirectly = 0L + final protected def randomName(): String = { + @tailrec def inc(): Long = { + val current = Unsafe.instance.getLongVolatile(this, AbstractActorCell.nextNameOffset) + if (Unsafe.instance.compareAndSwapLong(this, AbstractActorCell.nextNameOffset, current, current + 1)) current + else inc() + } + Helpers.base64(inc()) + } + + final def stop(actor: ActorRef): Unit = { + val started = actor match { + case r: RepointableRef ⇒ r.isStarted + case _ ⇒ true + } + if (childrenRefs.getByRef(actor).isDefined && started) shallDie(actor) + actor.asInstanceOf[InternalActorRef].stop() + } + + /* + * low level CAS helpers + */ + + @inline private def swapChildrenRefs(oldChildren: ChildrenContainer, newChildren: ChildrenContainer): Boolean = + Unsafe.instance.compareAndSwapObject(this, AbstractActorCell.childrenOffset, oldChildren, newChildren) + + @tailrec final protected def reserveChild(name: String): Boolean = { + val c = childrenRefs + swapChildrenRefs(c, c.reserve(name)) || reserveChild(name) + } + + @tailrec final protected def unreserveChild(name: String): Boolean = { + val c = childrenRefs + swapChildrenRefs(c, c.unreserve(name)) || unreserveChild(name) + } + + final protected def addChild(ref: ActorRef): Boolean = { + @tailrec def rec(): Boolean = { + val c = childrenRefs + swapChildrenRefs(c, c.add(ref)) || rec() + } + if (childrenRefs.getByRef(ref).isEmpty) rec() else false + } + + @tailrec final protected def shallDie(ref: ActorRef): Boolean = { + val c = childrenRefs + swapChildrenRefs(c, c.shallDie(ref)) || shallDie(ref) + } + + @tailrec final private def removeChild(ref: ActorRef): ChildrenContainer = { + val c = childrenRefs + val n = c.remove(ref) + if (swapChildrenRefs(c, n)) n + else removeChild(ref) + } + + @tailrec final protected def setChildrenTerminationReason(reason: ChildrenContainer.SuspendReason): Boolean = { + childrenRefs match { + case c: ChildrenContainer.TerminatingChildrenContainer ⇒ + swapChildrenRefs(c, c.copy(reason = reason)) || setChildrenTerminationReason(reason) + case _ ⇒ false + } + } + + /* + * ActorCell-internal API + */ + + protected def isNormal = childrenRefs.isNormal + + protected def isTerminating = childrenRefs.isTerminating + + protected def suspendChildren(skip: Set[ActorRef] = Set.empty): Unit = + childrenRefs.stats collect { + case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child + } foreach (_.asInstanceOf[InternalActorRef].suspend()) + + protected def resumeChildren(): Unit = + childrenRefs.stats foreach (_.child.asInstanceOf[InternalActorRef].resume(inResponseToFailure = false)) + + def getChildByName(name: String): Option[ChildRestartStats] = childrenRefs.getByName(name) + + protected def getChildByRef(ref: ActorRef): Option[ChildRestartStats] = childrenRefs.getByRef(ref) + + protected def getAllChildStats: Iterable[ChildRestartStats] = childrenRefs.stats + + protected def removeChildAndGetStateChange(child: ActorRef): Option[SuspendReason] = { + childrenRefs match { + case TerminatingChildrenContainer(_, _, reason) ⇒ + val n = removeChild(child) + if (!n.isInstanceOf[TerminatingChildrenContainer]) Some(reason) else None + case _ ⇒ removeChild(child); None + } + } + + /* + * Private helpers + */ + + private def checkName(name: String): String = { + import ActorPath.ElementRegex + name match { + case null ⇒ throw new InvalidActorNameException("actor name must not be null") + case "" ⇒ throw new InvalidActorNameException("actor name must not be empty") + case ElementRegex() ⇒ name + case _ ⇒ throw new InvalidActorNameException("illegal actor name '" + name + "', must conform to " + ElementRegex) + } + } + + private def makeChild(cell: ActorCell, props: Props, name: String, async: Boolean): ActorRef = { + if (cell.system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) { + val ser = SerializationExtension(cell.system) + ser.serialize(props.creator) match { + 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 + * (internal calls cannot happen anyway because we are suspended) + */ + if (cell.childrenRefs.isTerminating) throw new IllegalStateException("cannot create children while terminating or terminated") + else { + reserveChild(name) + // this name will either be unreserved or overwritten with a real child below + val actor = + try { + cell.provider.actorOf(cell.systemImpl, props, cell.self, cell.self.path / name, + systemService = false, deploy = None, lookupDeploy = true, async = async) + } catch { + case NonFatal(e) ⇒ + unreserveChild(name) + throw e + } + addChild(actor) + actor + } + } + +} \ No newline at end of file diff --git a/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala similarity index 67% rename from akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala rename to akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala index 692dd3ba07..641be552e5 100644 --- a/akka-actor/src/main/scala/akka/actor/ChildrenContainer.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala @@ -2,13 +2,24 @@ * Copyright (C) 2009-2012 Typesafe Inc. */ -package akka.actor +package akka.actor.cell import scala.collection.immutable.TreeMap import scala.annotation.tailrec import akka.util.Unsafe import akka.serialization.SerializationExtension import akka.util.NonFatal +import akka.actor.ActorPath.ElementRegex +import akka.actor.ActorCell +import akka.actor.ActorRef +import akka.actor.ChildNameReserved +import akka.actor.ChildRestartStats +import akka.actor.ChildStats +import akka.actor.InternalActorRef +import akka.actor.InvalidActorNameException +import akka.actor.NoSerializationVerificationNeeded +import akka.actor.Props +import scala.annotation.tailrec /** * INTERNAL API @@ -33,13 +44,6 @@ private[akka] trait ChildrenContainer { def isTerminating: Boolean = false def isNormal: Boolean = true - - def suspendChildren(skip: Set[ActorRef] = Set.empty): Unit = - stats collect { - case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child - } foreach (_.asInstanceOf[InternalActorRef].suspend()) - - def resumeChildren(): Unit = stats foreach (_.child.asInstanceOf[InternalActorRef].suspend()) } /** @@ -50,88 +54,6 @@ private[akka] trait ChildrenContainer { */ private[akka] object ChildrenContainer { - // low level CAS helpers - @inline private def swapChildrenRefs(cell: ActorCell, oldChildren: ChildrenContainer, newChildren: ChildrenContainer): Boolean = - Unsafe.instance.compareAndSwapObject(cell, AbstractActorCell.childrenOffset, oldChildren, newChildren) - - @tailrec final def reserveChild(cell: ActorCell, name: String): Boolean = { - val c = cell.childrenRefs - swapChildrenRefs(cell, c, c.reserve(name)) || reserveChild(cell, name) - } - - @tailrec final def unreserveChild(cell: ActorCell, name: String): Boolean = { - val c = cell.childrenRefs - swapChildrenRefs(cell, c, c.unreserve(name)) || unreserveChild(cell, name) - } - - @tailrec final def addChild(cell: ActorCell, ref: ActorRef): Boolean = { - val c = cell.childrenRefs - swapChildrenRefs(cell, c, c.add(ref)) || addChild(cell, ref) - } - - @tailrec final def shallDie(cell: ActorCell, ref: ActorRef): Boolean = { - val c = cell.childrenRefs - swapChildrenRefs(cell, c, c.shallDie(ref)) || shallDie(cell, ref) - } - - @tailrec final def removeChild(cell: ActorCell, ref: ActorRef): ChildrenContainer = { - val c = cell.childrenRefs - val n = c.remove(ref) - if (swapChildrenRefs(cell, c, n)) n - else removeChild(cell, ref) - } - - @tailrec final def setChildrenTerminationReason(cell: ActorCell, reason: ChildrenContainer.SuspendReason): Boolean = { - cell.childrenRefs match { - case c: ChildrenContainer.TerminatingChildrenContainer ⇒ - swapChildrenRefs(cell, c, c.copy(reason = reason)) || setChildrenTerminationReason(cell, reason) - case _ ⇒ false - } - } - - def checkName(name: String): String = { - import ActorPath.ElementRegex - name match { - case null ⇒ throw new InvalidActorNameException("actor name must not be null") - case "" ⇒ throw new InvalidActorNameException("actor name must not be empty") - case ElementRegex() ⇒ name - case _ ⇒ throw new InvalidActorNameException("illegal actor name '" + name + "', must conform to " + ElementRegex) - } - } - - def makeChild(cell: ActorCell, props: Props, name: String, async: Boolean): ActorRef = { - if (cell.system.settings.SerializeAllCreators && !props.creator.isInstanceOf[NoSerializationVerificationNeeded]) { - val ser = SerializationExtension(cell.system) - ser.serialize(props.creator) match { - 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 - * (internal calls cannot happen anyway because we are suspended) - */ - if (cell.childrenRefs.isTerminating) throw new IllegalStateException("cannot create children while terminating or terminated") - else { - reserveChild(cell, name) - // this name will either be unreserved or overwritten with a real child below - val actor = - try { - cell.provider.actorOf(cell.systemImpl, props, cell.self, cell.self.path / name, - systemService = false, deploy = None, lookupDeploy = true, async = async) - } catch { - case NonFatal(e) ⇒ - unreserveChild(cell, name) - throw e - } - addChild(cell, actor) - actor - } - } - sealed trait SuspendReason case object UserRequest extends SuspendReason case class Recreation(cause: Throwable) extends SuspendReason diff --git a/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala new file mode 100644 index 0000000000..e65cd134fe --- /dev/null +++ b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala @@ -0,0 +1,93 @@ +package akka.actor.cell + +import akka.actor.{ InternalActorRef, ActorRef, ActorCell } +import akka.dispatch.{ Watch, Unwatch } +import akka.event.Logging._ +import akka.util.NonFatal +import akka.actor.Terminated +import akka.actor.Actor + +trait DeathWatch { this: ActorCell ⇒ + + private var watching: Set[ActorRef] = ActorCell.emptyActorRefSet + private var watchedBy: Set[ActorRef] = ActorCell.emptyActorRefSet + + override final def watch(subject: ActorRef): ActorRef = subject match { + case a: InternalActorRef ⇒ + if (a != self && !watching.contains(a)) { + a.sendSystemMessage(Watch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + watching += a + } + a + } + + override final def unwatch(subject: ActorRef): ActorRef = subject match { + case a: InternalActorRef ⇒ + if (a != self && watching.contains(a)) { + a.sendSystemMessage(Unwatch(a, self)) // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + watching -= a + } + a + } + + protected def watchedActorTerminated(ref: ActorRef): Unit = watching -= ref + + protected def tellWatchersWeDied(actor: Actor): Unit = { + if (!watchedBy.isEmpty) { + val terminated = Terminated(self)(existenceConfirmed = true) + try { + watchedBy foreach { + watcher ⇒ + try watcher.tell(terminated, self) catch { + case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(actor), "deathwatch")) + } + } + } finally watchedBy = ActorCell.emptyActorRefSet + } + } + + protected def unwatchWatchedActors(actor: Actor): Unit = { + if (!watching.isEmpty) { + try { + watching foreach { // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + case watchee: InternalActorRef ⇒ try watchee.sendSystemMessage(Unwatch(watchee, self)) catch { + case NonFatal(t) ⇒ publish(Error(t, self.path.toString, clazz(actor), "deathwatch")) + } + } + } finally watching = ActorCell.emptyActorRefSet + } + } + + protected def addWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { + val watcheeSelf = watchee == self + val watcherSelf = watcher == self + + if (watcheeSelf && !watcherSelf) { + if (!watchedBy.contains(watcher)) { + watchedBy += watcher + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "now monitoring " + watcher)) + } + } else if (!watcheeSelf && watcherSelf) { + watch(watchee) + } else { + publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Watch(%s,%s) for %s".format(watchee, watcher, self))) + } + } + + protected def remWatcher(watchee: ActorRef, watcher: ActorRef): Unit = { + val watcheeSelf = watchee == self + val watcherSelf = watcher == self + + if (watcheeSelf && !watcherSelf) { + if (watchedBy.contains(watcher)) { + watchedBy -= watcher + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopped monitoring " + watcher)) + } + } else if (!watcheeSelf && watcherSelf) { + unwatch(watchee) + } else { + publish(Warning(self.path.toString, clazz(actor), "BUG: illegal Unwatch(%s,%s) for %s".format(watchee, watcher, self))) + } + } + +} \ No newline at end of file diff --git a/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala new file mode 100644 index 0000000000..7cffefba2e --- /dev/null +++ b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala @@ -0,0 +1,81 @@ +/** + * Copyright (C) 2009-2012 Typesafe Inc. + */ + +package akka.actor.cell + +import scala.annotation.tailrec +import akka.actor.ActorRef +import akka.dispatch.SystemMessage +import akka.util.Unsafe +import akka.dispatch.MessageDispatcher +import akka.dispatch.Suspend +import akka.dispatch.Recreate +import akka.actor.ActorCell +import akka.dispatch.Terminate +import akka.dispatch.Envelope +import akka.dispatch.Resume +import akka.dispatch.Mailbox +import akka.dispatch.Create + +trait Dispatch { this: ActorCell ⇒ + + @volatile private var _mailboxDoNotCallMeDirectly: Mailbox = _ //This must be volatile since it isn't protected by the mailbox status + + @inline final def mailbox: Mailbox = Unsafe.instance.getObjectVolatile(this, AbstractActorCell.mailboxOffset).asInstanceOf[Mailbox] + + @tailrec final def swapMailbox(newMailbox: Mailbox): Mailbox = { + val oldMailbox = mailbox + if (!Unsafe.instance.compareAndSwapObject(this, AbstractActorCell.mailboxOffset, oldMailbox, newMailbox)) swapMailbox(newMailbox) + else oldMailbox + } + + final def hasMessages: Boolean = mailbox.hasMessages + + final def numberOfMessages: Int = mailbox.numberOfMessages + + val dispatcher: MessageDispatcher = system.dispatchers.lookup(props.dispatcher) + + /** + * UntypedActorContext impl + */ + final def getDispatcher(): MessageDispatcher = dispatcher + + final def isTerminated: Boolean = mailbox.isClosed + + final def start(): this.type = { + + /* + * Create the mailbox and enqueue the Create() message to ensure that + * this is processed before anything else. + */ + swapMailbox(dispatcher.createMailbox(this)) + mailbox.setActor(this) + + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + mailbox.systemEnqueue(self, Create()) + + // This call is expected to start off the actor by scheduling its mailbox. + dispatcher.attach(this) + + this + } + + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + final def suspend(): Unit = dispatcher.systemDispatch(this, Suspend()) + + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + final def resume(inResponseToFailure: Boolean): Unit = dispatcher.systemDispatch(this, Resume(inResponseToFailure)) + + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + final def restart(cause: Throwable): Unit = dispatcher.systemDispatch(this, Recreate(cause)) + + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + final def stop(): Unit = dispatcher.systemDispatch(this, Terminate()) + + def tell(message: Any, sender: ActorRef): Unit = + dispatcher.dispatch(this, Envelope(message, if (sender eq null) system.deadLetters else sender, system)) + + override def sendSystemMessage(message: SystemMessage): Unit = dispatcher.systemDispatch(this, message) + +} \ No newline at end of file diff --git a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala new file mode 100644 index 0000000000..1ef83575ce --- /dev/null +++ b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala @@ -0,0 +1,184 @@ +package akka.actor.cell + +import akka.actor.PreRestartException +import akka.actor.ActorCell +import akka.util.NonFatal +import akka.actor.ActorRef +import akka.actor.PostRestartException +import akka.actor.Actor +import akka.dispatch.Envelope +import akka.dispatch.ChildTerminated +import akka.actor.Failed +import akka.actor.InternalActorRef +import akka.event.Logging._ +import akka.actor.ActorInterruptedException + +trait FaultHandling { this: ActorCell ⇒ + + /* ================= + * T H E R U L E S + * ================= + * + * Actors can be suspended for two reasons: + * - they fail + * - their supervisor gets suspended + * + * In particular they are not suspended multiple times because of cascading + * own failures, i.e. while currentlyFailed() they do not fail again. In case + * of a restart, failures in constructor/preStart count as new failures. + */ + + private def suspendNonRecursive(): Unit = dispatcher suspend this + + private def resumeNonRecursive(): Unit = dispatcher resume this + + /* + * have we told our supervisor that we Failed() and have not yet heard back? + * (actually: we might have heard back but not yet acted upon it, in case of + * a restart with dying children) + * might well be replaced by ref to a Cancellable in the future (see #2299) + */ + private var _failed = false + private def currentlyFailed: Boolean = _failed + private def setFailed(): Unit = _failed = true + private def setNotFailed(): Unit = _failed = false + + protected def faultRecreate(cause: Throwable): Unit = + if (isNormal) { + val failedActor = actor + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(failedActor), "restarting")) + if (failedActor ne null) { + try { + // if the actor fails in preRestart, we can do nothing but log it: it’s best-effort + if (failedActor.context ne null) failedActor.preRestart(cause, Option(currentMessage)) + } catch { + case NonFatal(e) ⇒ + val ex = new PreRestartException(self, e, cause, Option(currentMessage)) + publish(Error(ex, self.path.toString, clazz(failedActor), e.getMessage)) + } finally { + clearActorFields(failedActor) + } + } + assert(mailbox.isSuspended, "mailbox must be suspended during restart, status=" + mailbox.status) + if (!setChildrenTerminationReason(ChildrenContainer.Recreation(cause))) finishRecreate(cause, failedActor) + } else { + // need to keep that suspend counter balanced + faultResume(inResponseToFailure = false) + } + + protected def faultSuspend(): Unit = { + // done always to keep that suspend counter balanced + suspendNonRecursive() + suspendChildren() + } + + protected def faultResume(inResponseToFailure: Boolean): Unit = { + // done always to keep that suspend counter balanced + // must happen “atomically” + try resumeNonRecursive() + finally if (inResponseToFailure) setNotFailed() + resumeChildren() + } + + protected def terminate() { + setReceiveTimeout(None) + cancelReceiveTimeout + + // stop all children, which will turn childrenRefs into TerminatingChildrenContainer (if there are children) + children foreach stop + + if (setChildrenTerminationReason(ChildrenContainer.Termination)) { + // do not process normal messages while waiting for all children to terminate + suspendNonRecursive() + // do not propagate failures during shutdown to the supervisor + setFailed() + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) + } else finishTerminate() + } + + final def handleInvokeFailure(t: Throwable, message: String): Unit = { + publish(Error(t, self.path.toString, clazz(actor), message)) + // prevent any further messages to be processed until the actor has been restarted + if (!currentlyFailed) try { + suspendNonRecursive() + setFailed() + // suspend children + val skip: Set[ActorRef] = currentMessage match { + case Envelope(Failed(`t`), child) ⇒ Set(child) + case _ ⇒ Set.empty + } + suspendChildren(skip) + // tell supervisor + t match { // Wrap InterruptedExceptions and rethrow + case _: InterruptedException ⇒ parent.tell(Failed(new ActorInterruptedException(t)), self); throw t + case _ ⇒ parent.tell(Failed(t), self) + } + } catch { + case NonFatal(e) ⇒ + publish(Error(e, self.path.toString, clazz(actor), "emergency stop: exception in failure handling")) + try children foreach stop + finally finishTerminate() + } + } + + private def finishTerminate() { + val a = actor + try if (a ne null) a.postStop() + finally try dispatcher.detach(this) + finally try parent.sendSystemMessage(ChildTerminated(self)) + finally try tellWatchersWeDied(a) + finally try unwatchWatchedActors(a) + finally { + if (system.settings.DebugLifecycle) + publish(Debug(self.path.toString, clazz(a), "stopped")) + clearActorFields(a) + actor = null + } + } + + private def finishRecreate(cause: Throwable, failedActor: Actor): Unit = try { + try resumeNonRecursive() + finally setNotFailed() // must happen in any case, so that failure is propagated + + val survivors = children + + val freshActor = newActor() + actor = freshActor // this must happen before postRestart has a chance to fail + if (freshActor eq failedActor) setActorFields(freshActor, this, self) // If the creator returns the same instance, we need to restore our nulled out fields. + + freshActor.postRestart(cause) + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(freshActor), "restarted")) + + // only after parent is up and running again do restart the children which were not stopped + survivors foreach (child ⇒ + try child.asInstanceOf[InternalActorRef].restart(cause) + catch { + case NonFatal(e) ⇒ publish(Error(e, self.path.toString, clazz(freshActor), "restarting " + child)) + }) + } catch { + case NonFatal(e) ⇒ + clearActorFields(actor) // in order to prevent preRestart() from happening again + handleInvokeFailure(new PostRestartException(self, e, cause), e.getMessage) + } + + final protected def handleFailure(child: ActorRef, cause: Throwable): Unit = getChildByRef(child) match { + case Some(stats) ⇒ if (!actor.supervisorStrategy.handleFailure(this, child, cause, stats, getAllChildStats)) throw cause + case None ⇒ publish(Warning(self.path.toString, clazz(actor), "dropping Failed(" + cause + ") from unknown child " + child)) + } + + final protected def handleChildTerminated(child: ActorRef): Unit = try { + val status = removeChildAndGetStateChange(child) + actor.supervisorStrategy.handleChildTerminated(this, child, children) + /* + * if the removal changed the state of the (terminating) children container, + * then we are continuing the previously suspended recreate/terminate action + */ + status match { + case Some(ChildrenContainer.Recreation(cause)) ⇒ finishRecreate(cause, actor) + case Some(ChildrenContainer.Termination) ⇒ finishTerminate() + case _ ⇒ + } + } catch { + case NonFatal(e) ⇒ handleInvokeFailure(e, "handleChildTerminated failed") + } +} \ No newline at end of file diff --git a/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala b/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala new file mode 100644 index 0000000000..960372b488 --- /dev/null +++ b/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala @@ -0,0 +1,54 @@ +/** + * Copyright (C) 2009-2012 Typesafe Inc. + */ + +package akka.actor.cell + +import ReceiveTimeout.emptyReceiveTimeoutData +import akka.actor.ActorCell +import akka.actor.ActorCell.emptyCancellable +import akka.actor.Cancellable +import akka.util.Duration + +object ReceiveTimeout { + final val emptyReceiveTimeoutData: (Duration, Cancellable) = (Duration.Undefined, ActorCell.emptyCancellable) +} + +trait ReceiveTimeout { this: ActorCell ⇒ + + import ReceiveTimeout._ + import ActorCell._ + + private var receiveTimeoutData: (Duration, Cancellable) = emptyReceiveTimeoutData + + final def receiveTimeout: Option[Duration] = receiveTimeoutData._1 match { + case Duration.Undefined ⇒ None + case duration ⇒ Some(duration) + } + + final def setReceiveTimeout(timeout: Option[Duration]): Unit = setReceiveTimeout(timeout.getOrElse(Duration.Undefined)) + + final def setReceiveTimeout(timeout: Duration): Unit = + receiveTimeoutData = ( + if (Duration.Undefined == timeout || timeout.toMillis < 1) Duration.Undefined else timeout, + receiveTimeoutData._2) + + final def resetReceiveTimeout(): Unit = setReceiveTimeout(None) + + final def checkReceiveTimeout() { + val recvtimeout = receiveTimeoutData + if (Duration.Undefined != recvtimeout._1 && !mailbox.hasMessages) { + recvtimeout._2.cancel() //Cancel any ongoing future + //Only reschedule if desired and there are currently no more messages to be processed + receiveTimeoutData = (recvtimeout._1, system.scheduler.scheduleOnce(recvtimeout._1, self, akka.actor.ReceiveTimeout)) + } else cancelReceiveTimeout() + + } + + final def cancelReceiveTimeout(): Unit = + if (receiveTimeoutData._2 ne emptyCancellable) { + receiveTimeoutData._2.cancel() + receiveTimeoutData = (receiveTimeoutData._1, emptyCancellable) + } + +} \ No newline at end of file From 9ed65d9e8e4a7a92f4345f4f55c69baa9b7e2477 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 8 Jul 2012 17:46:14 +0200 Subject: [PATCH 06/14] fix handling of handleChildTerminated, see #2301 --- .../scala/akka/actor/SupervisorMiscSpec.scala | 3 +++ .../scala/akka/actor/cell/FaultHandling.scala | 16 +++++++++++----- 2 files changed, 14 insertions(+), 5 deletions(-) diff --git a/akka-actor-tests/src/test/scala/akka/actor/SupervisorMiscSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/SupervisorMiscSpec.scala index ec0c51e9ae..bf47abd037 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/SupervisorMiscSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/SupervisorMiscSpec.scala @@ -134,6 +134,9 @@ class SupervisorMiscSpec extends AkkaSpec(SupervisorMiscSpec.config) with Defaul })) parent ! "engage" expectMsg("green") + EventFilter[IllegalStateException]("handleChildTerminated failed", occurrences = 1) intercept { + system.stop(parent) + } } } diff --git a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala index 1ef83575ce..0b4e93c793 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala @@ -166,9 +166,17 @@ trait FaultHandling { this: ActorCell ⇒ case None ⇒ publish(Warning(self.path.toString, clazz(actor), "dropping Failed(" + cause + ") from unknown child " + child)) } - final protected def handleChildTerminated(child: ActorRef): Unit = try { + final protected def handleChildTerminated(child: ActorRef): Unit = { val status = removeChildAndGetStateChange(child) - actor.supervisorStrategy.handleChildTerminated(this, child, children) + /* + * if this fails, we do nothing in case of terminating/restarting state, + * otherwise tell the supervisor etc. (in that second case, the match + * below will hit the empty default case, too) + */ + try actor.supervisorStrategy.handleChildTerminated(this, child, children) + catch { + case NonFatal(e) ⇒ handleInvokeFailure(e, "handleChildTerminated failed") + } /* * if the removal changed the state of the (terminating) children container, * then we are continuing the previously suspended recreate/terminate action @@ -178,7 +186,5 @@ trait FaultHandling { this: ActorCell ⇒ case Some(ChildrenContainer.Termination) ⇒ finishTerminate() case _ ⇒ } - } catch { - case NonFatal(e) ⇒ handleInvokeFailure(e, "handleChildTerminated failed") } -} \ No newline at end of file +} From 3c809e24208b528e656999df3a01c9db304b0880 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 8 Jul 2012 17:51:19 +0200 Subject: [PATCH 07/14] =?UTF-8?q?review=20comments:=20don=E2=80=99t=20incl?= =?UTF-8?q?ude=20risky=20things=20in=20messages?= MIME-Version: 1.0 Content-Type: text/plain; charset=UTF-8 Content-Transfer-Encoding: 8bit --- akka-actor/src/main/scala/akka/actor/Actor.scala | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/akka-actor/src/main/scala/akka/actor/Actor.scala b/akka-actor/src/main/scala/akka/actor/Actor.scala index d7933f21cd..1b8ee9fe85 100644 --- a/akka-actor/src/main/scala/akka/actor/Actor.scala +++ b/akka-actor/src/main/scala/akka/actor/Actor.scala @@ -137,7 +137,7 @@ class ActorInitializationException private[akka] (val actor: ActorRef, message: * @param msg is the message which was optionally passed into preRestart() */ class PreRestartException private[akka] (actor: ActorRef, cause: Throwable, val origCause: Throwable, val msg: Option[Any]) - extends ActorInitializationException(actor, "exception in preRestart(" + origCause + ", " + msg + ")", cause) { + extends ActorInitializationException(actor, "exception in preRestart(" + origCause.getClass + ", " + msg.getClass + ")", cause) { } /** @@ -149,7 +149,7 @@ class PreRestartException private[akka] (actor: ActorRef, cause: Throwable, val * @param origCause is the exception which caused the restart in the first place */ class PostRestartException private[akka] (actor: ActorRef, cause: Throwable, val origCause: Throwable) - extends ActorInitializationException(actor, "exception post restart (" + origCause + ")", cause) { + extends ActorInitializationException(actor, "exception post restart (" + origCause.getClass + ")", cause) { } /** From f362966ee8e1e9cf8d1eaacfd38d98e59931b6c7 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 8 Jul 2012 17:59:59 +0200 Subject: [PATCH 08/14] add test for scrapping of behaviorStack during restart --- .../scala/akka/actor/ActorLifeCycleSpec.scala | 24 +++++++++++++++++++ 1 file changed, 24 insertions(+) diff --git a/akka-actor-tests/src/test/scala/akka/actor/ActorLifeCycleSpec.scala b/akka-actor-tests/src/test/scala/akka/actor/ActorLifeCycleSpec.scala index d87aaaaee6..9536755d0a 100644 --- a/akka-actor-tests/src/test/scala/akka/actor/ActorLifeCycleSpec.scala +++ b/akka-actor-tests/src/test/scala/akka/actor/ActorLifeCycleSpec.scala @@ -116,6 +116,30 @@ class ActorLifeCycleSpec extends AkkaSpec with BeforeAndAfterEach with ImplicitS expectNoMsg(1 seconds) system.stop(supervisor) } + + "clear the behavior stack upon restart" in { + case class Become(recv: ActorContext ⇒ Receive) + val a = system.actorOf(Props(new Actor { + def receive = { + case Become(beh) ⇒ context.become(beh(context), discardOld = false); sender ! "ok" + case x ⇒ sender ! 42 + } + })) + a ! "hello" + expectMsg(42) + a ! Become(ctx ⇒ { + case "fail" ⇒ throw new RuntimeException("buh") + case x ⇒ ctx.sender ! 43 + }) + expectMsg("ok") + a ! "hello" + expectMsg(43) + EventFilter[RuntimeException]("buh", occurrences = 1) intercept { + a ! "fail" + } + a ! "hello" + expectMsg(42) + } } } From d5248a3346e851c618793041096a20de9aae34f3 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 8 Jul 2012 19:05:15 +0200 Subject: [PATCH 09/14] add copyright headers to akka.actor.cell.* --- akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala | 4 ++++ akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala | 4 ++++ 2 files changed, 8 insertions(+) diff --git a/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala index e65cd134fe..1af0aca67a 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala @@ -1,3 +1,7 @@ +/** + * Copyright (C) 2009-2012 Typesafe Inc. + */ + package akka.actor.cell import akka.actor.{ InternalActorRef, ActorRef, ActorCell } diff --git a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala index 0b4e93c793..0b54136d8d 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala @@ -1,3 +1,7 @@ +/** + * Copyright (C) 2009-2012 Typesafe Inc. + */ + package akka.actor.cell import akka.actor.PreRestartException From 783a59fb99ea43ab4ef029d6e41c3d44f1c661ee Mon Sep 17 00:00:00 2001 From: Roland Date: Wed, 11 Jul 2012 11:23:30 +0200 Subject: [PATCH 10/14] fix imports (and some other style things) --- .../src/main/scala/akka/actor/ActorCell.scala | 31 ++++++------- .../main/scala/akka/actor/FaultHandling.scala | 2 + .../main/scala/akka/actor/cell/Children.scala | 40 +++++++++-------- .../akka/actor/cell/ChildrenContainer.scala | 21 ++------- .../scala/akka/actor/cell/DeathWatch.scala | 8 ++-- .../main/scala/akka/actor/cell/Dispatch.scala | 18 +++----- .../scala/akka/actor/cell/FaultHandling.scala | 44 +++++++++++-------- .../akka/actor/cell/ReceiveTimeout.scala | 4 +- 8 files changed, 77 insertions(+), 91 deletions(-) diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 5aac54dce1..9fb535dcb2 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -4,22 +4,17 @@ package akka.actor -import akka.dispatch._ -import scala.annotation.tailrec -import java.util.concurrent.TimeUnit -import java.util.concurrent.TimeUnit.MILLISECONDS -import akka.event.Logging.{ Debug, Warning, Error } -import akka.japi.Procedure -import java.io.{ NotSerializableException, ObjectOutputStream } -import akka.serialization.SerializationExtension -import akka.event.Logging.{ LogEventException, LogEvent } -import collection.immutable.{ TreeSet, TreeMap } -import akka.util.{ Unsafe, Duration, Helpers, NonFatal } -import java.util.concurrent.atomic.AtomicLong -import scala.collection.JavaConverters.asJavaIterableConverter -import akka.actor.cell.ChildrenContainer +import java.io.{ ObjectOutputStream, NotSerializableException } -//TODO: everything here for current compatibility - could be limited more +import scala.annotation.tailrec +import scala.collection.immutable.TreeSet + +import ActorCell.{ emptyBehaviorStack, contextStack } +import akka.actor.cell.ChildrenContainer +import akka.dispatch._ +import akka.event.Logging.{ LogEvent, Debug } +import akka.japi.Procedure +import akka.util.{ NonFatal, Duration } /** * The actor context - the view of the actor cell from the actor. @@ -95,9 +90,9 @@ trait ActorContext extends ActorRefFactory { def sender: ActorRef /** - * Returns all supervised children; this method returns a view onto the - * internal collection of children. Targeted lookups should be using - * `actorFor` instead for performance reasons: + * Returns all supervised children; this method returns a view (i.e. a lazy + * collection) onto the internal collection of children. Targeted lookups + * should be using `actorFor` instead for performance reasons: * * {{{ * val badLookup = context.children find (_.path.name == "kid") diff --git a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala index 61a28f3ccb..e13b75113c 100644 --- a/akka-actor/src/main/scala/akka/actor/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/FaultHandling.scala @@ -246,6 +246,8 @@ abstract class SupervisorStrategy { * child (and returning `true`), or it returns `false` to escalate the * failure, which will lead to this actor re-throwing the exception which * caused the failure. The exception will not be wrapped. + * + * @param children is a lazy collection (a view) */ def handleFailure(context: ActorContext, child: ActorRef, cause: Throwable, stats: ChildRestartStats, children: Iterable[ChildRestartStats]): Boolean = { val directive = if (decider.isDefinedAt(cause)) decider(cause) else Escalate //FIXME applyOrElse in Scala 2.10 diff --git a/akka-actor/src/main/scala/akka/actor/cell/Children.scala b/akka-actor/src/main/scala/akka/actor/cell/Children.scala index 07871ed6d2..a949297b1b 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/Children.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/Children.scala @@ -6,19 +6,13 @@ package akka.actor.cell import scala.annotation.tailrec import scala.collection.JavaConverters.asJavaIterableConverter -import akka.actor.{ RepointableRef, Props, InternalActorRef, ActorRef, ActorCell } -import akka.util.{ Unsafe, Helpers } -import akka.actor.InvalidActorNameException -import akka.serialization.SerializationExtension -import akka.util.NonFatal -import akka.actor.NoSerializationVerificationNeeded -import akka.actor.ActorPath -import akka.actor.ChildRestartStats -import akka.actor.ChildRestartStats -import akka.actor.ChildRestartStats -import akka.actor.ChildRestartStats -trait Children { this: ActorCell ⇒ +import akka.actor._ +import akka.actor.ActorPath.ElementRegex +import akka.serialization.SerializationExtension +import akka.util.{ Unsafe, NonFatal, Helpers } + +private[akka] trait Children { this: ActorCell ⇒ import ChildrenContainer._ @@ -77,6 +71,13 @@ trait Children { this: ActorCell ⇒ val c = childrenRefs swapChildrenRefs(c, c.add(ref)) || rec() } + /* + * This does not need to check getByRef every tailcall, because the change + * cannot happen in that direction as a race: the only entity removing a + * child is the actor itself, and the only entity which could be racing is + * somebody who calls attachChild, and there we are guaranteed that that + * child cannot yet have died (since it has not yet been created). + */ if (childrenRefs.getByRef(ref).isEmpty) rec() else false } @@ -109,9 +110,10 @@ trait Children { this: ActorCell ⇒ protected def isTerminating = childrenRefs.isTerminating protected def suspendChildren(skip: Set[ActorRef] = Set.empty): Unit = - childrenRefs.stats collect { - case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child - } foreach (_.asInstanceOf[InternalActorRef].suspend()) + childrenRefs.stats foreach { + case ChildRestartStats(child, _, _) if !(skip contains child) ⇒ child.asInstanceOf[InternalActorRef].suspend() + case _ ⇒ + } protected def resumeChildren(): Unit = childrenRefs.stats foreach (_.child.asInstanceOf[InternalActorRef].resume(inResponseToFailure = false)) @@ -125,9 +127,11 @@ trait Children { this: ActorCell ⇒ protected def removeChildAndGetStateChange(child: ActorRef): Option[SuspendReason] = { childrenRefs match { case TerminatingChildrenContainer(_, _, reason) ⇒ - val n = removeChild(child) - if (!n.isInstanceOf[TerminatingChildrenContainer]) Some(reason) else None - case _ ⇒ removeChild(child); None + val newContainer = removeChild(child) + if (!newContainer.isInstanceOf[TerminatingChildrenContainer]) Some(reason) else None + case _ ⇒ + removeChild(child) + None } } diff --git a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala index 641be552e5..b31bbbd4af 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala @@ -5,21 +5,8 @@ package akka.actor.cell import scala.collection.immutable.TreeMap -import scala.annotation.tailrec -import akka.util.Unsafe -import akka.serialization.SerializationExtension -import akka.util.NonFatal -import akka.actor.ActorPath.ElementRegex -import akka.actor.ActorCell -import akka.actor.ActorRef -import akka.actor.ChildNameReserved -import akka.actor.ChildRestartStats -import akka.actor.ChildStats -import akka.actor.InternalActorRef -import akka.actor.InvalidActorNameException -import akka.actor.NoSerializationVerificationNeeded -import akka.actor.Props -import scala.annotation.tailrec + +import akka.actor.{ InvalidActorNameException, ChildStats, ChildRestartStats, ChildNameReserved, ActorRef } /** * INTERNAL API @@ -117,7 +104,7 @@ private[akka] object ChildrenContainer { def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } - def stats: Iterable[ChildRestartStats] = c.values.collect { case c: ChildRestartStats ⇒ c } + def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } def shallDie(actor: ActorRef): ChildrenContainer = TerminatingChildrenContainer(c, Set(actor), UserRequest) @@ -178,7 +165,7 @@ private[akka] object ChildrenContainer { def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } - def stats: Iterable[ChildRestartStats] = c.values.collect { case c: ChildRestartStats ⇒ c } + def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } def shallDie(actor: ActorRef): ChildrenContainer = copy(toDie = toDie + actor) diff --git a/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala index 1af0aca67a..a9a02965a5 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/DeathWatch.scala @@ -4,14 +4,12 @@ package akka.actor.cell -import akka.actor.{ InternalActorRef, ActorRef, ActorCell } +import akka.actor.{ Terminated, InternalActorRef, ActorRef, ActorCell, Actor } import akka.dispatch.{ Watch, Unwatch } -import akka.event.Logging._ +import akka.event.Logging.{ Warning, Error, Debug } import akka.util.NonFatal -import akka.actor.Terminated -import akka.actor.Actor -trait DeathWatch { this: ActorCell ⇒ +private[akka] trait DeathWatch { this: ActorCell ⇒ private var watching: Set[ActorRef] = ActorCell.emptyActorRefSet private var watchedBy: Set[ActorRef] = ActorCell.emptyActorRefSet diff --git a/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala index 7cffefba2e..5dd9f16a6c 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala @@ -5,20 +5,12 @@ package akka.actor.cell import scala.annotation.tailrec -import akka.actor.ActorRef -import akka.dispatch.SystemMessage -import akka.util.Unsafe -import akka.dispatch.MessageDispatcher -import akka.dispatch.Suspend -import akka.dispatch.Recreate -import akka.actor.ActorCell -import akka.dispatch.Terminate -import akka.dispatch.Envelope -import akka.dispatch.Resume -import akka.dispatch.Mailbox -import akka.dispatch.Create -trait Dispatch { this: ActorCell ⇒ +import akka.actor.{ ActorRef, ActorCell } +import akka.dispatch.{ Terminate, SystemMessage, Suspend, Resume, Recreate, MessageDispatcher, Mailbox, Envelope, Create } +import akka.util.Unsafe + +private[akka] trait Dispatch { this: ActorCell ⇒ @volatile private var _mailboxDoNotCallMeDirectly: Mailbox = _ //This must be volatile since it isn't protected by the mailbox status diff --git a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala index 0b54136d8d..88fca60eb4 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala @@ -4,20 +4,14 @@ package akka.actor.cell -import akka.actor.PreRestartException -import akka.actor.ActorCell -import akka.util.NonFatal -import akka.actor.ActorRef -import akka.actor.PostRestartException -import akka.actor.Actor -import akka.dispatch.Envelope -import akka.dispatch.ChildTerminated -import akka.actor.Failed -import akka.actor.InternalActorRef -import akka.event.Logging._ -import akka.actor.ActorInterruptedException +import scala.annotation.tailrec -trait FaultHandling { this: ActorCell ⇒ +import akka.actor.{ PreRestartException, PostRestartException, InternalActorRef, Failed, ActorRef, ActorInterruptedException, ActorCell, Actor } +import akka.dispatch.{ Envelope, ChildTerminated } +import akka.event.Logging.{ Warning, Error, Debug } +import akka.util.NonFatal + +private[akka] trait FaultHandling { this: ActorCell ⇒ /* ================= * T H E R U L E S @@ -43,10 +37,13 @@ trait FaultHandling { this: ActorCell ⇒ * might well be replaced by ref to a Cancellable in the future (see #2299) */ private var _failed = false - private def currentlyFailed: Boolean = _failed + private def isFailed: Boolean = _failed private def setFailed(): Unit = _failed = true - private def setNotFailed(): Unit = _failed = false + private def clearFailed(): Unit = _failed = false + /** + * Do re-create the actor in response to a failure. + */ protected def faultRecreate(cause: Throwable): Unit = if (isNormal) { val failedActor = actor @@ -70,17 +67,27 @@ trait FaultHandling { this: ActorCell ⇒ faultResume(inResponseToFailure = false) } + /** + * Do suspend the actor in response to a failure of a parent (i.e. the + * “recursive suspend” feature). + */ protected def faultSuspend(): Unit = { // done always to keep that suspend counter balanced suspendNonRecursive() suspendChildren() } + /** + * Do resume the actor in response to a failure. + * + * @param inResponseToFailure signifies if it was our own failure which + * prompted this action. + */ protected def faultResume(inResponseToFailure: Boolean): Unit = { // done always to keep that suspend counter balanced // must happen “atomically” try resumeNonRecursive() - finally if (inResponseToFailure) setNotFailed() + finally if (inResponseToFailure) clearFailed() resumeChildren() } @@ -103,7 +110,7 @@ trait FaultHandling { this: ActorCell ⇒ final def handleInvokeFailure(t: Throwable, message: String): Unit = { publish(Error(t, self.path.toString, clazz(actor), message)) // prevent any further messages to be processed until the actor has been restarted - if (!currentlyFailed) try { + if (!isFailed) try { suspendNonRecursive() setFailed() // suspend children @@ -142,8 +149,9 @@ trait FaultHandling { this: ActorCell ⇒ private def finishRecreate(cause: Throwable, failedActor: Actor): Unit = try { try resumeNonRecursive() - finally setNotFailed() // must happen in any case, so that failure is propagated + finally clearFailed() // must happen in any case, so that failure is propagated + // need to keep a snapshot of the surviving children before the new actor instance creates new ones val survivors = children val freshActor = newActor() diff --git a/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala b/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala index 960372b488..ae4976feaa 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/ReceiveTimeout.scala @@ -10,11 +10,11 @@ import akka.actor.ActorCell.emptyCancellable import akka.actor.Cancellable import akka.util.Duration -object ReceiveTimeout { +private[akka] object ReceiveTimeout { final val emptyReceiveTimeoutData: (Duration, Cancellable) = (Duration.Undefined, ActorCell.emptyCancellable) } -trait ReceiveTimeout { this: ActorCell ⇒ +private[akka] trait ReceiveTimeout { this: ActorCell ⇒ import ReceiveTimeout._ import ActorCell._ From 8517d24c3a17309175ede644f105d2fa40cbb074 Mon Sep 17 00:00:00 2001 From: Roland Date: Fri, 13 Jul 2012 11:49:26 +0200 Subject: [PATCH 11/14] send Supervise before attaching to dispatcher, see #2301 --- .../src/main/scala/akka/actor/ActorCell.scala | 7 ++++--- .../src/main/scala/akka/actor/ActorRef.scala | 5 +---- .../main/scala/akka/actor/ActorSystem.scala | 17 ++++++++++++++--- .../akka/actor/RepointableActorRef.scala | 2 +- .../main/scala/akka/actor/cell/Children.scala | 2 ++ .../akka/actor/cell/ChildrenContainer.scala | 8 +++++--- .../main/scala/akka/actor/cell/Dispatch.scala | 7 ++++++- .../scala/akka/actor/cell/FaultHandling.scala | 19 +++++++++++++------ .../src/main/scala/akka/routing/Routing.scala | 2 +- .../test/scala/akka/testkit/AkkaSpec.scala | 5 ++++- 10 files changed, 51 insertions(+), 23 deletions(-) diff --git a/akka-actor/src/main/scala/akka/actor/ActorCell.scala b/akka-actor/src/main/scala/akka/actor/ActorCell.scala index 9fb535dcb2..bc8e8f1cbe 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorCell.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorCell.scala @@ -273,7 +273,7 @@ private[akka] class ActorCell( val system: ActorSystemImpl, val self: InternalActorRef, val props: Props, - @volatile var parent: InternalActorRef) + val parent: InternalActorRef) extends UntypedActorContext with Cell with cell.ReceiveTimeout with cell.Children @@ -290,14 +290,15 @@ private[akka] class ActorCell( protected final def lookupRoot = self final def provider = system.provider - var actor: Actor = _ + private[this] var _actor: Actor = _ + def actor: Actor = _actor + protected def actor_=(a: Actor): Unit = _actor = a var currentMessage: Envelope = _ private var behaviorStack: List[Actor.Receive] = emptyBehaviorStack /* * MESSAGE PROCESSING */ - //Memory consistency is handled by the Mailbox (reading mailbox status then processing messages, then writing mailbox status final def systemInvoke(message: SystemMessage): Unit = try { message match { diff --git a/akka-actor/src/main/scala/akka/actor/ActorRef.scala b/akka-actor/src/main/scala/akka/actor/ActorRef.scala index 6cbd821af4..00a84f956a 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorRef.scala @@ -262,10 +262,7 @@ private[akka] class LocalActorRef private[akka] ( * that is reached). */ private val actorCell: ActorCell = newActorCell(_system, this, _props, _supervisor) - actorCell.start() - - // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ - _supervisor.sendSystemMessage(akka.dispatch.Supervise(this)) + actorCell.start(sendSupervise = true) protected def newActorCell(system: ActorSystemImpl, ref: InternalActorRef, props: Props, supervisor: InternalActorRef): ActorCell = new ActorCell(system, ref, props, supervisor) diff --git a/akka-actor/src/main/scala/akka/actor/ActorSystem.scala b/akka-actor/src/main/scala/akka/actor/ActorSystem.scala index c155f9d092..a83c78e0bb 100644 --- a/akka-actor/src/main/scala/akka/actor/ActorSystem.scala +++ b/akka-actor/src/main/scala/akka/actor/ActorSystem.scala @@ -698,19 +698,30 @@ private[akka] class ActorSystemImpl(val name: String, applicationConfig: Config, node match { case wc: ActorRefWithCell ⇒ val cell = wc.underlying - indent + "-> " + node.path.name + " " + Logging.simpleName(node) + " " + + (if (indent.isEmpty) "-> " else indent.dropRight(1) + "⌊-> ") + + node.path.name + " " + Logging.simpleName(node) + " " + (cell match { case real: ActorCell ⇒ if (real.actor ne null) real.actor.getClass else "null" case _ ⇒ Logging.simpleName(cell) }) + + (cell match { + case real: ActorCell ⇒ " status=" + real.mailbox.status + case _ ⇒ "" + }) + " " + (cell.childrenRefs match { case ChildrenContainer.TerminatingChildrenContainer(_, toDie, reason) ⇒ "Terminating(" + reason + ")" + - (toDie.toSeq.sorted mkString ("\n" + indent + " toDie: ", "\n" + indent + " ", "")) + (toDie.toSeq.sorted mkString ("\n" + indent + " | toDie: ", "\n" + indent + " | ", "")) + case x @ (ChildrenContainer.TerminatedChildrenContainer | ChildrenContainer.EmptyChildrenContainer) ⇒ x.toString + case n: ChildrenContainer.NormalChildrenContainer ⇒ n.c.size + " children" case x ⇒ Logging.simpleName(x) }) + (if (cell.childrenRefs.children.isEmpty) "" else "\n") + - (cell.childrenRefs.children.toSeq.sorted map (printNode(_, indent + " |")) mkString ("\n")) + ({ + val children = cell.childrenRefs.children.toSeq.sorted + val bulk = children.dropRight(1) map (printNode(_, indent + " |")) + bulk ++ (children.lastOption map (printNode(_, indent + " "))) + } mkString ("\n")) case _ ⇒ indent + node.path.name + " " + Logging.simpleName(node) } diff --git a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala index b08c7adbee..ca1bb2492e 100644 --- a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala @@ -77,7 +77,7 @@ private[akka] class RepointableActorRef( * This is called by activate() to obtain the cell which is to replace the * unstarted cell. The cell must be fully functional. */ - def newCell(): Cell = new ActorCell(system, this, props, supervisor).start() + def newCell(): Cell = new ActorCell(system, this, props, supervisor).start(sendSupervise = false) def suspend(): Unit = underlying.suspend() diff --git a/akka-actor/src/main/scala/akka/actor/cell/Children.scala b/akka-actor/src/main/scala/akka/actor/cell/Children.scala index a949297b1b..e20a8addbd 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/Children.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/Children.scala @@ -101,6 +101,8 @@ private[akka] trait Children { this: ActorCell ⇒ } } + final protected def setTerminated(): Unit = Unsafe.instance.putObjectVolatile(this, AbstractActorCell.childrenOffset, TerminatedChildrenContainer) + /* * ActorCell-internal API */ diff --git a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala index b31bbbd4af..8b85a22ea8 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala @@ -58,13 +58,14 @@ private[akka] object ChildrenContainer { def shallDie(actor: ActorRef): ChildrenContainer = this def reserve(name: String): ChildrenContainer = new NormalChildrenContainer(emptyStats.updated(name, ChildNameReserved)) def unreserve(name: String): ChildrenContainer = this - override def toString = "no children" } /** * This is the empty container, shared among all leaf actors. */ - object EmptyChildrenContainer extends EmptyChildrenContainer + object EmptyChildrenContainer extends EmptyChildrenContainer { + override def toString = "no children" + } /** * This is the empty container which is installed after the last child has @@ -77,6 +78,7 @@ private[akka] object ChildrenContainer { throw new IllegalStateException("cannot reserve actor name '" + name + "': already terminated") override def isTerminating: Boolean = true override def isNormal: Boolean = false + override def toString = "terminated" } /** @@ -85,7 +87,7 @@ private[akka] object ChildrenContainer { * calling context.stop(child) and processing the ChildTerminated() system * message). */ - class NormalChildrenContainer(c: TreeMap[String, ChildStats]) extends ChildrenContainer { + class NormalChildrenContainer(val c: TreeMap[String, ChildStats]) extends ChildrenContainer { def add(child: ActorRef): ChildrenContainer = new NormalChildrenContainer(c.updated(child.path.name, ChildRestartStats(child))) diff --git a/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala index 5dd9f16a6c..8c849366d8 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/Dispatch.scala @@ -35,7 +35,7 @@ private[akka] trait Dispatch { this: ActorCell ⇒ final def isTerminated: Boolean = mailbox.isClosed - final def start(): this.type = { + final def start(sendSupervise: Boolean): this.type = { /* * Create the mailbox and enqueue the Create() message to ensure that @@ -47,6 +47,11 @@ private[akka] trait Dispatch { this: ActorCell ⇒ // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ mailbox.systemEnqueue(self, Create()) + if (sendSupervise) { + // ➡➡➡ NEVER SEND THE SAME SYSTEM MESSAGE OBJECT TO TWO ACTORS ⬅⬅⬅ + parent.sendSystemMessage(akka.dispatch.Supervise(self)) + } + // This call is expected to start off the actor by scheduling its mailbox. dispatcher.attach(this) diff --git a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala index 88fca60eb4..6d0c6ce283 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/FaultHandling.scala @@ -98,13 +98,20 @@ private[akka] trait FaultHandling { this: ActorCell ⇒ // stop all children, which will turn childrenRefs into TerminatingChildrenContainer (if there are children) children foreach stop + val wasTerminating = isTerminating + if (setChildrenTerminationReason(ChildrenContainer.Termination)) { - // do not process normal messages while waiting for all children to terminate - suspendNonRecursive() - // do not propagate failures during shutdown to the supervisor - setFailed() - if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) - } else finishTerminate() + if (!wasTerminating) { + // do not process normal messages while waiting for all children to terminate + suspendNonRecursive() + // do not propagate failures during shutdown to the supervisor + setFailed() + if (system.settings.DebugLifecycle) publish(Debug(self.path.toString, clazz(actor), "stopping")) + } + } else { + setTerminated() + finishTerminate() + } } final def handleInvokeFailure(t: Throwable, message: String): Unit = { diff --git a/akka-actor/src/main/scala/akka/routing/Routing.scala b/akka-actor/src/main/scala/akka/routing/Routing.scala index cb0f5ee09b..5f68694d8f 100644 --- a/akka-actor/src/main/scala/akka/routing/Routing.scala +++ b/akka-actor/src/main/scala/akka/routing/Routing.scala @@ -72,7 +72,7 @@ private[akka] class RoutedActorCell(_system: ActorSystemImpl, _ref: InternalActo if (routerConfig.resizer.isEmpty && _routees.isEmpty) throw new ActorInitializationException("router " + routerConfig + " did not register routees!") - start() + start(sendSupervise = false) /* * end of construction diff --git a/akka-testkit/src/test/scala/akka/testkit/AkkaSpec.scala b/akka-testkit/src/test/scala/akka/testkit/AkkaSpec.scala index f381e53013..4e9053722e 100644 --- a/akka-testkit/src/test/scala/akka/testkit/AkkaSpec.scala +++ b/akka-testkit/src/test/scala/akka/testkit/AkkaSpec.scala @@ -17,6 +17,7 @@ import java.util.concurrent.TimeoutException import akka.dispatch.{ Await, MessageDispatcher } import akka.dispatch.Dispatchers import akka.pattern.ask +import akka.actor.ActorSystemImpl object TimingTest extends Tag("timing") object LongRunningTest extends Tag("long-running") @@ -78,7 +79,9 @@ abstract class AkkaSpec(_system: ActorSystem) beforeShutdown() system.shutdown() try system.awaitTermination(5 seconds) catch { - case _: TimeoutException ⇒ system.log.warning("Failed to stop [{}] within 5 seconds", system.name) + case _: TimeoutException ⇒ + system.log.warning("Failed to stop [{}] within 5 seconds", system.name) + println(system.asInstanceOf[ActorSystemImpl].printTree) } atTermination() } From 7ab9fb41353f9fbbe2301d559af6ca892640697e Mon Sep 17 00:00:00 2001 From: Roland Date: Fri, 13 Jul 2012 12:11:58 +0200 Subject: [PATCH 12/14] code style fixes --- .../akka/actor/RepointableActorRef.scala | 6 +-- .../akka/actor/cell/ChildrenContainer.scala | 54 +++++++++---------- 2 files changed, 30 insertions(+), 30 deletions(-) diff --git a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala index ca1bb2492e..caad67503a 100644 --- a/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala +++ b/akka-actor/src/main/scala/akka/actor/RepointableActorRef.scala @@ -170,9 +170,9 @@ private[akka] class UnstartedCell(val systemImpl: ActorSystemImpl, val self: Rep } def system: ActorSystem = systemImpl - def suspend(): Unit = { lock.lock(); suspendCount += 1; lock.unlock() } - def resume(inResponseToFailure: Boolean): Unit = { lock.lock(); suspendCount -= 1; lock.unlock() } - def restart(cause: Throwable): Unit = { lock.lock(); suspendCount -= 1; lock.unlock() } + def suspend(): Unit = { lock.lock(); try suspendCount += 1 finally lock.unlock() } + def resume(inResponseToFailure: Boolean): Unit = { lock.lock(); try suspendCount -= 1 finally lock.unlock() } + def restart(cause: Throwable): Unit = { lock.lock(); try suspendCount -= 1 finally lock.unlock() } def stop(): Unit = sendSystemMessage(Terminate()) def isTerminated: Boolean = false def parent: InternalActorRef = supervisor diff --git a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala index 8b85a22ea8..98679862ba 100644 --- a/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala +++ b/akka-actor/src/main/scala/akka/actor/cell/ChildrenContainer.scala @@ -48,16 +48,16 @@ private[akka] object ChildrenContainer { trait EmptyChildrenContainer extends ChildrenContainer { val emptyStats = TreeMap.empty[String, ChildStats] - def add(child: ActorRef): ChildrenContainer = + override def add(child: ActorRef): ChildrenContainer = new NormalChildrenContainer(emptyStats.updated(child.path.name, ChildRestartStats(child))) - def remove(child: ActorRef): ChildrenContainer = this - def getByName(name: String): Option[ChildRestartStats] = None - def getByRef(actor: ActorRef): Option[ChildRestartStats] = None - def children: Iterable[ActorRef] = Nil - def stats: Iterable[ChildRestartStats] = Nil - def shallDie(actor: ActorRef): ChildrenContainer = this - def reserve(name: String): ChildrenContainer = new NormalChildrenContainer(emptyStats.updated(name, ChildNameReserved)) - def unreserve(name: String): ChildrenContainer = this + override def remove(child: ActorRef): ChildrenContainer = this + override def getByName(name: String): Option[ChildRestartStats] = None + override def getByRef(actor: ActorRef): Option[ChildRestartStats] = None + override def children: Iterable[ActorRef] = Nil + override def stats: Iterable[ChildRestartStats] = Nil + override def shallDie(actor: ActorRef): ChildrenContainer = this + override def reserve(name: String): ChildrenContainer = new NormalChildrenContainer(emptyStats.updated(name, ChildNameReserved)) + override def unreserve(name: String): ChildrenContainer = this } /** @@ -89,33 +89,33 @@ private[akka] object ChildrenContainer { */ class NormalChildrenContainer(val c: TreeMap[String, ChildStats]) extends ChildrenContainer { - def add(child: ActorRef): ChildrenContainer = + override def add(child: ActorRef): ChildrenContainer = new NormalChildrenContainer(c.updated(child.path.name, ChildRestartStats(child))) - def remove(child: ActorRef): ChildrenContainer = NormalChildrenContainer(c - child.path.name) + override def remove(child: ActorRef): ChildrenContainer = NormalChildrenContainer(c - child.path.name) - def getByName(name: String): Option[ChildRestartStats] = c.get(name) match { + override def getByName(name: String): Option[ChildRestartStats] = c.get(name) match { case s @ Some(_: ChildRestartStats) ⇒ s.asInstanceOf[Option[ChildRestartStats]] case _ ⇒ None } - def getByRef(actor: ActorRef): Option[ChildRestartStats] = c.get(actor.path.name) match { + override def getByRef(actor: ActorRef): Option[ChildRestartStats] = c.get(actor.path.name) match { case c @ Some(crs: ChildRestartStats) if (crs.child == actor) ⇒ c.asInstanceOf[Option[ChildRestartStats]] case _ ⇒ None } - def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } + override def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } - def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } + override def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } - def shallDie(actor: ActorRef): ChildrenContainer = TerminatingChildrenContainer(c, Set(actor), UserRequest) + override def shallDie(actor: ActorRef): ChildrenContainer = TerminatingChildrenContainer(c, Set(actor), UserRequest) - def reserve(name: String): ChildrenContainer = + override def reserve(name: String): ChildrenContainer = if (c contains name) throw new InvalidActorNameException("actor name " + name + " is not unique!") else new NormalChildrenContainer(c.updated(name, ChildNameReserved)) - def unreserve(name: String): ChildrenContainer = c.get(name) match { + override def unreserve(name: String): ChildrenContainer = c.get(name) match { case Some(ChildNameReserved) ⇒ NormalChildrenContainer(c - name) case _ ⇒ this } @@ -144,9 +144,9 @@ private[akka] object ChildrenContainer { case class TerminatingChildrenContainer(c: TreeMap[String, ChildStats], toDie: Set[ActorRef], reason: SuspendReason) extends ChildrenContainer { - def add(child: ActorRef): ChildrenContainer = copy(c.updated(child.path.name, ChildRestartStats(child))) + override def add(child: ActorRef): ChildrenContainer = copy(c.updated(child.path.name, ChildRestartStats(child))) - def remove(child: ActorRef): ChildrenContainer = { + override def remove(child: ActorRef): ChildrenContainer = { val t = toDie - child if (t.isEmpty) reason match { case Termination ⇒ TerminatedChildrenContainer @@ -155,23 +155,23 @@ private[akka] object ChildrenContainer { else copy(c - child.path.name, t) } - def getByName(name: String): Option[ChildRestartStats] = c.get(name) match { + override def getByName(name: String): Option[ChildRestartStats] = c.get(name) match { case s @ Some(_: ChildRestartStats) ⇒ s.asInstanceOf[Option[ChildRestartStats]] case _ ⇒ None } - def getByRef(actor: ActorRef): Option[ChildRestartStats] = c.get(actor.path.name) match { + override def getByRef(actor: ActorRef): Option[ChildRestartStats] = c.get(actor.path.name) match { case c @ Some(crs: ChildRestartStats) if (crs.child == actor) ⇒ c.asInstanceOf[Option[ChildRestartStats]] case _ ⇒ None } - def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } + override def children: Iterable[ActorRef] = c.values.view.collect { case ChildRestartStats(child, _, _) ⇒ child } - def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } + override def stats: Iterable[ChildRestartStats] = c.values.view.collect { case c: ChildRestartStats ⇒ c } - def shallDie(actor: ActorRef): ChildrenContainer = copy(toDie = toDie + actor) + override def shallDie(actor: ActorRef): ChildrenContainer = copy(toDie = toDie + actor) - def reserve(name: String): ChildrenContainer = reason match { + override def reserve(name: String): ChildrenContainer = reason match { case Termination ⇒ throw new IllegalStateException("cannot reserve actor name '" + name + "': terminating") case _ ⇒ if (c contains name) @@ -179,7 +179,7 @@ private[akka] object ChildrenContainer { else copy(c = c.updated(name, ChildNameReserved)) } - def unreserve(name: String): ChildrenContainer = c.get(name) match { + override def unreserve(name: String): ChildrenContainer = c.get(name) match { case Some(ChildNameReserved) ⇒ copy(c = c - name) case _ ⇒ this } From 4bbb1dbcbd5d3f60f4a0b20e19568f8b83ebd53d Mon Sep 17 00:00:00 2001 From: Roland Date: Fri, 13 Jul 2012 12:25:26 +0200 Subject: [PATCH 13/14] rename becomeOpen/becomeSuspended to resume/suspend on Mailbox --- .../scala/akka/dispatch/AbstractDispatcher.scala | 4 ++-- .../src/main/scala/akka/dispatch/Mailbox.scala | 12 ++++++------ .../scala/akka/testkit/CallingThreadDispatcher.scala | 4 ++-- 3 files changed, 10 insertions(+), 10 deletions(-) diff --git a/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala b/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala index a2f4b844dd..ffdf1987fd 100644 --- a/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala +++ b/akka-actor/src/main/scala/akka/dispatch/AbstractDispatcher.scala @@ -392,7 +392,7 @@ abstract class MessageDispatcher(val prerequisites: DispatcherPrerequisites) ext def suspend(actor: ActorCell): Unit = { val mbox = actor.mailbox if ((mbox.actor eq actor) && (mbox.dispatcher eq this)) - mbox.becomeSuspended() + mbox.suspend() } /* @@ -400,7 +400,7 @@ abstract class MessageDispatcher(val prerequisites: DispatcherPrerequisites) ext */ def resume(actor: ActorCell): Unit = { val mbox = actor.mailbox - if ((mbox.actor eq actor) && (mbox.dispatcher eq this) && mbox.becomeOpen()) + if ((mbox.actor eq actor) && (mbox.dispatcher eq this) && mbox.resume()) registerForExecution(mbox, false, false) } diff --git a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala index 2ae4616c95..a419cfd7f9 100644 --- a/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala +++ b/akka-actor/src/main/scala/akka/dispatch/Mailbox.scala @@ -124,32 +124,32 @@ private[akka] abstract class Mailbox(val messageQueue: MessageQueue) Unsafe.instance.putIntVolatile(this, AbstractMailbox.mailboxStatusOffset, newStatus) /** - * set new primary status Open. Caller does not need to worry about whether + * Reduce the suspend count by one. Caller does not need to worry about whether * status was Scheduled or not. * * @returns true if the suspend count reached zero */ @tailrec - final def becomeOpen(): Boolean = status match { + final def resume(): Boolean = status match { case Closed ⇒ setStatus(Closed); false case s ⇒ val next = if (s < suspendUnit) s else s - suspendUnit if (updateStatus(s, next)) next < suspendUnit - else becomeOpen() + else resume() } /** - * set new primary status Suspended. Caller does not need to worry about whether + * Increment the suspend count by one. Caller does not need to worry about whether * status was Scheduled or not. * * @returns true if the previous suspend count was zero */ @tailrec - final def becomeSuspended(): Boolean = status match { + final def suspend(): Boolean = status match { case Closed ⇒ setStatus(Closed); false case s ⇒ if (updateStatus(s, s + suspendUnit)) s < suspendUnit - else becomeSuspended() + else suspend() } /** diff --git a/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala b/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala index fded5e5640..07830a43b6 100644 --- a/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala +++ b/akka-testkit/src/main/scala/akka/testkit/CallingThreadDispatcher.scala @@ -151,7 +151,7 @@ class CallingThreadDispatcher( override def suspend(actor: ActorCell) { actor.mailbox match { - case m: CallingThreadMailbox ⇒ m.suspendSwitch.switchOn; m.becomeSuspended() + case m: CallingThreadMailbox ⇒ m.suspendSwitch.switchOn; m.suspend() case m ⇒ m.systemEnqueue(actor.self, Suspend()) } } @@ -163,7 +163,7 @@ class CallingThreadDispatcher( val wasActive = queue.isActive val switched = mbox.suspendSwitch.switchOff { CallingThreadDispatcherQueues(actor.system).gatherFromAllOtherQueues(mbox, queue) - mbox.becomeOpen() + mbox.resume() } if (switched && !wasActive) { runQueue(mbox, queue) From 6ca3050bb08b26716e335f89981621cc1aa0e044 Mon Sep 17 00:00:00 2001 From: Roland Date: Sun, 22 Jul 2012 16:30:40 +0200 Subject: [PATCH 14/14] small fix in stringifying ActorInitializationException --- akka-actor/src/main/scala/akka/actor/Actor.scala | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/akka-actor/src/main/scala/akka/actor/Actor.scala b/akka-actor/src/main/scala/akka/actor/Actor.scala index 1b8ee9fe85..c1ae9c57bf 100644 --- a/akka-actor/src/main/scala/akka/actor/Actor.scala +++ b/akka-actor/src/main/scala/akka/actor/Actor.scala @@ -137,7 +137,7 @@ class ActorInitializationException private[akka] (val actor: ActorRef, message: * @param msg is the message which was optionally passed into preRestart() */ class PreRestartException private[akka] (actor: ActorRef, cause: Throwable, val origCause: Throwable, val msg: Option[Any]) - extends ActorInitializationException(actor, "exception in preRestart(" + origCause.getClass + ", " + msg.getClass + ")", cause) { + extends ActorInitializationException(actor, "exception in preRestart(" + origCause.getClass + ", " + msg.map(_.getClass) + ")", cause) { } /**