2011-03-11 14:51:24 +01:00
|
|
|
package akka.dispatch
|
2011-12-19 21:46:37 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
import org.scalatest.{ BeforeAndAfterAll, BeforeAndAfterEach }
|
2011-11-23 15:15:44 +01:00
|
|
|
import java.util.concurrent.{ TimeUnit, BlockingQueue }
|
2011-12-19 21:46:37 +01:00
|
|
|
import java.util.concurrent.ConcurrentLinkedQueue
|
2011-03-11 14:51:24 +01:00
|
|
|
import akka.util._
|
2011-11-23 15:15:44 +01:00
|
|
|
import akka.util.duration._
|
2011-10-11 16:05:48 +02:00
|
|
|
import akka.testkit.AkkaSpec
|
2012-02-19 10:28:56 +01:00
|
|
|
import akka.actor.{ ActorRef, ActorContext, Props, LocalActorRef }
|
2011-12-21 19:37:18 +01:00
|
|
|
import com.typesafe.config.Config
|
2012-02-26 21:26:25 +01:00
|
|
|
import akka.actor.ActorSystem
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2011-10-21 17:01:22 +02:00
|
|
|
@org.junit.runner.RunWith(classOf[org.scalatest.junit.JUnitRunner])
|
2011-10-11 16:05:48 +02:00
|
|
|
abstract class MailboxSpec extends AkkaSpec with BeforeAndAfterAll with BeforeAndAfterEach {
|
2011-05-18 17:25:30 +02:00
|
|
|
def name: String
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2012-02-19 10:28:56 +01:00
|
|
|
def factory: MailboxType ⇒ MessageQueue
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
name should {
|
|
|
|
|
"create an unbounded mailbox" in {
|
|
|
|
|
val config = UnboundedMailbox()
|
|
|
|
|
val q = factory(config)
|
|
|
|
|
ensureInitialMailboxState(config, q)
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2011-12-11 00:40:52 +01:00
|
|
|
val f = spawn { q.dequeue }
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2011-12-12 22:50:08 +01:00
|
|
|
Await.result(f, 1 second) must be(null)
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
"create a bounded mailbox with 10 capacity and with push timeout" in {
|
2011-11-23 15:15:44 +01:00
|
|
|
val config = BoundedMailbox(10, 10 milliseconds)
|
2011-05-18 17:25:30 +02:00
|
|
|
val q = factory(config)
|
|
|
|
|
ensureInitialMailboxState(config, q)
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
val exampleMessage = createMessageInvocation("test")
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-11-12 10:57:28 +01:00
|
|
|
for (i ← 1 to config.capacity) q.enqueue(null, exampleMessage)
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-09-21 16:27:31 +02:00
|
|
|
q.numberOfMessages must be === config.capacity
|
|
|
|
|
q.hasMessages must be === true
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
intercept[MessageQueueAppendFailedException] {
|
2011-11-12 10:57:28 +01:00
|
|
|
q.enqueue(null, exampleMessage)
|
2011-03-11 16:48:44 +01:00
|
|
|
}
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
q.dequeue must be === exampleMessage
|
2011-09-21 16:27:31 +02:00
|
|
|
q.numberOfMessages must be(config.capacity - 1)
|
|
|
|
|
q.hasMessages must be === true
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
2010-09-09 15:49:59 +02:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
"dequeue what was enqueued properly for unbounded mailboxes" in {
|
|
|
|
|
testEnqueueDequeue(UnboundedMailbox())
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
|
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
"dequeue what was enqueued properly for bounded mailboxes" in {
|
2011-11-23 15:15:44 +01:00
|
|
|
testEnqueueDequeue(BoundedMailbox(10000, -1 millisecond))
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
|
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
"dequeue what was enqueued properly for bounded mailboxes with pushTimeout" in {
|
2011-11-23 15:15:44 +01:00
|
|
|
testEnqueueDequeue(BoundedMailbox(10000, 100 milliseconds))
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
2011-03-11 14:51:24 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
//CANDIDATE FOR TESTKIT
|
2011-12-11 00:40:52 +01:00
|
|
|
def spawn[T <: AnyRef](fun: ⇒ T): Future[T] = {
|
|
|
|
|
val result = Promise[T]()
|
2011-05-18 17:25:30 +02:00
|
|
|
val t = new Thread(new Runnable {
|
|
|
|
|
def run = try {
|
2011-12-12 17:25:34 +01:00
|
|
|
result.success(fun)
|
2011-05-18 17:25:30 +02:00
|
|
|
} catch {
|
2011-12-12 17:25:34 +01:00
|
|
|
case e: Throwable ⇒ result.failure(e)
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
})
|
2011-09-08 11:02:17 +02:00
|
|
|
t.start
|
2011-05-18 17:25:30 +02:00
|
|
|
result
|
|
|
|
|
}
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-12-27 16:22:24 +01:00
|
|
|
def createMessageInvocation(msg: Any): Envelope = Envelope(msg, system.deadLetters)(system)
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2012-02-19 10:28:56 +01:00
|
|
|
def ensureInitialMailboxState(config: MailboxType, q: MessageQueue) {
|
2011-05-18 17:25:30 +02:00
|
|
|
q must not be null
|
|
|
|
|
q match {
|
|
|
|
|
case aQueue: BlockingQueue[_] ⇒
|
|
|
|
|
config match {
|
|
|
|
|
case BoundedMailbox(capacity, _) ⇒ aQueue.remainingCapacity must be === capacity
|
|
|
|
|
case UnboundedMailbox() ⇒ aQueue.remainingCapacity must be === Int.MaxValue
|
2011-03-11 16:48:44 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
case _ ⇒
|
|
|
|
|
}
|
2011-09-21 16:27:31 +02:00
|
|
|
q.numberOfMessages must be === 0
|
|
|
|
|
q.hasMessages must be === false
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def testEnqueueDequeue(config: MailboxType) {
|
2011-11-23 15:15:44 +01:00
|
|
|
implicit val within = 10 seconds
|
2011-05-18 17:25:30 +02:00
|
|
|
val q = factory(config)
|
|
|
|
|
ensureInitialMailboxState(config, q)
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-09-21 15:01:47 +02:00
|
|
|
def createProducer(fromNum: Int, toNum: Int): Future[Vector[Envelope]] = spawn {
|
2011-05-18 17:25:30 +02:00
|
|
|
val messages = Vector() ++ (for (i ← fromNum to toNum) yield createMessageInvocation(i))
|
2011-11-12 10:57:28 +01:00
|
|
|
for (i ← messages) q.enqueue(null, i)
|
2011-05-18 17:25:30 +02:00
|
|
|
messages
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
val totalMessages = 10000
|
|
|
|
|
val step = 500
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
val producers = for (i ← (1 to totalMessages by step).toList) yield createProducer(i, i + step - 1)
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-09-21 15:01:47 +02:00
|
|
|
def createConsumer: Future[Vector[Envelope]] = spawn {
|
|
|
|
|
var r = Vector[Envelope]()
|
2012-05-03 21:14:47 +02:00
|
|
|
while (producers.exists(_.isCompleted == false) || q.hasMessages) {
|
2011-05-18 17:25:30 +02:00
|
|
|
q.dequeue match {
|
|
|
|
|
case null ⇒
|
|
|
|
|
case message ⇒ r = r :+ message
|
2011-03-11 16:48:44 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
|
|
|
|
r
|
|
|
|
|
}
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
val consumers = for (i ← (1 to 4).toList) yield createConsumer
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-12-12 22:50:08 +01:00
|
|
|
val ps = producers.map(Await.result(_, within))
|
|
|
|
|
val cs = consumers.map(Await.result(_, within))
|
2011-03-11 16:48:44 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
ps.map(_.size).sum must be === totalMessages //Must have produced 1000 messages
|
|
|
|
|
cs.map(_.size).sum must be === totalMessages //Must have consumed all produced messages
|
|
|
|
|
//No message is allowed to be consumed by more than one consumer
|
|
|
|
|
cs.flatten.distinct.size must be === totalMessages
|
|
|
|
|
//All produced messages should have been consumed
|
|
|
|
|
(cs.flatten diff ps.flatten).size must be === 0
|
|
|
|
|
(ps.flatten diff cs.flatten).size must be === 0
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
}
|
2011-03-11 14:51:24 +01:00
|
|
|
|
|
|
|
|
class DefaultMailboxSpec extends MailboxSpec {
|
|
|
|
|
lazy val name = "The default mailbox implementation"
|
|
|
|
|
def factory = {
|
2012-02-19 10:28:56 +01:00
|
|
|
case u: UnboundedMailbox ⇒ u.create(None)
|
|
|
|
|
case b: BoundedMailbox ⇒ b.create(None)
|
2010-09-09 15:49:59 +02:00
|
|
|
}
|
|
|
|
|
}
|
2011-03-11 14:51:24 +01:00
|
|
|
|
|
|
|
|
class PriorityMailboxSpec extends MailboxSpec {
|
2011-04-11 17:08:10 +02:00
|
|
|
val comparator = PriorityGenerator(_.##)
|
2011-03-11 14:51:24 +01:00
|
|
|
lazy val name = "The priority mailbox implementation"
|
|
|
|
|
def factory = {
|
2012-02-21 17:23:54 +01:00
|
|
|
case UnboundedMailbox() ⇒ new UnboundedPriorityMailbox(comparator).create(None)
|
|
|
|
|
case BoundedMailbox(capacity, pushTimeOut) ⇒ new BoundedPriorityMailbox(comparator, capacity, pushTimeOut).create(None)
|
2011-03-11 14:51:24 +01:00
|
|
|
}
|
2011-06-13 22:36:46 +02:00
|
|
|
}
|
2011-12-19 20:36:06 +01:00
|
|
|
|
|
|
|
|
object CustomMailboxSpec {
|
|
|
|
|
val config = """
|
|
|
|
|
my-dispatcher {
|
2012-02-09 20:40:09 +01:00
|
|
|
mailbox-type = "akka.dispatch.CustomMailboxSpec$MyMailboxType"
|
2011-12-19 20:36:06 +01:00
|
|
|
}
|
|
|
|
|
"""
|
|
|
|
|
|
2012-02-26 21:26:25 +01:00
|
|
|
class MyMailboxType(settings: ActorSystem.Settings, config: Config) extends MailboxType {
|
2012-02-19 10:28:56 +01:00
|
|
|
override def create(owner: Option[ActorContext]) = owner match {
|
|
|
|
|
case Some(o) ⇒ new MyMailbox(o)
|
|
|
|
|
case None ⇒ throw new Exception("no mailbox owner given")
|
|
|
|
|
}
|
2011-12-21 19:37:18 +01:00
|
|
|
}
|
|
|
|
|
|
2012-02-19 10:28:56 +01:00
|
|
|
class MyMailbox(owner: ActorContext) extends QueueBasedMessageQueue with UnboundedMessageQueueSemantics {
|
2011-12-19 20:36:06 +01:00
|
|
|
final val queue = new ConcurrentLinkedQueue[Envelope]()
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
@org.junit.runner.RunWith(classOf[org.scalatest.junit.JUnitRunner])
|
|
|
|
|
class CustomMailboxSpec extends AkkaSpec(CustomMailboxSpec.config) {
|
|
|
|
|
"Dispatcher configuration" must {
|
|
|
|
|
"support custom mailboxType" in {
|
2012-02-19 10:28:56 +01:00
|
|
|
val actor = system.actorOf(Props.empty.withDispatcher("my-dispatcher"))
|
|
|
|
|
val queue = actor.asInstanceOf[LocalActorRef].underlying.mailbox.messageQueue
|
|
|
|
|
queue.getClass must be(classOf[CustomMailboxSpec.MyMailbox])
|
2011-12-19 20:36:06 +01:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|