Refactored mailbox configuration
This commit is contained in:
parent
e90d5b1b69
commit
9ea09c3e36
15 changed files with 416 additions and 329 deletions
|
|
@ -5,11 +5,13 @@
|
|||
package se.scalablesolutions.akka.dispatch
|
||||
|
||||
import se.scalablesolutions.akka.actor.{Actor, ActorRef}
|
||||
import se.scalablesolutions.akka.config.Config.config
|
||||
import se.scalablesolutions.akka.config.Config._
|
||||
import se.scalablesolutions.akka.util.{Duration, Logging, UUID}
|
||||
|
||||
import net.lag.configgy.ConfigMap
|
||||
|
||||
import java.util.concurrent.ThreadPoolExecutor.{AbortPolicy, CallerRunsPolicy, DiscardOldestPolicy, DiscardPolicy}
|
||||
import java.util.concurrent.TimeUnit
|
||||
import se.scalablesolutions.akka.util.{Duration, Logging, UUID}
|
||||
|
||||
/**
|
||||
* Scala API. Dispatcher factory.
|
||||
|
|
@ -45,13 +47,11 @@ import se.scalablesolutions.akka.util.{Duration, Logging, UUID}
|
|||
*/
|
||||
object Dispatchers extends Logging {
|
||||
val THROUGHPUT = config.getInt("akka.actor.throughput", 5)
|
||||
val THROUGHPUT_DEADLINE_MS = config.getInt("akka.actor.throughput-deadline-ms",-1)
|
||||
val MAILBOX_CAPACITY = config.getInt("akka.actor.default-dispatcher.mailbox-capacity", -1)
|
||||
val MAILBOX_CONFIG = MailboxConfig(
|
||||
capacity = Dispatchers.MAILBOX_CAPACITY,
|
||||
pushTimeOut = config.getInt("akka.actor.default-dispatcher.mailbox-push-timeout-ms").map(Duration(_,TimeUnit.MILLISECONDS)),
|
||||
blockingDequeue = false
|
||||
)
|
||||
val MAILBOX_PUSH_TIME_OUT = Duration(config.getInt("akka.actor.default-dispatcher.mailbox-push-timeout-time", 10), TIME_UNIT)
|
||||
val THROUGHPUT_DEADLINE_TIME = Duration(config.getInt("akka.actor.throughput-deadline-time",-1), TIME_UNIT)
|
||||
val THROUGHPUT_DEADLINE_TIME_MILLIS = THROUGHPUT_DEADLINE_TIME.toMillis.toInt
|
||||
val MAILBOX_TYPE = if (MAILBOX_CAPACITY < 0) UnboundedMailbox() else BoundedMailbox()
|
||||
|
||||
lazy val defaultGlobalDispatcher = {
|
||||
config.getConfigMap("akka.actor.default-dispatcher").flatMap(from).getOrElse(globalExecutorBasedEventDrivenDispatcher)
|
||||
|
|
@ -59,7 +59,8 @@ object Dispatchers extends Logging {
|
|||
|
||||
object globalHawtDispatcher extends HawtDispatcher
|
||||
|
||||
object globalExecutorBasedEventDrivenDispatcher extends ExecutorBasedEventDrivenDispatcher("global",THROUGHPUT,THROUGHPUT_DEADLINE_MS,MAILBOX_CONFIG) {
|
||||
object globalExecutorBasedEventDrivenDispatcher extends ExecutorBasedEventDrivenDispatcher(
|
||||
"global", THROUGHPUT, THROUGHPUT_DEADLINE_TIME_MILLIS, MAILBOX_TYPE) {
|
||||
override def register(actor: ActorRef) = {
|
||||
if (isShutdown) init
|
||||
super.register(actor)
|
||||
|
|
@ -81,7 +82,7 @@ object Dispatchers extends Logging {
|
|||
* <p/>
|
||||
* E.g. each actor consumes its own thread.
|
||||
*/
|
||||
def newThreadBasedDispatcher(actor: ActorRef) = new ThreadBasedDispatcher(actor)
|
||||
def newThreadBasedDispatcher(actor: ActorRef) = new ThreadBasedDispatcher(actor, BoundedMailbox(true))
|
||||
|
||||
/**
|
||||
* Creates an thread based dispatcher serving a single actor through the same single thread.
|
||||
|
|
@ -96,36 +97,32 @@ object Dispatchers extends Logging {
|
|||
* <p/>
|
||||
* E.g. each actor consumes its own thread.
|
||||
*/
|
||||
def newThreadBasedDispatcher(actor: ActorRef, mailboxCapacity: Int, pushTimeOut: Duration) = new ThreadBasedDispatcher(actor, MailboxConfig(mailboxCapacity,Option(pushTimeOut),true))
|
||||
def newThreadBasedDispatcher(actor: ActorRef, mailboxCapacity: Int, pushTimeOut: Duration) =
|
||||
new ThreadBasedDispatcher(actor, mailboxCapacity, pushTimeOut)
|
||||
|
||||
/**
|
||||
* Creates a executor-based event-driven dispatcher serving multiple (millions) of actors through a thread pool.
|
||||
* <p/>
|
||||
* Has a fluent builder interface for configuring its semantics.
|
||||
*/
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String) = new ExecutorBasedEventDrivenDispatcher(name, THROUGHPUT)
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String) = new ExecutorBasedEventDrivenDispatcher(name)
|
||||
|
||||
/**
|
||||
* Creates a executor-based event-driven dispatcher serving multiple (millions) of actors through a thread pool.
|
||||
* <p/>
|
||||
* Has a fluent builder interface for configuring its semantics.
|
||||
*/
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String, throughput: Int) = new ExecutorBasedEventDrivenDispatcher(name, throughput)
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String, throughput: Int, mailboxType: MailboxType) =
|
||||
new ExecutorBasedEventDrivenDispatcher(name, throughput, mailboxType)
|
||||
|
||||
|
||||
/**
|
||||
* Creates a executor-based event-driven dispatcher serving multiple (millions) of actors through a thread pool.
|
||||
* <p/>
|
||||
* Has a fluent builder interface for configuring its semantics.
|
||||
*/
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String, throughput: Int, throughputDeadlineMs: Int, mailboxCapacity: Int) = new ExecutorBasedEventDrivenDispatcher(name, throughput, throughputDeadlineMs, mailboxCapacity)
|
||||
|
||||
/**
|
||||
* Creates a executor-based event-driven dispatcher serving multiple (millions) of actors through a thread pool.
|
||||
* <p/>
|
||||
* Has a fluent builder interface for configuring its semantics.
|
||||
*/
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String, throughput: Int, throughputDeadlineMs: Int, mailboxCapacity: Int, pushTimeOut: Duration) = new ExecutorBasedEventDrivenDispatcher(name, throughput, throughputDeadlineMs, MailboxConfig(mailboxCapacity,Some(pushTimeOut),false))
|
||||
|
||||
def newExecutorBasedEventDrivenDispatcher(name: String, throughput: Int, throughputDeadlineMs: Int, mailboxType: MailboxType) =
|
||||
new ExecutorBasedEventDrivenDispatcher(name, throughput, throughputDeadlineMs, mailboxType)
|
||||
|
||||
/**
|
||||
* Creates a executor-based event-driven dispatcher with work stealing (TODO: better doc) serving multiple (millions) of actors through a thread pool.
|
||||
|
|
@ -139,7 +136,8 @@ object Dispatchers extends Logging {
|
|||
* <p/>
|
||||
* Has a fluent builder interface for configuring its semantics.
|
||||
*/
|
||||
def newExecutorBasedEventDrivenWorkStealingDispatcher(name: String, mailboxCapacity: Int) = new ExecutorBasedEventDrivenWorkStealingDispatcher(name, mailboxCapacity)
|
||||
def newExecutorBasedEventDrivenWorkStealingDispatcher(name: String, mailboxType: MailboxType) =
|
||||
new ExecutorBasedEventDrivenWorkStealingDispatcher(name, mailboxType = mailboxType)
|
||||
|
||||
/**
|
||||
* Utility function that tries to load the specified dispatcher config from the akka.conf
|
||||
|
|
@ -155,7 +153,7 @@ object Dispatchers extends Logging {
|
|||
* type = "GlobalExecutorBasedEventDriven" # Must be one of the following, all "Global*" are non-configurable
|
||||
* # (ExecutorBasedEventDrivenWorkStealing), ExecutorBasedEventDriven,
|
||||
* # Hawt, GlobalExecutorBasedEventDriven, GlobalHawt
|
||||
* keep-alive-ms = 60000 # Keep alive time for threads
|
||||
* keep-alive-time = 60 # Keep alive time for threads
|
||||
* core-pool-size-factor = 1.0 # No of core threads ... ceil(available processors * factor)
|
||||
* max-pool-size-factor = 4.0 # Max no of threads ... ceil(available processors * factor)
|
||||
* executor-bounds = -1 # Makes the Executor bounded, -1 is unbounded
|
||||
|
|
@ -175,7 +173,7 @@ object Dispatchers extends Logging {
|
|||
|
||||
def threadPoolConfig(b: ThreadPoolBuilder) {
|
||||
b.configureIfPossible( builder => {
|
||||
cfg.getInt("keep-alive-ms").foreach(builder.setKeepAliveTimeInMillis(_))
|
||||
cfg.getInt("keep-alive-time").foreach(time => builder.setKeepAliveTimeInMillis(Duration(time, TIME_UNIT).toMillis.toInt))
|
||||
cfg.getDouble("core-pool-size-factor").foreach(builder.setCorePoolSizeFromFactor(_))
|
||||
cfg.getDouble("max-pool-size-factor").foreach(builder.setMaxPoolSizeFromFactor(_))
|
||||
cfg.getInt("executor-bounds").foreach(builder.setExecutorBounds(_))
|
||||
|
|
@ -192,37 +190,27 @@ object Dispatchers extends Logging {
|
|||
})
|
||||
}
|
||||
|
||||
lazy val mailboxBounds: MailboxConfig = {
|
||||
val capacity = cfg.getInt("mailbox-capacity",Dispatchers.MAILBOX_CAPACITY)
|
||||
val timeout = cfg.getInt("mailbox-push-timeout-ms").map(Duration(_,TimeUnit.MILLISECONDS))
|
||||
MailboxConfig(capacity,timeout,false)
|
||||
lazy val mailboxType: MailboxType = {
|
||||
val capacity = cfg.getInt("mailbox-capacity", MAILBOX_CAPACITY)
|
||||
// FIXME how do we read in isBlocking for mailbox? Now set to 'false'.
|
||||
if (capacity < 0) UnboundedMailbox()
|
||||
else BoundedMailbox(false, capacity, Duration(cfg.getInt("mailbox-push-timeout", MAILBOX_PUSH_TIME_OUT.toMillis.toInt), TIME_UNIT))
|
||||
}
|
||||
|
||||
val dispatcher: Option[MessageDispatcher] = cfg.getString("type") map {
|
||||
case "ExecutorBasedEventDrivenWorkStealing" =>
|
||||
new ExecutorBasedEventDrivenWorkStealingDispatcher(name,MAILBOX_CAPACITY,threadPoolConfig)
|
||||
|
||||
cfg.getString("type") map {
|
||||
case "ExecutorBasedEventDriven" =>
|
||||
new ExecutorBasedEventDrivenDispatcher(
|
||||
name,
|
||||
cfg.getInt("throughput", THROUGHPUT),
|
||||
cfg.getInt("throughput-deadline-ms",THROUGHPUT_DEADLINE_MS),
|
||||
mailboxBounds,
|
||||
cfg.getInt("throughput-deadline", THROUGHPUT_DEADLINE_TIME_MILLIS),
|
||||
mailboxType,
|
||||
threadPoolConfig)
|
||||
|
||||
case "Hawt" =>
|
||||
new HawtDispatcher(cfg.getBool("aggregate").getOrElse(true))
|
||||
|
||||
case "GlobalExecutorBasedEventDriven" =>
|
||||
globalExecutorBasedEventDrivenDispatcher
|
||||
|
||||
case "GlobalHawt" =>
|
||||
globalHawtDispatcher
|
||||
|
||||
case unknown =>
|
||||
throw new IllegalArgumentException("Unknown dispatcher type [%s]" format unknown)
|
||||
}
|
||||
|
||||
dispatcher
|
||||
case "ExecutorBasedEventDrivenWorkStealing" => new ExecutorBasedEventDrivenWorkStealingDispatcher(name, mailboxType, threadPoolConfig)
|
||||
case "Hawt" => new HawtDispatcher(cfg.getBool("aggregate").getOrElse(true))
|
||||
case "GlobalExecutorBasedEventDriven" => globalExecutorBasedEventDrivenDispatcher
|
||||
case "GlobalHawt" => globalHawtDispatcher
|
||||
case unknown => throw new IllegalArgumentException("Unknown dispatcher type [%s]" format unknown)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -65,18 +65,26 @@ import java.util.concurrent.{RejectedExecutionException, ConcurrentLinkedQueue,
|
|||
class ExecutorBasedEventDrivenDispatcher(
|
||||
_name: String,
|
||||
val throughput: Int = Dispatchers.THROUGHPUT,
|
||||
val throughputDeadlineMs: Int = Dispatchers.THROUGHPUT_DEADLINE_MS,
|
||||
mailboxConfig: MailboxConfig = Dispatchers.MAILBOX_CONFIG,
|
||||
config: (ThreadPoolBuilder) => Unit = _ => ()) extends MessageDispatcher with ThreadPoolBuilder {
|
||||
val throughputDeadlineTime: Int = Dispatchers.THROUGHPUT_DEADLINE_TIME_MILLIS,
|
||||
_mailboxType: MailboxType = Dispatchers.MAILBOX_TYPE,
|
||||
config: (ThreadPoolBuilder) => Unit = _ => ())
|
||||
extends MessageDispatcher with ThreadPoolBuilder {
|
||||
|
||||
def this(_name: String, throughput: Int, throughputDeadlineMs: Int, capacity: Int) = this(_name,throughput,throughputDeadlineMs,MailboxConfig(capacity,None,false))
|
||||
def this(_name: String, throughput: Int) = this(_name, throughput, Dispatchers.THROUGHPUT_DEADLINE_MS, Dispatchers.MAILBOX_CAPACITY) // Needed for Java API usage
|
||||
def this(_name: String) = this(_name,Dispatchers.THROUGHPUT, Dispatchers.THROUGHPUT_DEADLINE_MS,Dispatchers.MAILBOX_CAPACITY) // Needed for Java API usage
|
||||
def this(_name: String, throughput: Int, throughputDeadlineTime: Int, mailboxType: MailboxType) =
|
||||
this(_name, throughput, throughputDeadlineTime, mailboxType, _ => ()) // Needed for Java API usage
|
||||
|
||||
//FIXME remove this from ThreadPoolBuilder
|
||||
mailboxCapacity = mailboxConfig.capacity
|
||||
def this(_name: String, throughput: Int, mailboxType: MailboxType) =
|
||||
this(_name, throughput, Dispatchers.THROUGHPUT_DEADLINE_TIME_MILLIS, mailboxType) // Needed for Java API usage
|
||||
|
||||
@volatile private var active: Boolean = false
|
||||
def this(_name: String, throughput: Int) =
|
||||
this(_name, throughput, Dispatchers.THROUGHPUT_DEADLINE_TIME_MILLIS, Dispatchers.MAILBOX_TYPE) // Needed for Java API usage
|
||||
|
||||
def this(_name: String) =
|
||||
this(_name, Dispatchers.THROUGHPUT, Dispatchers.THROUGHPUT_DEADLINE_TIME_MILLIS, Dispatchers.MAILBOX_TYPE) // Needed for Java API usage
|
||||
|
||||
val mailboxType = Some(_mailboxType)
|
||||
|
||||
@volatile private var active = false
|
||||
|
||||
val name = "akka:event-driven:dispatcher:" + _name
|
||||
init
|
||||
|
|
@ -86,15 +94,12 @@ class ExecutorBasedEventDrivenDispatcher(
|
|||
*/
|
||||
trait ExecutableMailbox extends Runnable { self: MessageQueue =>
|
||||
final def run = {
|
||||
|
||||
val reschedule = try {
|
||||
processMailbox()
|
||||
} finally {
|
||||
dispatcherLock.unlock()
|
||||
}
|
||||
|
||||
if (reschedule || !self.isEmpty)
|
||||
registerForExecution(self)
|
||||
if (reschedule || !self.isEmpty) registerForExecution(self)
|
||||
}
|
||||
|
||||
/**
|
||||
|
|
@ -107,23 +112,19 @@ class ExecutorBasedEventDrivenDispatcher(
|
|||
if (nextMessage ne null) {
|
||||
val throttle = throughput > 0
|
||||
var processedMessages = 0
|
||||
val isDeadlineEnabled = throttle && throughputDeadlineMs > 0
|
||||
val isDeadlineEnabled = throttle && throughputDeadlineTime > 0
|
||||
val started = if (isDeadlineEnabled) System.currentTimeMillis else 0
|
||||
|
||||
do {
|
||||
nextMessage.invoke
|
||||
|
||||
if (throttle) { // Will be elided when false
|
||||
processedMessages += 1
|
||||
if ((processedMessages >= throughput)
|
||||
|| (isDeadlineEnabled && (System.currentTimeMillis - started) >= throughputDeadlineMs)) //If we're throttled, break out
|
||||
if ((processedMessages >= throughput) ||
|
||||
(isDeadlineEnabled && (System.currentTimeMillis - started) >= throughputDeadlineTime)) // If we're throttled, break out
|
||||
return !self.isEmpty
|
||||
}
|
||||
nextMessage = self.dequeue
|
||||
} while (nextMessage ne null)
|
||||
}
|
||||
while (nextMessage ne null)
|
||||
}
|
||||
|
||||
false
|
||||
}
|
||||
}
|
||||
|
|
@ -144,9 +145,7 @@ class ExecutorBasedEventDrivenDispatcher(
|
|||
throw e
|
||||
}
|
||||
}
|
||||
} else {
|
||||
log.warning("%s is shut down,\n\tignoring the rest of the messages in the mailbox of\n\t%s", toString, mailbox)
|
||||
}
|
||||
} else log.warning("%s is shut down,\n\tignoring the rest of the messages in the mailbox of\n\t%s", toString, mailbox)
|
||||
|
||||
/**
|
||||
* @return the mailbox associated with the actor
|
||||
|
|
@ -155,14 +154,14 @@ class ExecutorBasedEventDrivenDispatcher(
|
|||
|
||||
override def mailboxSize(actorRef: ActorRef) = getMailbox(actorRef).size
|
||||
|
||||
override def createMailbox(actorRef: ActorRef): AnyRef = {
|
||||
if (mailboxCapacity > 0)
|
||||
new DefaultBoundedMessageQueue(mailboxCapacity,mailboxConfig.pushTimeOut,blockDequeue = false) with ExecutableMailbox
|
||||
else
|
||||
new DefaultUnboundedMessageQueue(blockDequeue = false) with ExecutableMailbox
|
||||
def createTransientMailbox(actorRef: ActorRef, mailboxType: TransientMailboxType): AnyRef = mailboxType match {
|
||||
case UnboundedMailbox(blocking) =>
|
||||
new DefaultUnboundedMessageQueue(blocking) with ExecutableMailbox
|
||||
case BoundedMailbox(blocking, capacity, pushTimeOut) =>
|
||||
val cap = if (mailboxCapacity == -1) capacity else mailboxCapacity
|
||||
new DefaultBoundedMessageQueue(cap, pushTimeOut, blocking) with ExecutableMailbox
|
||||
}
|
||||
|
||||
|
||||
def start = if (!active) {
|
||||
log.debug("Starting up %s\n\twith throughput [%d]", toString, throughput)
|
||||
active = true
|
||||
|
|
|
|||
|
|
@ -31,12 +31,14 @@ import se.scalablesolutions.akka.actor.{Actor, ActorRef, IllegalActorStateExcept
|
|||
*/
|
||||
class ExecutorBasedEventDrivenWorkStealingDispatcher(
|
||||
_name: String,
|
||||
capacity: Int = Dispatchers.MAILBOX_CAPACITY,
|
||||
_mailboxType: MailboxType = Dispatchers.MAILBOX_TYPE,
|
||||
config: (ThreadPoolBuilder) => Unit = _ => ()) extends MessageDispatcher with ThreadPoolBuilder {
|
||||
|
||||
def this(_name: String, capacity: Int) = this(_name,capacity, _ => ())
|
||||
def this(_name: String, mailboxType: MailboxType) = this(_name, mailboxType, _ => ())
|
||||
|
||||
mailboxCapacity = capacity
|
||||
def this(_name: String) = this(_name, Dispatchers.MAILBOX_TYPE, _ => ())
|
||||
|
||||
val mailboxType = Some(_mailboxType)
|
||||
|
||||
@volatile private var active: Boolean = false
|
||||
|
||||
|
|
@ -182,36 +184,33 @@ class ExecutorBasedEventDrivenWorkStealingDispatcher(
|
|||
buildThreadPool
|
||||
}
|
||||
|
||||
protected override def createMailbox(actorRef: ActorRef): AnyRef = {
|
||||
if (mailboxCapacity <= 0) {
|
||||
def createTransientMailbox(actorRef: ActorRef, mailboxType: TransientMailboxType): AnyRef = mailboxType match {
|
||||
case UnboundedMailbox(blocking) => // FIXME make use of 'blocking' in work stealer ConcurrentLinkedDeque
|
||||
new ConcurrentLinkedDeque[MessageInvocation] with MessageQueue with Runnable {
|
||||
def enqueue(handle: MessageInvocation): Unit = this.add(handle)
|
||||
|
||||
def dequeue: MessageInvocation = this.poll()
|
||||
|
||||
def run = {
|
||||
if (!tryProcessMailbox(this)) {
|
||||
def run = if (!tryProcessMailbox(this)) {
|
||||
// we are not able to process our mailbox (another thread is busy with it), so lets donate some of our mailbox
|
||||
// to another actor and then process his mailbox in stead.
|
||||
findThief(actorRef).foreach( tryDonateAndProcessMessages(actorRef,_) )
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
else {
|
||||
new LinkedBlockingDeque[MessageInvocation](mailboxCapacity) with MessageQueue with Runnable {
|
||||
case BoundedMailbox(blocking, capacity, pushTimeOut) =>
|
||||
val cap = if (mailboxCapacity == -1) capacity else mailboxCapacity
|
||||
new LinkedBlockingDeque[MessageInvocation](cap) with MessageQueue with Runnable {
|
||||
def enqueue(handle: MessageInvocation): Unit = this.add(handle)
|
||||
|
||||
def dequeue: MessageInvocation = this.poll()
|
||||
|
||||
def run = {
|
||||
if (!tryProcessMailbox(this)) {
|
||||
def run = if (!tryProcessMailbox(this)) {
|
||||
// we are not able to process our mailbox (another thread is busy with it), so lets donate some of our mailbox
|
||||
// to another actor and then process his mailbox in stead.
|
||||
findThief(actorRef).foreach( tryDonateAndProcessMessages(actorRef, _) )
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
override def register(actorRef: ActorRef) = {
|
||||
verifyActorsAreOfSameType(actorRef)
|
||||
|
|
|
|||
|
|
@ -15,16 +15,15 @@ import java.util.concurrent.atomic.{AtomicInteger, AtomicBoolean}
|
|||
import java.util.concurrent.CountDownLatch
|
||||
|
||||
/**
|
||||
* Holds helper methods for working with actors that are using
|
||||
* a HawtDispatcher as it's dispatcher.
|
||||
* Holds helper methods for working with actors that are using a HawtDispatcher as it's dispatcher.
|
||||
*/
|
||||
object HawtDispatcher {
|
||||
|
||||
private val retained = new AtomicInteger()
|
||||
|
||||
@volatile private var shutdownLatch: CountDownLatch = _
|
||||
|
||||
private def retainNonDaemon = {
|
||||
if( retained.getAndIncrement == 0 ) {
|
||||
private def retainNonDaemon = if (retained.getAndIncrement == 0) {
|
||||
shutdownLatch = new CountDownLatch(1)
|
||||
new Thread("HawtDispatch Non-Daemon") {
|
||||
override def run = {
|
||||
|
|
@ -36,28 +35,21 @@ object HawtDispatcher {
|
|||
}
|
||||
}.start()
|
||||
}
|
||||
}
|
||||
|
||||
private def releaseNonDaemon = {
|
||||
if( retained.decrementAndGet == 0 ) {
|
||||
private def releaseNonDaemon = if (retained.decrementAndGet == 0) {
|
||||
shutdownLatch.countDown
|
||||
shutdownLatch = null
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* @return the mailbox associated with the actor
|
||||
*/
|
||||
private def mailbox(actorRef: ActorRef) = {
|
||||
actorRef.mailbox.asInstanceOf[HawtDispatcherMailbox]
|
||||
}
|
||||
private def mailbox(actorRef: ActorRef) = actorRef.mailbox.asInstanceOf[HawtDispatcherMailbox]
|
||||
|
||||
/**
|
||||
* @return the dispatch queue associated with the actor
|
||||
*/
|
||||
def queue(actorRef: ActorRef) = {
|
||||
mailbox(actorRef).queue
|
||||
}
|
||||
def queue(actorRef: ActorRef) = mailbox(actorRef).queue
|
||||
|
||||
/**
|
||||
* <p>
|
||||
|
|
@ -71,14 +63,12 @@ object HawtDispatcher {
|
|||
*
|
||||
* @return true if the actor was pinned
|
||||
*/
|
||||
def pin(actorRef: ActorRef) = {
|
||||
actorRef.mailbox match {
|
||||
def pin(actorRef: ActorRef) = actorRef.mailbox match {
|
||||
case x: HawtDispatcherMailbox =>
|
||||
x.queue.setTargetQueue( getRandomThreadQueue )
|
||||
true
|
||||
case _ => false
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* <p>
|
||||
|
|
@ -91,20 +81,15 @@ object HawtDispatcher {
|
|||
* </p>
|
||||
* @return true if the actor was unpinned
|
||||
*/
|
||||
def unpin(actorRef: ActorRef) = {
|
||||
target(actorRef, globalQueue)
|
||||
}
|
||||
def unpin(actorRef: ActorRef) = target(actorRef, globalQueue)
|
||||
|
||||
/**
|
||||
* @return true if the actor was pinned to a thread.
|
||||
*/
|
||||
def pinned(actorRef: ActorRef):Boolean = {
|
||||
actorRef.mailbox match {
|
||||
case x:HawtDispatcherMailbox=>
|
||||
x.queue.getTargetQueue.getQueueType == QueueType.THREAD_QUEUE
|
||||
def pinned(actorRef: ActorRef):Boolean = actorRef.mailbox match {
|
||||
case x: HawtDispatcherMailbox => x.queue.getTargetQueue.getQueueType == QueueType.THREAD_QUEUE
|
||||
case _ => false
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* <p>
|
||||
|
|
@ -117,8 +102,7 @@ object HawtDispatcher {
|
|||
* </p>
|
||||
* @return true if the actor was unpinned
|
||||
*/
|
||||
def target(actorRef: ActorRef, parent:DispatchQueue) = {
|
||||
actorRef.mailbox match {
|
||||
def target(actorRef: ActorRef, parent: DispatchQueue) = actorRef.mailbox match {
|
||||
case x: HawtDispatcherMailbox =>
|
||||
x.queue.setTargetQueue(parent)
|
||||
true
|
||||
|
|
@ -126,8 +110,6 @@ object HawtDispatcher {
|
|||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* <p>
|
||||
* A HawtDispatch based MessageDispatcher. Actors with this dispatcher are executed
|
||||
|
|
@ -158,19 +140,14 @@ object HawtDispatcher {
|
|||
*/
|
||||
class HawtDispatcher(val aggregate: Boolean = true, val parent: DispatchQueue = globalQueue) extends MessageDispatcher {
|
||||
import HawtDispatcher._
|
||||
|
||||
private val active = new AtomicBoolean(false)
|
||||
|
||||
def start = {
|
||||
if( active.compareAndSet(false, true) ) {
|
||||
retainNonDaemon
|
||||
}
|
||||
}
|
||||
val mailboxType: Option[MailboxType] = None
|
||||
|
||||
def shutdown = {
|
||||
if( active.compareAndSet(true, false) ) {
|
||||
releaseNonDaemon
|
||||
}
|
||||
}
|
||||
def start = if (active.compareAndSet(false, true)) retainNonDaemon
|
||||
|
||||
def shutdown = if (active.compareAndSet(true, false)) releaseNonDaemon
|
||||
|
||||
def isShutdown = !active.get
|
||||
|
||||
|
|
@ -191,11 +168,13 @@ class HawtDispatcher(val aggregate:Boolean=true, val parent:DispatchQueue=global
|
|||
else new HawtDispatcherMailbox(queue)
|
||||
}
|
||||
|
||||
def createTransientMailbox(actorRef: ActorRef, mailboxType: TransientMailboxType): AnyRef = null.asInstanceOf[AnyRef]
|
||||
|
||||
override def toString = "HawtDispatchEventDrivenDispatcher"
|
||||
}
|
||||
|
||||
class HawtDispatcherMailbox(val queue: DispatchQueue) {
|
||||
def dispatch(invocation: MessageInvocation):Unit = {
|
||||
def dispatch(invocation: MessageInvocation) {
|
||||
queue {
|
||||
invocation.invoke
|
||||
}
|
||||
|
|
@ -207,14 +186,10 @@ class AggregatingHawtDispatcherMailbox(queue:DispatchQueue) extends HawtDispatch
|
|||
source.setEventHandler (^{drain_source} )
|
||||
source.resume
|
||||
|
||||
private def drain_source = {
|
||||
source.getData.foreach { invocation =>
|
||||
invocation.invoke
|
||||
}
|
||||
}
|
||||
private def drain_source = source.getData.foreach(_.invoke)
|
||||
|
||||
override def dispatch(invocation: MessageInvocation):Unit = {
|
||||
if ( getCurrentQueue == null ) {
|
||||
override def dispatch(invocation: MessageInvocation) {
|
||||
if (getCurrentQueue eq null) {
|
||||
// we are being call from a non hawtdispatch thread, can't aggregate
|
||||
// it's events
|
||||
super.dispatch(invocation)
|
||||
|
|
|
|||
114
akka-actor/src/main/scala/dispatch/MailboxHandling.scala
Normal file
114
akka-actor/src/main/scala/dispatch/MailboxHandling.scala
Normal file
|
|
@ -0,0 +1,114 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2010 Scalable Solutions AB <http://scalablesolutions.se>
|
||||
*/
|
||||
|
||||
package se.scalablesolutions.akka.dispatch
|
||||
|
||||
import se.scalablesolutions.akka.actor.{Actor, ActorRef, ActorInitializationException}
|
||||
import se.scalablesolutions.akka.util.{SimpleLock, Duration, HashCode, Logging}
|
||||
import se.scalablesolutions.akka.util.ReflectiveAccess.EnterpriseModule
|
||||
import se.scalablesolutions.akka.AkkaException
|
||||
|
||||
import java.util.{Queue, List}
|
||||
import java.util.concurrent._
|
||||
import concurrent.forkjoin.LinkedTransferQueue
|
||||
|
||||
class MessageQueueAppendFailedException(message: String) extends AkkaException(message)
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
trait MessageQueue {
|
||||
val dispatcherLock = new SimpleLock
|
||||
def enqueue(handle: MessageInvocation)
|
||||
def dequeue(): MessageInvocation
|
||||
def size: Int
|
||||
def isEmpty: Boolean
|
||||
}
|
||||
|
||||
/**
|
||||
* Mailbox configuration.
|
||||
*/
|
||||
sealed trait MailboxType
|
||||
|
||||
abstract class TransientMailboxType(val blocking: Boolean = false) extends MailboxType
|
||||
case class UnboundedMailbox(block: Boolean = false) extends TransientMailboxType(block)
|
||||
case class BoundedMailbox(
|
||||
block: Boolean = false,
|
||||
val capacity: Int = { if (Dispatchers.MAILBOX_CAPACITY < 0) Int.MaxValue else Dispatchers.MAILBOX_CAPACITY },
|
||||
val pushTimeOut: Duration = Dispatchers.MAILBOX_PUSH_TIME_OUT) extends TransientMailboxType(block) {
|
||||
if (capacity < 0) throw new IllegalArgumentException("The capacity for BoundedMailbox can not be negative")
|
||||
if (pushTimeOut eq null) throw new IllegalArgumentException("The push time-out for BoundedMailbox can not be null")
|
||||
}
|
||||
|
||||
abstract class DurableMailboxType(val serializer: EnterpriseModule.Serializer) extends MailboxType {
|
||||
if (serializer eq null) throw new IllegalArgumentException("The serializer for DurableMailboxType can not be null")
|
||||
}
|
||||
case class FileBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
case class RedisBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
case class BeanstalkBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
case class ZooKeeperBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
case class AMQPBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
case class JMSBasedDurableMailbox(ser: EnterpriseModule.Serializer) extends DurableMailboxType(ser)
|
||||
|
||||
class DefaultUnboundedMessageQueue(blockDequeue: Boolean)
|
||||
extends LinkedBlockingQueue[MessageInvocation] with MessageQueue {
|
||||
|
||||
final def enqueue(handle: MessageInvocation) {
|
||||
this add handle
|
||||
}
|
||||
|
||||
final def dequeue(): MessageInvocation = {
|
||||
if (blockDequeue) this.take()
|
||||
else this.poll()
|
||||
}
|
||||
}
|
||||
|
||||
class DefaultBoundedMessageQueue(capacity: Int, pushTimeOut: Duration, blockDequeue: Boolean)
|
||||
extends LinkedBlockingQueue[MessageInvocation](capacity) with MessageQueue {
|
||||
|
||||
final def enqueue(handle: MessageInvocation) {
|
||||
if (pushTimeOut.toMillis > 0) {
|
||||
if (!this.offer(handle, pushTimeOut.length, pushTimeOut.unit))
|
||||
throw new MessageQueueAppendFailedException("Couldn't enqueue message " + handle + " to " + toString)
|
||||
} else this put handle
|
||||
}
|
||||
|
||||
final def dequeue(): MessageInvocation =
|
||||
if (blockDequeue) this.take()
|
||||
else this.poll()
|
||||
}
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
trait MailboxFactory {
|
||||
|
||||
val mailboxType: Option[MailboxType]
|
||||
|
||||
/**
|
||||
* Creates a MessageQueue (Mailbox) with the specified properties.
|
||||
*/
|
||||
protected def createMailbox(actorRef: ActorRef): AnyRef =
|
||||
mailboxType.getOrElse(throw new IllegalStateException("No mailbox type defined")) match {
|
||||
case mb: TransientMailboxType => createTransientMailbox(actorRef, mb)
|
||||
case mb: DurableMailboxType => createDurableMailbox(actorRef, mb)
|
||||
}
|
||||
|
||||
/**
|
||||
* Creates and returns a transient mailbox for the given actor.
|
||||
*/
|
||||
protected def createTransientMailbox(actorRef: ActorRef, mailboxType: TransientMailboxType): AnyRef
|
||||
|
||||
/**
|
||||
* Creates and returns a durable mailbox for the given actor.
|
||||
*/
|
||||
protected def createDurableMailbox(actorRef: ActorRef, mailboxType: DurableMailboxType): AnyRef = mailboxType match {
|
||||
case FileBasedDurableMailbox(serializer) => EnterpriseModule.createFileBasedMailbox(actorRef.uuid, serializer).asInstanceOf[MessageQueue]
|
||||
case RedisBasedDurableMailbox(serializer) => throw new UnsupportedOperationException("RedisBasedDurableMailbox is not yet supported")
|
||||
case BeanstalkBasedDurableMailbox(serializer) => throw new UnsupportedOperationException("BeanstalkBasedDurableMailbox is not yet supported")
|
||||
case ZooKeeperBasedDurableMailbox(serializer) => throw new UnsupportedOperationException("ZooKeeperBasedDurableMailbox is not yet supported")
|
||||
case AMQPBasedDurableMailbox(serializer) => throw new UnsupportedOperationException("AMQPBasedDurableMailbox is not yet supported")
|
||||
case JMSBasedDurableMailbox(serializer) => throw new UnsupportedOperationException("JMSBasedDurableMailbox is not yet supported")
|
||||
}
|
||||
}
|
||||
|
|
@ -5,13 +5,15 @@
|
|||
package se.scalablesolutions.akka.dispatch
|
||||
|
||||
import se.scalablesolutions.akka.actor.{Actor, ActorRef, ActorInitializationException}
|
||||
import se.scalablesolutions.akka.util.{SimpleLock, Duration, HashCode, Logging}
|
||||
import se.scalablesolutions.akka.util.ReflectiveAccess.EnterpriseModule
|
||||
import se.scalablesolutions.akka.AkkaException
|
||||
|
||||
import org.multiverse.commitbarriers.CountDownCommitBarrier
|
||||
import se.scalablesolutions.akka.AkkaException
|
||||
|
||||
import java.util.{Queue, List}
|
||||
import java.util.concurrent._
|
||||
import concurrent.forkjoin.LinkedTransferQueue
|
||||
import se.scalablesolutions.akka.util.{SimpleLock, Duration, HashCode, Logging}
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
|
|
@ -21,30 +23,30 @@ final class MessageInvocation(val receiver: ActorRef,
|
|||
val sender: Option[ActorRef],
|
||||
val senderFuture: Option[CompletableFuture[Any]],
|
||||
val transactionSet: Option[CountDownCommitBarrier]) {
|
||||
if (receiver eq null) throw new IllegalArgumentException("receiver is null")
|
||||
if (receiver eq null) throw new IllegalArgumentException("Receiver can't be null")
|
||||
|
||||
def invoke = try {
|
||||
receiver.invoke(this)
|
||||
} catch {
|
||||
case e: NullPointerException => throw new ActorInitializationException(
|
||||
"Don't call 'self ! message' in the Actor's constructor (e.g. body of the class).")
|
||||
"Don't call 'self ! message' in the Actor's constructor (in Scala this means in the body of the class).")
|
||||
}
|
||||
|
||||
override def hashCode(): Int = synchronized {
|
||||
override def hashCode(): Int = {
|
||||
var result = HashCode.SEED
|
||||
result = HashCode.hash(result, receiver.actor)
|
||||
result = HashCode.hash(result, message.asInstanceOf[AnyRef])
|
||||
result
|
||||
}
|
||||
|
||||
override def equals(that: Any): Boolean = synchronized {
|
||||
override def equals(that: Any): Boolean = {
|
||||
that != null &&
|
||||
that.isInstanceOf[MessageInvocation] &&
|
||||
that.asInstanceOf[MessageInvocation].receiver.actor == receiver.actor &&
|
||||
that.asInstanceOf[MessageInvocation].message == message
|
||||
}
|
||||
|
||||
override def toString = synchronized {
|
||||
override def toString = {
|
||||
"MessageInvocation[" +
|
||||
"\n\tmessage = " + message +
|
||||
"\n\treceiver = " + receiver +
|
||||
|
|
@ -55,83 +57,24 @@ final class MessageInvocation(val receiver: ActorRef,
|
|||
}
|
||||
}
|
||||
|
||||
class MessageQueueAppendFailedException(message: String) extends AkkaException(message)
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
trait MessageQueue {
|
||||
val dispatcherLock = new SimpleLock
|
||||
def enqueue(handle: MessageInvocation)
|
||||
def dequeue(): MessageInvocation
|
||||
def size: Int
|
||||
def isEmpty: Boolean
|
||||
}
|
||||
trait MessageDispatcher extends MailboxFactory with Logging {
|
||||
|
||||
/* Tells the dispatcher that it should create a bounded mailbox with the specified push timeout
|
||||
* (If capacity > 0)
|
||||
*/
|
||||
case class MailboxConfig(capacity: Int, pushTimeOut: Option[Duration], blockingDequeue: Boolean) {
|
||||
|
||||
/**
|
||||
* Creates a MessageQueue (Mailbox) with the specified properties
|
||||
* bounds = whether the mailbox should be bounded (< 0 means unbounded)
|
||||
* pushTime = only used if bounded, indicates if and how long an enqueue should block
|
||||
* blockDequeue = whether dequeues should block or not
|
||||
*
|
||||
* The bounds + pushTime generates a MessageQueueAppendFailedException if enqueue times out
|
||||
*/
|
||||
def newMailbox(bounds: Int = capacity,
|
||||
pushTime: Option[Duration] = pushTimeOut,
|
||||
blockDequeue: Boolean = blockingDequeue) : MessageQueue =
|
||||
if (capacity > 0) new DefaultBoundedMessageQueue(bounds,pushTime,blockDequeue)
|
||||
else new DefaultUnboundedMessageQueue(blockDequeue)
|
||||
}
|
||||
|
||||
class DefaultUnboundedMessageQueue(blockDequeue: Boolean) extends LinkedBlockingQueue[MessageInvocation] with MessageQueue {
|
||||
final def enqueue(handle: MessageInvocation) {
|
||||
this add handle
|
||||
}
|
||||
|
||||
final def dequeue(): MessageInvocation =
|
||||
if (blockDequeue) this.take()
|
||||
else this.poll()
|
||||
}
|
||||
|
||||
class DefaultBoundedMessageQueue(capacity: Int, pushTimeOut: Option[Duration], blockDequeue: Boolean) extends LinkedBlockingQueue[MessageInvocation](capacity) with MessageQueue {
|
||||
final def enqueue(handle: MessageInvocation) {
|
||||
if (pushTimeOut.isDefined) {
|
||||
if(!this.offer(handle,pushTimeOut.get.length,pushTimeOut.get.unit))
|
||||
throw new MessageQueueAppendFailedException("Couldn't enqueue message " + handle + " to " + toString)
|
||||
}
|
||||
else {
|
||||
this put handle
|
||||
}
|
||||
}
|
||||
|
||||
final def dequeue(): MessageInvocation =
|
||||
if (blockDequeue) this.take()
|
||||
else this.poll()
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
trait MessageDispatcher extends Logging {
|
||||
protected val uuids = new ConcurrentSkipListSet[String]
|
||||
|
||||
def dispatch(invocation: MessageInvocation)
|
||||
def dispatch(invocation: MessageInvocation): Unit
|
||||
|
||||
def start
|
||||
def start: Unit
|
||||
|
||||
def shutdown
|
||||
def shutdown: Unit
|
||||
|
||||
def register(actorRef: ActorRef) {
|
||||
if(actorRef.mailbox eq null)
|
||||
actorRef.mailbox = createMailbox(actorRef)
|
||||
if (actorRef.mailbox eq null) actorRef.mailbox = createMailbox(actorRef)
|
||||
uuids add actorRef.uuid
|
||||
}
|
||||
|
||||
def unregister(actorRef: ActorRef) = {
|
||||
uuids remove actorRef.uuid
|
||||
actorRef.mailbox = null
|
||||
|
|
@ -146,9 +89,4 @@ trait MessageDispatcher extends Logging {
|
|||
* Returns the size of the mailbox for the specified actor
|
||||
*/
|
||||
def mailboxSize(actorRef: ActorRef): Int
|
||||
|
||||
/**
|
||||
* Creates and returns a mailbox for the given actor
|
||||
*/
|
||||
protected def createMailbox(actorRef: ActorRef): AnyRef = null
|
||||
}
|
||||
|
|
@ -4,11 +4,11 @@
|
|||
|
||||
package se.scalablesolutions.akka.dispatch
|
||||
|
||||
import java.util.Queue
|
||||
|
||||
import se.scalablesolutions.akka.actor.{Actor, ActorRef}
|
||||
import se.scalablesolutions.akka.config.Config.config
|
||||
import concurrent.forkjoin.{TransferQueue, LinkedTransferQueue}
|
||||
import se.scalablesolutions.akka.util.Duration
|
||||
|
||||
import java.util.Queue
|
||||
import java.util.concurrent.{ConcurrentLinkedQueue, BlockingQueue, TimeUnit, LinkedBlockingQueue}
|
||||
|
||||
/**
|
||||
|
|
@ -16,23 +16,30 @@ import java.util.concurrent.{ConcurrentLinkedQueue, BlockingQueue, TimeUnit, Lin
|
|||
*
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
class ThreadBasedDispatcher(private val actor: ActorRef,
|
||||
val mailboxConfig: MailboxConfig
|
||||
) extends MessageDispatcher {
|
||||
def this(actor: ActorRef, capacity: Int) = this(actor,MailboxConfig(capacity,None,true))
|
||||
def this(actor: ActorRef) = this(actor, Dispatchers.MAILBOX_CAPACITY)// For Java
|
||||
class ThreadBasedDispatcher(private val actor: ActorRef, _mailboxType: MailboxType) extends MessageDispatcher {
|
||||
|
||||
def this(actor: ActorRef) = this(actor, BoundedMailbox(true)) // For Java API
|
||||
|
||||
def this(actor: ActorRef, capacity: Int) = this(actor, BoundedMailbox(true, capacity))
|
||||
|
||||
def this(actor: ActorRef, capacity: Int, pushTimeOut: Duration) = this(actor, BoundedMailbox(true, capacity, pushTimeOut))
|
||||
|
||||
val mailboxType = Some(_mailboxType)
|
||||
|
||||
private val name = actor.getClass.getName + ":" + actor.uuid
|
||||
private val threadName = "akka:thread-based:dispatcher:" + name
|
||||
private var selectorThread: Thread = _
|
||||
@volatile private var active: Boolean = false
|
||||
|
||||
override def createMailbox(actorRef: ActorRef): AnyRef = mailboxConfig.newMailbox(blockDequeue = true)
|
||||
def createTransientMailbox(actorRef: ActorRef, mailboxType: TransientMailboxType): AnyRef = mailboxType match {
|
||||
case UnboundedMailbox(blocking) =>
|
||||
new DefaultUnboundedMessageQueue(blocking)
|
||||
case BoundedMailbox(blocking, capacity, pushTimeOut) =>
|
||||
new DefaultBoundedMessageQueue(capacity, pushTimeOut, blocking)
|
||||
}
|
||||
|
||||
override def register(actorRef: ActorRef) = {
|
||||
if(actorRef != actor)
|
||||
throw new IllegalArgumentException("Cannot register to anyone but " + actor)
|
||||
|
||||
if (actorRef != actor) throw new IllegalArgumentException("Cannot register to anyone but " + actor)
|
||||
super.register(actorRef)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -5,29 +5,31 @@
|
|||
package se.scalablesolutions.akka.util
|
||||
|
||||
import se.scalablesolutions.akka.actor.{ActorRef, IllegalActorStateException, ActorType}
|
||||
import se.scalablesolutions.akka.dispatch.{Future, CompletableFuture}
|
||||
import se.scalablesolutions.akka.dispatch.{Future, CompletableFuture, MessageInvocation}
|
||||
import se.scalablesolutions.akka.config.{Config, ModuleNotAvailableException}
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
import se.scalablesolutions.akka.stm.Transaction
|
||||
import se.scalablesolutions.akka.AkkaException
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
|
||||
/**
|
||||
* Helper class for reflective access to different modules in order to allow optional loading of modules.
|
||||
*
|
||||
* @author <a href="http://jonasboner.com">Jonas Bonér</a>
|
||||
*/
|
||||
object ReflectiveAccess {
|
||||
object ReflectiveAccess extends Logging {
|
||||
|
||||
val loader = getClass.getClassLoader
|
||||
|
||||
lazy val isRemotingEnabled = RemoteClientModule.isRemotingEnabled
|
||||
lazy val isTypedActorEnabled = TypedActorModule.isTypedActorEnabled
|
||||
lazy val isJtaEnabled = JtaModule.isJtaEnabled
|
||||
lazy val isEnterpriseEnabled = EnterpriseModule.isEnterpriseEnabled
|
||||
|
||||
def ensureRemotingEnabled = RemoteClientModule.ensureRemotingEnabled
|
||||
def ensureTypedActorEnabled = TypedActorModule.ensureTypedActorEnabled
|
||||
def ensureJtaEnabled = JtaModule.ensureJtaEnabled
|
||||
def ensureEnterpriseEnabled = EnterpriseModule.ensureEnterpriseEnabled
|
||||
|
||||
/**
|
||||
* Reflective access to the RemoteClient module.
|
||||
|
|
@ -63,7 +65,7 @@ object ReflectiveAccess {
|
|||
"Can't load the remoting module, make sure that akka-remote.jar is on the classpath")
|
||||
|
||||
val remoteClientObjectInstance: Option[RemoteClientObject] =
|
||||
getObject("se.scalablesolutions.akka.remote.RemoteClient$")
|
||||
getObjectFor("se.scalablesolutions.akka.remote.RemoteClient$")
|
||||
|
||||
def register(address: InetSocketAddress, uuid: String) = {
|
||||
ensureRemotingEnabled
|
||||
|
|
@ -121,10 +123,10 @@ object ReflectiveAccess {
|
|||
}
|
||||
|
||||
val remoteServerObjectInstance: Option[RemoteServerObject] =
|
||||
getObject("se.scalablesolutions.akka.remote.RemoteServer$")
|
||||
getObjectFor("se.scalablesolutions.akka.remote.RemoteServer$")
|
||||
|
||||
val remoteNodeObjectInstance: Option[RemoteNodeObject] =
|
||||
getObject("se.scalablesolutions.akka.remote.RemoteNode$")
|
||||
getObjectFor("se.scalablesolutions.akka.remote.RemoteNode$")
|
||||
|
||||
def registerActor(address: InetSocketAddress, uuid: String, actorRef: ActorRef) = {
|
||||
ensureRemotingEnabled
|
||||
|
|
@ -160,7 +162,7 @@ object ReflectiveAccess {
|
|||
"Can't load the typed actor module, make sure that akka-typed-actor.jar is on the classpath")
|
||||
|
||||
val typedActorObjectInstance: Option[TypedActorObject] =
|
||||
getObject("se.scalablesolutions.akka.actor.TypedActor$")
|
||||
getObjectFor("se.scalablesolutions.akka.actor.TypedActor$")
|
||||
|
||||
def resolveFutureIfMessageIsJoinPoint(message: Any, future: Future[_]): Boolean = {
|
||||
ensureTypedActorEnabled
|
||||
|
|
@ -189,7 +191,7 @@ object ReflectiveAccess {
|
|||
"Can't load the typed actor module, make sure that akka-jta.jar is on the classpath")
|
||||
|
||||
val transactionContainerObjectInstance: Option[TransactionContainerObject] =
|
||||
getObject("se.scalablesolutions.akka.actor.TransactionContainer$")
|
||||
getObjectFor("se.scalablesolutions.akka.actor.TransactionContainer$")
|
||||
|
||||
def createTransactionContainer: TransactionContainer = {
|
||||
ensureJtaEnabled
|
||||
|
|
@ -197,36 +199,100 @@ object ReflectiveAccess {
|
|||
}
|
||||
}
|
||||
|
||||
object EnterpriseModule {
|
||||
|
||||
type FileBasedMailbox = {
|
||||
def enqueue(message: MessageInvocation)
|
||||
def dequeue: MessageInvocation
|
||||
}
|
||||
|
||||
type Serializer = {
|
||||
def toBinary(obj: AnyRef): Array[Byte]
|
||||
def fromBinary(bytes: Array[Byte], clazz: Option[Class[_]]): AnyRef
|
||||
}
|
||||
|
||||
lazy val isEnterpriseEnabled = clusterObjectInstance.isDefined
|
||||
|
||||
val clusterObjectInstance: Option[AnyRef] =
|
||||
getObjectFor("se.scalablesolutions.akka.cluster.Cluster$")
|
||||
|
||||
val serializerClass: Option[Class[_]] =
|
||||
getClassFor("se.scalablesolutions.akka.serialization.Serializer")
|
||||
|
||||
def ensureEnterpriseEnabled = if (!isEnterpriseEnabled) throw new ModuleNotAvailableException(
|
||||
"Feature is only available in Akka Enterprise")
|
||||
|
||||
def createFileBasedMailbox(name: String, serializer: Serializer): FileBasedMailbox = {
|
||||
ensureEnterpriseEnabled
|
||||
createInstance(
|
||||
"se.scalablesolutions.akka.cluster.FileBasedMailbox",
|
||||
Array(classOf[String], serializerClass.get),
|
||||
Array(name, serializer).asInstanceOf[Array[AnyRef]],
|
||||
loader)
|
||||
.getOrElse(throw new IllegalActorStateException("Could not create file-based mailbox"))
|
||||
.asInstanceOf[FileBasedMailbox]
|
||||
}
|
||||
}
|
||||
|
||||
val noParams = Array[Class[_]]()
|
||||
val noArgs = Array[AnyRef]()
|
||||
|
||||
def createInstance[T](clazz: Class[_],
|
||||
params: Array[Class[_]],
|
||||
args: Array[AnyRef]): Option[T] = try {
|
||||
assert(clazz ne null)
|
||||
assert(params ne null)
|
||||
assert(args ne null)
|
||||
val ctor = clazz.getDeclaredConstructor(params: _*)
|
||||
ctor.setAccessible(true)
|
||||
Some(ctor.newInstance(args: _*).asInstanceOf[T])
|
||||
} catch {
|
||||
case e: Exception => None
|
||||
case e: java.lang.reflect.InvocationTargetException =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", clazz.getName)
|
||||
None
|
||||
case e: Exception =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", clazz.getName)
|
||||
None
|
||||
}
|
||||
|
||||
def createInstance[T](fqn: String,
|
||||
params: Array[Class[_]],
|
||||
args: Array[AnyRef],
|
||||
classloader: ClassLoader = loader): Option[T] = try {
|
||||
assert(fqn ne null)
|
||||
assert(params ne null)
|
||||
assert(args ne null)
|
||||
val clazz = classloader.loadClass(fqn)
|
||||
val ctor = clazz.getDeclaredConstructor(params: _*)
|
||||
ctor.setAccessible(true)
|
||||
Some(ctor.newInstance(args: _*).asInstanceOf[T])
|
||||
} catch {
|
||||
case e: Exception => None
|
||||
case e: java.lang.reflect.InvocationTargetException =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", fqn)
|
||||
None
|
||||
case e: Exception =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", fqn)
|
||||
None
|
||||
}
|
||||
|
||||
def getObject[T](fqn: String, classloader: ClassLoader = loader): Option[T] = try {//Obtains a reference to $MODULE$
|
||||
def getObjectFor[T](fqn: String, classloader: ClassLoader = loader): Option[T] = try {//Obtains a reference to $MODULE$
|
||||
assert(fqn ne null)
|
||||
val clazz = classloader.loadClass(fqn)
|
||||
val instance = clazz.getDeclaredField("MODULE$")
|
||||
instance.setAccessible(true)
|
||||
Option(instance.get(null).asInstanceOf[T])
|
||||
} catch {
|
||||
case e: java.lang.reflect.InvocationTargetException =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", fqn)
|
||||
None
|
||||
case e: Exception =>
|
||||
log.error(e.getCause, "Could not instantiate class [%s]", fqn)
|
||||
None
|
||||
}
|
||||
|
||||
def getClassFor[T](fqn: String, classloader: ClassLoader = loader): Option[Class[T]] = try {
|
||||
assert(fqn ne null)
|
||||
Some(classloader.loadClass(fqn).asInstanceOf[Class[T]])
|
||||
} catch {
|
||||
case e: Exception => None
|
||||
}
|
||||
|
|
|
|||
|
|
@ -10,7 +10,6 @@ import Actor._
|
|||
|
||||
object ActorFireForgetRequestReplySpec {
|
||||
class ReplyActor extends Actor {
|
||||
self.dispatcher = Dispatchers.newThreadBasedDispatcher(self)
|
||||
|
||||
def receive = {
|
||||
case "Send" =>
|
||||
|
|
@ -31,10 +30,10 @@ object ActorFireForgetRequestReplySpec {
|
|||
}
|
||||
|
||||
class SenderActor(replyActor: ActorRef) extends Actor {
|
||||
self.dispatcher = Dispatchers.newThreadBasedDispatcher(self)
|
||||
|
||||
def receive = {
|
||||
case "Init" => replyActor ! "Send"
|
||||
case "Init" =>
|
||||
replyActor ! "Send"
|
||||
case "Reply" => {
|
||||
state.s = "Reply"
|
||||
state.finished.await
|
||||
|
|
@ -84,7 +83,7 @@ class ActorFireForgetRequestReplySpec extends JUnitSuite {
|
|||
val actor = actorOf[CrashingTemporaryActor].start
|
||||
assert(actor.isRunning)
|
||||
actor ! "Die"
|
||||
try { state.finished.await(1L, TimeUnit.SECONDS) }
|
||||
try { state.finished.await(10L, TimeUnit.SECONDS) }
|
||||
catch { case e: TimeoutException => fail("Never got the message") }
|
||||
Thread.sleep(100)
|
||||
assert(actor.isShutdown)
|
||||
|
|
|
|||
|
|
@ -15,7 +15,7 @@ object DispatchersSpec {
|
|||
import Dispatchers._
|
||||
//
|
||||
val tipe = "type"
|
||||
val keepalivems = "keep-alive-ms"
|
||||
val keepalivems = "keep-alive-time"
|
||||
val corepoolsizefactor = "core-pool-size-factor"
|
||||
val maxpoolsizefactor = "max-pool-size-factor"
|
||||
val executorbounds = "executor-bounds"
|
||||
|
|
|
|||
|
|
@ -68,7 +68,7 @@ class ExecutorBasedEventDrivenDispatcherActorSpec extends JUnitSuite {
|
|||
}
|
||||
|
||||
@Test def shouldRespectThroughput {
|
||||
val throughputDispatcher = new ExecutorBasedEventDrivenDispatcher("THROUGHPUT",101,0,Dispatchers.MAILBOX_CONFIG, (e) => {
|
||||
val throughputDispatcher = new ExecutorBasedEventDrivenDispatcher("THROUGHPUT",101,0,Dispatchers.MAILBOX_TYPE, (e) => {
|
||||
e.setCorePoolSize(1)
|
||||
})
|
||||
|
||||
|
|
@ -103,7 +103,7 @@ class ExecutorBasedEventDrivenDispatcherActorSpec extends JUnitSuite {
|
|||
|
||||
@Test def shouldRespectThroughputDeadline {
|
||||
val deadlineMs = 100
|
||||
val throughputDispatcher = new ExecutorBasedEventDrivenDispatcher("THROUGHPUT",2,deadlineMs,Dispatchers.MAILBOX_CONFIG, (e) => {
|
||||
val throughputDispatcher = new ExecutorBasedEventDrivenDispatcher("THROUGHPUT",2,deadlineMs,Dispatchers.MAILBOX_TYPE, (e) => {
|
||||
e.setCorePoolSize(1)
|
||||
})
|
||||
|
||||
|
|
|
|||
|
|
@ -1,44 +1,44 @@
|
|||
package se.scalablesolutions.akka.actor.dispatch
|
||||
|
||||
import org.scalatest.junit.JUnitSuite
|
||||
|
||||
import org.junit.Test
|
||||
|
||||
import se.scalablesolutions.akka.actor.Actor
|
||||
import Actor._
|
||||
import java.util.concurrent.{BlockingQueue, CountDownLatch, TimeUnit}
|
||||
import se.scalablesolutions.akka.util.Duration
|
||||
import se.scalablesolutions.akka.dispatch.{MessageQueueAppendFailedException, MessageInvocation, MailboxConfig, Dispatchers}
|
||||
import java.util.concurrent.atomic.{AtomicReference}
|
||||
import se.scalablesolutions.akka.dispatch._
|
||||
import Actor._
|
||||
|
||||
object MailboxConfigSpec {
|
||||
import java.util.concurrent.{BlockingQueue, CountDownLatch, TimeUnit}
|
||||
import java.util.concurrent.atomic.AtomicReference
|
||||
|
||||
}
|
||||
|
||||
class MailboxConfigSpec extends JUnitSuite {
|
||||
import MailboxConfigSpec._
|
||||
class MailboxTypeSpec extends JUnitSuite {
|
||||
@Test def shouldDoNothing = assert(true)
|
||||
|
||||
/*
|
||||
private val unit = TimeUnit.MILLISECONDS
|
||||
|
||||
@Test def shouldCreateUnboundedQueue = {
|
||||
val m = MailboxConfig(-1,None,false)
|
||||
assert(m.newMailbox().asInstanceOf[BlockingQueue[MessageInvocation]].remainingCapacity === Integer.MAX_VALUE)
|
||||
val m = UnboundedMailbox(false)
|
||||
assert(m.newMailbox("uuid").asInstanceOf[BlockingQueue[MessageInvocation]].remainingCapacity === Integer.MAX_VALUE)
|
||||
}
|
||||
|
||||
@Test def shouldCreateBoundedQueue = {
|
||||
val m = MailboxConfig(1,None,false)
|
||||
assert(m.newMailbox().asInstanceOf[BlockingQueue[MessageInvocation]].remainingCapacity === 1)
|
||||
val m = BoundedMailbox(blocking = false, capacity = 1)
|
||||
assert(m.newMailbox("uuid").asInstanceOf[BlockingQueue[MessageInvocation]].remainingCapacity === 1)
|
||||
}
|
||||
|
||||
@Test(expected = classOf[MessageQueueAppendFailedException]) def shouldThrowMessageQueueAppendFailedExceptionWhenTimeOutEnqueue = {
|
||||
val m = MailboxConfig(1,Some(Duration(1,unit)),false)
|
||||
val m = BoundedMailbox(false, 1, Duration(1, unit))
|
||||
val testActor = actorOf( new Actor { def receive = { case _ => }} )
|
||||
val mbox = m.newMailbox()
|
||||
val mbox = m.newMailbox("uuid")
|
||||
(1 to 10000) foreach { i => mbox.enqueue(new MessageInvocation(testActor, i, None, None, None)) }
|
||||
}
|
||||
|
||||
|
||||
@Test def shouldBeAbleToDequeueUnblocking = {
|
||||
val m = MailboxConfig(1,Some(Duration(1,unit)),false)
|
||||
val mbox = m.newMailbox()
|
||||
val m = BoundedMailbox(false, 1, Duration(1, unit))
|
||||
val mbox = m.newMailbox("uuid")
|
||||
val latch = new CountDownLatch(1)
|
||||
val t = new Thread { override def run = {
|
||||
mbox.dequeue
|
||||
|
|
@ -50,4 +50,5 @@ class MailboxConfigSpec extends JUnitSuite {
|
|||
t.interrupt
|
||||
assert(result === true)
|
||||
}
|
||||
*/
|
||||
}
|
||||
|
|
|
|||
|
|
@ -58,7 +58,7 @@ class DispatcherSpringFeatureTest extends FeatureSpec with ShouldMatchers {
|
|||
assert(executor.getQueue().remainingCapacity() === Integer.MAX_VALUE)
|
||||
assert(dispatcher.name === EVENT_DRIVEN_PREFIX + "dispatcher-2")
|
||||
}
|
||||
|
||||
/*
|
||||
scenario("get a executor-event-driven-dispatcher with bounded-blocking-queue and with bounded mailbox capacity") {
|
||||
val context = new ClassPathXmlApplicationContext("/dispatcher-config.xml")
|
||||
val dispatcher = context.getBean("executor-event-driven-dispatcher-mc").asInstanceOf[ExecutorBasedEventDrivenDispatcher]
|
||||
|
|
@ -69,7 +69,7 @@ class DispatcherSpringFeatureTest extends FeatureSpec with ShouldMatchers {
|
|||
assert(actorRef.mailbox.isInstanceOf[BlockingQueue[MessageInvocation]])
|
||||
assert((actorRef.mailbox.asInstanceOf[BlockingQueue[MessageInvocation]]).remainingCapacity === 1000)
|
||||
}
|
||||
|
||||
*/
|
||||
scenario("get a executor-event-driven-dispatcher with unbounded-linked-blocking-queue with bounded capacity from context") {
|
||||
val context = new ClassPathXmlApplicationContext("/dispatcher-config.xml")
|
||||
val dispatcher = context.getBean("executor-event-driven-dispatcher-4").asInstanceOf[ExecutorBasedEventDrivenDispatcher]
|
||||
|
|
|
|||
|
|
@ -25,7 +25,7 @@ akka {
|
|||
# - TypedActor: methods with non-void return type
|
||||
serialize-messages = off # Does a deep clone of (non-primitive) messages to ensure immutability
|
||||
throughput = 5 # Default throughput for all ExecutorBasedEventDrivenDispatcher, set to 1 for complete fairness
|
||||
throughput-deadline-ms = -1 # Default throughput deadline for all ExecutorBasedEventDrivenDispatcher, set to 0 or negative for no deadline
|
||||
throughput-deadline-time = -1 # Default throughput deadline for all ExecutorBasedEventDrivenDispatcher, set to 0 or negative for no deadline
|
||||
|
||||
default-dispatcher {
|
||||
type = "GlobalExecutorBasedEventDriven" # Must be one of the following, all "Global*" are non-configurable
|
||||
|
|
@ -38,14 +38,14 @@ akka {
|
|||
# - GlobalExecutorBasedEventDriven
|
||||
# - GlobalReactorBasedSingleThreadEventDriven
|
||||
# - GlobalReactorBasedThreadPoolEventDriven
|
||||
keep-alive-ms = 60000 # Keep alive time for threads
|
||||
keep-alive-time = 60 # Keep alive time for threads
|
||||
core-pool-size-factor = 1.0 # No of core threads ... ceil(available processors * factor)
|
||||
max-pool-size-factor = 4.0 # Max no of threads ... ceil(available processors * factor)
|
||||
executor-bounds = -1 # Makes the Executor bounded, -1 is unbounded
|
||||
allow-core-timeout = on # Allow core threads to time out
|
||||
rejection-policy = "caller-runs" # abort, caller-runs, discard-oldest, discard
|
||||
throughput = 5 # Throughput for ExecutorBasedEventDrivenDispatcher, set to 1 for complete fairness
|
||||
throughput-deadline-ms = -1 # Throughput deadline for ExecutorBasedEventDrivenDispatcher, set to 0 or negative for no deadline
|
||||
throughput-deadline-time = -1 # Throughput deadline for ExecutorBasedEventDrivenDispatcher, set to 0 or negative for no deadline
|
||||
aggregate = off # Aggregate on/off for HawtDispatchers
|
||||
mailbox-capacity = -1 # If negative (or zero) then an unbounded mailbox is used (default)
|
||||
# If positive then a bounded mailbox is used and the capacity is set using the property
|
||||
|
|
@ -54,7 +54,8 @@ akka {
|
|||
#
|
||||
# The following are only used for ExecutorBasedEventDriven
|
||||
# and only if mailbox-capacity > 0
|
||||
mailbox-push-timeout-ms = 10000 # Specifies the timeout (in milliseconds) to add a new message to a mailbox that is full
|
||||
mailbox-push-timeout-time = 10 # Specifies the timeout to add a new message to a mailbox that is full - negative number means infinite timeout
|
||||
# (in unit defined by the time-unit property)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue