move deadLetterMailbox into Mailboxes and fix review comments, see #3342
This commit is contained in:
parent
141656a054
commit
20eb28a03c
13 changed files with 72 additions and 46 deletions
|
|
@ -293,7 +293,8 @@ akka {
|
|||
}
|
||||
|
||||
default-mailbox {
|
||||
# FQCN of the MailboxType. The Class of the FQCN must have a public constructor with
|
||||
# FQCN of the MailboxType. The Class of the FQCN must have a public
|
||||
# constructor with
|
||||
# (akka.actor.ActorSystem.Settings, com.typesafe.config.Config) parameters.
|
||||
mailbox-type = "akka.dispatch.UnboundedMailbox"
|
||||
|
||||
|
|
@ -301,7 +302,7 @@ akka {
|
|||
# capacity. The provided value must be positive.
|
||||
# NOTICE:
|
||||
# Up to version 2.1 the mailbox type was determined based on this setting;
|
||||
# this is no longer the case, the type must explicitly name a bounded mailbox.
|
||||
# this is no longer the case, the type must explicitly be a bounded mailbox.
|
||||
mailbox-capacity = 1000
|
||||
|
||||
# If the mailbox is bounded then this is the timeout for enqueueing
|
||||
|
|
|
|||
|
|
@ -544,25 +544,10 @@ private[akka] class ActorSystemImpl(val name: String, applicationConfig: Config,
|
|||
|
||||
def deadLetters: ActorRef = provider.deadLetters
|
||||
|
||||
val deadLetterMailbox: Mailbox = new Mailbox(new MessageQueue {
|
||||
def enqueue(receiver: ActorRef, envelope: Envelope): Unit =
|
||||
deadLetters.tell(DeadLetter(envelope.message, envelope.sender, receiver), envelope.sender)
|
||||
def dequeue() = null
|
||||
def hasMessages = false
|
||||
def numberOfMessages = 0
|
||||
def cleanUp(owner: ActorRef, deadLetters: MessageQueue): Unit = ()
|
||||
}) {
|
||||
becomeClosed()
|
||||
def systemEnqueue(receiver: ActorRef, handle: SystemMessage): Unit =
|
||||
deadLetters ! DeadLetter(handle, receiver, receiver)
|
||||
def systemDrain(newContents: LatestFirstSystemMessageList): EarliestFirstSystemMessageList = SystemMessageList.ENil
|
||||
def hasSystemMessages = false
|
||||
}
|
||||
|
||||
val mailboxes: Mailboxes = new Mailboxes(settings, eventStream, dynamicAccess)
|
||||
val mailboxes: Mailboxes = new Mailboxes(settings, eventStream, dynamicAccess, deadLetters)
|
||||
|
||||
val dispatchers: Dispatchers = new Dispatchers(settings, DefaultDispatcherPrerequisites(
|
||||
threadFactory, eventStream, deadLetterMailbox, scheduler, dynamicAccess, settings, mailboxes))
|
||||
threadFactory, eventStream, scheduler, dynamicAccess, settings, mailboxes))
|
||||
|
||||
val dispatcher: ExecutionContext = dispatchers.defaultGlobalDispatcher
|
||||
|
||||
|
|
|
|||
|
|
@ -59,7 +59,7 @@ private[akka] trait Dispatch { this: ActorCell ⇒
|
|||
val req = system.mailboxes.getRequiredType(actorClass)
|
||||
if (req isInstance mbox.messageQueue) Create(None)
|
||||
else Create(Some(ActorInitializationException(self,
|
||||
s"Actor [$self] requires mailbox type [$req] got [${mbox.messageQueue.getClass}]")))
|
||||
s"Actor [$self] requires mailbox type [$req] got [${mbox.messageQueue.getClass.getName}]")))
|
||||
case _ ⇒ Create(None)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -87,7 +87,10 @@ abstract class MessageDispatcher(val configurator: MessageDispatcherConfigurator
|
|||
|
||||
import MessageDispatcher._
|
||||
import AbstractMessageDispatcher.{ inhabitantsOffset, shutdownScheduleOffset }
|
||||
import configurator.prerequisites._
|
||||
import configurator.prerequisites
|
||||
|
||||
val mailboxes = prerequisites.mailboxes
|
||||
val eventStream = prerequisites.eventStream
|
||||
|
||||
@volatile private[this] var _inhabitantsDoNotCallMeDirectly: Long = _ // DO NOT TOUCH!
|
||||
@volatile private[this] var _shutdownScheduleDoNotCallMeDirectly: Int = _ // DO NOT TOUCH!
|
||||
|
|
@ -168,7 +171,7 @@ abstract class MessageDispatcher(val configurator: MessageDispatcherConfigurator
|
|||
|
||||
private def scheduleShutdownAction(): Unit = {
|
||||
// IllegalStateException is thrown if scheduler has been shutdown
|
||||
try scheduler.scheduleOnce(shutdownTimeout, shutdownAction)(new ExecutionContext {
|
||||
try prerequisites.scheduler.scheduleOnce(shutdownTimeout, shutdownAction)(new ExecutionContext {
|
||||
override def execute(runnable: Runnable): Unit = runnable.run()
|
||||
override def reportFailure(t: Throwable): Unit = MessageDispatcher.this.reportFailure(t)
|
||||
}) catch {
|
||||
|
|
@ -196,7 +199,7 @@ abstract class MessageDispatcher(val configurator: MessageDispatcherConfigurator
|
|||
protected[akka] def unregister(actor: ActorCell) {
|
||||
if (debug) actors.remove(this, actor.self)
|
||||
addInhabitants(-1)
|
||||
val mailBox = actor.swapMailbox(deadLetterMailbox)
|
||||
val mailBox = actor.swapMailbox(mailboxes.deadLetterMailbox)
|
||||
mailBox.becomeClosed()
|
||||
mailBox.cleanUp()
|
||||
}
|
||||
|
|
|
|||
|
|
@ -55,7 +55,7 @@ class BalancingDispatcher(
|
|||
private class SharingMailbox(val system: ActorSystemImpl, _messageQueue: MessageQueue)
|
||||
extends Mailbox(_messageQueue) with DefaultSystemMessageQueue {
|
||||
override def cleanUp(): Unit = {
|
||||
val dlq = system.deadLetterMailbox
|
||||
val dlq = mailboxes.deadLetterMailbox
|
||||
//Don't call the original implementation of this since it scraps all messages, and we don't want to do that
|
||||
var messages = systemDrain(new LatestFirstSystemMessageList(NoMessage))
|
||||
while (messages.nonEmpty) {
|
||||
|
|
|
|||
|
|
@ -19,7 +19,6 @@ import akka.actor.Deploy
|
|||
trait DispatcherPrerequisites {
|
||||
def threadFactory: ThreadFactory
|
||||
def eventStream: EventStream
|
||||
def deadLetterMailbox: Mailbox
|
||||
def scheduler: Scheduler
|
||||
def dynamicAccess: DynamicAccess
|
||||
def settings: ActorSystem.Settings
|
||||
|
|
@ -32,7 +31,6 @@ trait DispatcherPrerequisites {
|
|||
private[akka] case class DefaultDispatcherPrerequisites(
|
||||
val threadFactory: ThreadFactory,
|
||||
val eventStream: EventStream,
|
||||
val deadLetterMailbox: Mailbox,
|
||||
val scheduler: Scheduler,
|
||||
val dynamicAccess: DynamicAccess,
|
||||
val settings: ActorSystem.Settings,
|
||||
|
|
@ -188,11 +186,14 @@ class DispatcherConfigurator(config: Config, prerequisites: DispatcherPrerequisi
|
|||
override def dispatcher(): MessageDispatcher = instance
|
||||
}
|
||||
|
||||
object BalancingDispatcherConfigurator {
|
||||
/**
|
||||
* INTERNAL API
|
||||
*/
|
||||
private[akka] object BalancingDispatcherConfigurator {
|
||||
private val defaultRequirement =
|
||||
ConfigFactory.parseString("mailbox-requirement = akka.dispatch.MultipleConsumerSemantics")
|
||||
def amendConfig(config: Config): Config =
|
||||
if (config.getString("mailbox-requirement") != "") config
|
||||
if (config.getString("mailbox-requirement") != Mailboxes.NoMailboxRequirement) config
|
||||
else defaultRequirement.withFallback(config)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -269,7 +269,7 @@ private[akka] abstract class Mailbox(val messageQueue: MessageQueue)
|
|||
* if we closed the mailbox, we must dump the remaining system messages
|
||||
* to deadLetters (this is essential for DeathWatch)
|
||||
*/
|
||||
val dlm = actor.systemImpl.deadLetterMailbox
|
||||
val dlm = actor.dispatcher.mailboxes.deadLetterMailbox
|
||||
while (messageList.nonEmpty) {
|
||||
val msg = messageList.head
|
||||
messageList = messageList.tail
|
||||
|
|
@ -295,7 +295,7 @@ private[akka] abstract class Mailbox(val messageQueue: MessageQueue)
|
|||
*/
|
||||
protected[dispatch] def cleanUp(): Unit =
|
||||
if (actor ne null) { // actor is null for the deadLetterMailbox
|
||||
val dlm = actor.systemImpl.deadLetterMailbox
|
||||
val dlm = actor.dispatcher.mailboxes.deadLetterMailbox
|
||||
var messageList = systemDrain(new LatestFirstSystemMessageList(NoMessage))
|
||||
while (messageList.nonEmpty) {
|
||||
// message must be “virgin” before being able to systemEnqueue again
|
||||
|
|
@ -306,7 +306,7 @@ private[akka] abstract class Mailbox(val messageQueue: MessageQueue)
|
|||
}
|
||||
|
||||
if (messageQueue ne null) // needed for CallingThreadDispatcher, which never calls Mailbox.run()
|
||||
messageQueue.cleanUp(actor.self, actor.systemImpl.deadLetterMailbox.messageQueue)
|
||||
messageQueue.cleanUp(actor.self, actor.dispatcher.mailboxes.deadLetterMailbox.messageQueue)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -394,7 +394,7 @@ private[akka] trait DefaultSystemMessageQueue { self: Mailbox ⇒
|
|||
if (Mailbox.debug) println(receiver + " having enqueued " + message)
|
||||
val currentList = systemQueueGet
|
||||
if (currentList.head == NoMessage) {
|
||||
if (actor ne null) actor.systemImpl.deadLetterMailbox.systemEnqueue(receiver, message)
|
||||
if (actor ne null) actor.dispatcher.mailboxes.deadLetterMailbox.systemEnqueue(receiver, message)
|
||||
} else {
|
||||
if (!systemQueuePut(currentList, message :: currentList)) {
|
||||
message.unlink()
|
||||
|
|
|
|||
|
|
@ -18,16 +18,40 @@ import akka.actor.Deploy
|
|||
import scala.util.Try
|
||||
import scala.util.Failure
|
||||
import scala.util.control.NonFatal
|
||||
import akka.dispatch.DispatcherPrerequisites
|
||||
import akka.actor.ActorRef
|
||||
import akka.actor.DeadLetter
|
||||
import akka.dispatch.sysmsg.SystemMessage
|
||||
import akka.dispatch.sysmsg.LatestFirstSystemMessageList
|
||||
import akka.dispatch.sysmsg.EarliestFirstSystemMessageList
|
||||
import akka.dispatch.sysmsg.SystemMessageList
|
||||
|
||||
object Mailboxes {
|
||||
final val DefaultMailboxId = "akka.actor.default-mailbox"
|
||||
final val NoMailboxRequirement = ""
|
||||
}
|
||||
|
||||
private[akka] class Mailboxes(
|
||||
val settings: ActorSystem.Settings,
|
||||
val eventStream: EventStream,
|
||||
dynamicAccess: DynamicAccess) {
|
||||
dynamicAccess: DynamicAccess,
|
||||
deadLetters: ActorRef) {
|
||||
|
||||
import Mailboxes._
|
||||
|
||||
val deadLetterMailbox: Mailbox = new Mailbox(new MessageQueue {
|
||||
def enqueue(receiver: ActorRef, envelope: Envelope): Unit =
|
||||
deadLetters.tell(DeadLetter(envelope.message, envelope.sender, receiver), envelope.sender)
|
||||
def dequeue() = null
|
||||
def hasMessages = false
|
||||
def numberOfMessages = 0
|
||||
def cleanUp(owner: ActorRef, deadLetters: MessageQueue): Unit = ()
|
||||
}) {
|
||||
becomeClosed()
|
||||
def systemEnqueue(receiver: ActorRef, handle: SystemMessage): Unit =
|
||||
deadLetters ! DeadLetter(handle, receiver, receiver)
|
||||
def systemDrain(newContents: LatestFirstSystemMessageList): EarliestFirstSystemMessageList = SystemMessageList.ENil
|
||||
def hasSystemMessages = false
|
||||
}
|
||||
|
||||
private val mailboxTypeConfigurators = new ConcurrentHashMap[String, MailboxType]
|
||||
|
||||
|
|
@ -72,7 +96,7 @@ private[akka] class Mailboxes(
|
|||
private var mailboxSizeWarningIssued = false
|
||||
|
||||
def getMailboxRequirement(config: Config) = config.getString("mailbox-requirement") match {
|
||||
case "" ⇒ classOf[MessageQueue]
|
||||
case NoMailboxRequirement ⇒ classOf[MessageQueue]
|
||||
case x ⇒ dynamicAccess.getClassFor[AnyRef](x).get
|
||||
}
|
||||
|
||||
|
|
@ -136,7 +160,7 @@ private[akka] class Mailboxes(
|
|||
} else if (hasMailboxRequirement) {
|
||||
verifyRequirements(lookupByQueueType(mailboxRequirement))
|
||||
} else {
|
||||
verifyRequirements(lookup(Mailboxes.DefaultMailboxId))
|
||||
verifyRequirements(lookup(DefaultMailboxId))
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -156,6 +180,7 @@ private[akka] class Mailboxes(
|
|||
case null ⇒
|
||||
// It doesn't matter if we create a mailbox type configurator that isn't used due to concurrent lookup.
|
||||
val newConfigurator = id match {
|
||||
// TODO RK remove these two for Akka 2.3
|
||||
case "unbounded" ⇒ UnboundedMailbox()
|
||||
case "bounded" ⇒ new BoundedMailbox(settings, config(id))
|
||||
case _ ⇒
|
||||
|
|
@ -184,7 +209,7 @@ private[akka] class Mailboxes(
|
|||
}
|
||||
}
|
||||
|
||||
private val defaultMailboxConfig = settings.config.getConfig(Mailboxes.DefaultMailboxId)
|
||||
private val defaultMailboxConfig = settings.config.getConfig(DefaultMailboxId)
|
||||
|
||||
//INTERNAL API
|
||||
private def config(id: String): Config = {
|
||||
|
|
|
|||
|
|
@ -79,8 +79,8 @@ dispatcher which will execute it. Then the mailbox is determined as follows:
|
|||
|
||||
6. The default mailbox ``akka.actor.default-mailbox`` will be used.
|
||||
|
||||
Which Configuration is pass to the Mailbox Type
|
||||
-----------------------------------------------
|
||||
Which Configuration is passed to the Mailbox Type
|
||||
-------------------------------------------------
|
||||
|
||||
Each mailbox type is implemented by a class which extends :class:`MailboxType`
|
||||
and takes two constructor arguments: a :class:`ActorSystem.Settings` object and
|
||||
|
|
|
|||
|
|
@ -85,12 +85,12 @@ Search Replace with
|
|||
If you need to convert from Java to ``scala.collection.immutable.Seq`` or ``scala.collection.immutable.Iterable`` you should use ``akka.japi.Util.immutableSeq(…)``,
|
||||
and if you need to convert from Scala you can simply switch to using immutable collections yourself or use the ``to[immutable.<collection-type>]`` method.
|
||||
|
||||
ActorContext & ActorRefFactory dispatcher
|
||||
ActorContext & ActorRefFactory Dispatcher
|
||||
=========================================
|
||||
|
||||
The return type of ``ActorContext``'s and ``ActorRefFactory``'s ``dispatcher``-method now returns ``ExecutionContext`` instead of ``MessageDispatcher``.
|
||||
|
||||
Removed fallback to default dispatcher
|
||||
Removed Fallback to Default Dispatcher
|
||||
======================================
|
||||
|
||||
If deploying an actor with a specific dispatcher, e.g.
|
||||
|
|
@ -106,6 +106,17 @@ Akka 2.2 introduces the possibility to add dispatcher configuration to the
|
|||
The fallback was removed because in many cases its application was neither
|
||||
intended nor noticed.
|
||||
|
||||
Changed Configuration Section for Dispatcher & Mailbox
|
||||
======================================================
|
||||
|
||||
The mailbox configuration defaults moved from ``akka.actor.default-dispatcher``
|
||||
to ``akka.actor.default-mailbox``. You will not have to change anything unless
|
||||
your configuration overrides a setting in the default dispatcher section.
|
||||
|
||||
The ``mailbox-type`` now requires a fully-qualified class name for the mailbox
|
||||
to use. The special words ``bounded`` and ``unbounded`` are retained for a
|
||||
migration period throughout the 2.2 series.
|
||||
|
||||
API changes to FSM and TestFSMRef
|
||||
=================================
|
||||
|
||||
|
|
|
|||
|
|
@ -79,8 +79,8 @@ dispatcher which will execute it. Then the mailbox is determined as follows:
|
|||
|
||||
6. The default mailbox ``akka.actor.default-mailbox`` will be used.
|
||||
|
||||
Which Configuration is pass to the Mailbox Type
|
||||
-----------------------------------------------
|
||||
Which Configuration is passed to the Mailbox Type
|
||||
-------------------------------------------------
|
||||
|
||||
Each mailbox type is implemented by a class which extends :class:`MailboxType`
|
||||
and takes two constructor arguments: a :class:`ActorSystem.Settings` object and
|
||||
|
|
|
|||
|
|
@ -340,7 +340,7 @@ class CallingThreadMailbox(_receiver: akka.actor.Cell, val mailboxType: MailboxT
|
|||
val qq = queue
|
||||
CallingThreadDispatcherQueues(actor.system).gatherFromAllOtherQueues(this, qq)
|
||||
super.cleanUp()
|
||||
qq.cleanUp(actor.self, actor.systemImpl.deadLetterMailbox.messageQueue)
|
||||
qq.cleanUp(actor.self, actor.dispatcher.mailboxes.deadLetterMailbox.messageQueue)
|
||||
q.remove()
|
||||
}
|
||||
}
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue