fix warnings in contrib, docs, osgi, persistence and slf4j
This commit is contained in:
parent
82b8238a9c
commit
a029a90502
17 changed files with 116 additions and 89 deletions
|
|
@ -191,7 +191,7 @@ object DistributedPubSubMediator {
|
||||||
|
|
||||||
def business: Receive
|
def business: Receive
|
||||||
|
|
||||||
def receive = business orElse defaultReceive
|
def receive = business.orElse[Any, Unit](defaultReceive)
|
||||||
|
|
||||||
def remove(ref: ActorRef): Unit = {
|
def remove(ref: ActorRef): Unit = {
|
||||||
if (subscribers.contains(ref))
|
if (subscribers.contains(ref))
|
||||||
|
|
|
||||||
|
|
@ -38,7 +38,7 @@ trait ReceivePipeline extends Actor {
|
||||||
val around = aroundCache match {
|
val around = aroundCache match {
|
||||||
case Some((`receive`, cached)) ⇒ cached
|
case Some((`receive`, cached)) ⇒ cached
|
||||||
case _ ⇒
|
case _ ⇒
|
||||||
val zipped = pipeline.foldRight(receive)((outer, inner) ⇒ outer(inner) orElse inner)
|
val zipped = pipeline.foldRight[Receive](receive)((outer, inner) ⇒ outer(inner).orElse[Any, Unit](inner))
|
||||||
aroundCache = Some((receive, zipped))
|
aroundCache = Some((receive, zipped))
|
||||||
zipped
|
zipped
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -2,49 +2,60 @@ package akka.contrib.pattern
|
||||||
|
|
||||||
import akka.testkit.{ ImplicitSender, AkkaSpec }
|
import akka.testkit.{ ImplicitSender, AkkaSpec }
|
||||||
import akka.actor.{ Actor, Props }
|
import akka.actor.{ Actor, Props }
|
||||||
|
import scala.concurrent.duration._
|
||||||
|
|
||||||
class ReplierActor extends Actor with ReceivePipeline {
|
object ReceivePipelineSpec {
|
||||||
def receive: Actor.Receive = becomeAndReply
|
|
||||||
def becomeAndReply: Actor.Receive = {
|
class ReplierActor extends Actor with ReceivePipeline {
|
||||||
case "become" ⇒ context.become(justReply)
|
def receive: Actor.Receive = becomeAndReply
|
||||||
case m ⇒ sender ! m
|
def becomeAndReply: Actor.Receive = {
|
||||||
|
case "become" ⇒ context.become(justReply)
|
||||||
|
case m ⇒ sender ! m
|
||||||
|
}
|
||||||
|
def justReply: Actor.Receive = {
|
||||||
|
case m ⇒ sender ! m
|
||||||
|
}
|
||||||
}
|
}
|
||||||
def justReply: Actor.Receive = {
|
|
||||||
case m ⇒ sender ! m
|
case class IntList(l: List[Int]) {
|
||||||
|
override def toString: String = s"IntList(${l.mkString(", ")})"
|
||||||
}
|
}
|
||||||
}
|
|
||||||
|
|
||||||
trait ListBuilderInterceptor {
|
trait ListBuilderInterceptor {
|
||||||
this: ReceivePipeline ⇒
|
this: ReceivePipeline ⇒
|
||||||
|
|
||||||
pipelineOuter(inner ⇒
|
pipelineOuter(inner ⇒
|
||||||
{
|
{
|
||||||
case n: Int ⇒ inner((n until n + 3).toList)
|
case n: Int ⇒ inner(IntList((n until n + 3).toList))
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
|
||||||
trait AdderInterceptor {
|
trait AdderInterceptor {
|
||||||
this: ReceivePipeline ⇒
|
this: ReceivePipeline ⇒
|
||||||
|
|
||||||
pipelineInner(inner ⇒
|
pipelineInner(inner ⇒
|
||||||
{
|
{
|
||||||
case n: Int ⇒ inner(n + 10)
|
case n: Int ⇒ inner(n + 10)
|
||||||
case l: List[Int] ⇒ inner(l.map(_ + 10))
|
case IntList(l) ⇒ inner(IntList(l.map(_ + 10)))
|
||||||
case "explicitly ignored" ⇒
|
case "explicitly ignored" ⇒
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
|
||||||
trait ToStringInterceptor {
|
trait ToStringInterceptor {
|
||||||
this: ReceivePipeline ⇒
|
this: ReceivePipeline ⇒
|
||||||
|
|
||||||
|
pipelineInner(inner ⇒
|
||||||
|
{
|
||||||
|
case i: Int ⇒ inner(i.toString)
|
||||||
|
case IntList(l) ⇒ inner(l.toString)
|
||||||
|
case other: Iterable[_] ⇒ inner(other.toString)
|
||||||
|
})
|
||||||
|
}
|
||||||
|
|
||||||
pipelineInner(inner ⇒
|
|
||||||
{
|
|
||||||
case i: Int ⇒ inner(i.toString)
|
|
||||||
case l: Iterable[_] ⇒ inner(l.toString())
|
|
||||||
})
|
|
||||||
}
|
}
|
||||||
|
|
||||||
class ReceivePipelineSpec extends AkkaSpec with ImplicitSender {
|
class ReceivePipelineSpec extends AkkaSpec with ImplicitSender {
|
||||||
|
import ReceivePipelineSpec._
|
||||||
|
|
||||||
"A ReceivePipeline" must {
|
"A ReceivePipeline" must {
|
||||||
|
|
||||||
|
|
@ -82,7 +93,8 @@ class ReceivePipelineSpec extends AkkaSpec with ImplicitSender {
|
||||||
val replier = system.actorOf(Props(
|
val replier = system.actorOf(Props(
|
||||||
new ReplierActor with ListBuilderInterceptor with AdderInterceptor with ToStringInterceptor))
|
new ReplierActor with ListBuilderInterceptor with AdderInterceptor with ToStringInterceptor))
|
||||||
replier ! "explicitly ignored"
|
replier ! "explicitly ignored"
|
||||||
expectNoMsg()
|
replier ! 8L // unhandled by all interceptors but still replied
|
||||||
|
expectMsg(8L)
|
||||||
}
|
}
|
||||||
|
|
||||||
"support changing behavior without losing the interceptions" in {
|
"support changing behavior without losing the interceptions" in {
|
||||||
|
|
@ -101,9 +113,9 @@ class ReceivePipelineSpec extends AkkaSpec with ImplicitSender {
|
||||||
val innerOuterReplier = system.actorOf(Props(
|
val innerOuterReplier = system.actorOf(Props(
|
||||||
new ReplierActor with AdderInterceptor with ListBuilderInterceptor))
|
new ReplierActor with AdderInterceptor with ListBuilderInterceptor))
|
||||||
outerInnerReplier ! 4
|
outerInnerReplier ! 4
|
||||||
expectMsg(List(14, 15, 16))
|
expectMsg(IntList(List(14, 15, 16)))
|
||||||
innerOuterReplier ! 6
|
innerOuterReplier ! 6
|
||||||
expectMsg(List(16, 17, 18))
|
expectMsg(IntList(List(16, 17, 18)))
|
||||||
}
|
}
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
@ -231,7 +243,7 @@ object MixinSample extends App {
|
||||||
// The Dude says 'Yeah, well, you know, that's just, like, your opinion, man.'
|
// The Dude says 'Yeah, well, you know, that's just, like, your opinion, man.'
|
||||||
//#mixin-actor
|
//#mixin-actor
|
||||||
|
|
||||||
system.shutdown()
|
system.terminate()
|
||||||
}
|
}
|
||||||
|
|
||||||
object UnhandledSample extends App {
|
object UnhandledSample extends App {
|
||||||
|
|
|
||||||
|
|
@ -21,6 +21,9 @@ class MyActor extends Actor {
|
||||||
case BarMessage(bar) => sender() ! BazMessage("Got " + bar)
|
case BarMessage(bar) => sender() ! BazMessage("Got " + bar)
|
||||||
// warning here:
|
// warning here:
|
||||||
// "match may not be exhaustive. It would fail on the following input: FooMessage(_)"
|
// "match may not be exhaustive. It would fail on the following input: FooMessage(_)"
|
||||||
|
//#exhaustiveness-check
|
||||||
|
case FooMessage(_) => // avoid the warning in our build logs
|
||||||
|
//#exhaustiveness-check
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -554,6 +554,7 @@ public class FutureDocTest {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test(expected = IllegalStateException.class)
|
@Test(expected = IllegalStateException.class)
|
||||||
|
@SuppressWarnings("unchecked")
|
||||||
public void useAfter() throws Exception {
|
public void useAfter() throws Exception {
|
||||||
//#after
|
//#after
|
||||||
final ExecutionContext ec = system.dispatcher();
|
final ExecutionContext ec = system.dispatcher();
|
||||||
|
|
|
||||||
|
|
@ -92,7 +92,7 @@ class ActorWithMessagesWrapper {
|
||||||
import MyActor._
|
import MyActor._
|
||||||
def receive = {
|
def receive = {
|
||||||
case Greeting(greeter) => log.info(s"I was greeted by $greeter.")
|
case Greeting(greeter) => log.info(s"I was greeted by $greeter.")
|
||||||
case Goodbye => log.info("Someone said goodbye to me.")
|
case Goodbye => log.info("Someone said goodbye to me.")
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
//#messages-in-companion
|
//#messages-in-companion
|
||||||
|
|
@ -229,7 +229,7 @@ class Consumer extends Actor with ActorLogging with ConsumerBehavior {
|
||||||
class ProducerConsumer extends Actor with ActorLogging
|
class ProducerConsumer extends Actor with ActorLogging
|
||||||
with ProducerBehavior with ConsumerBehavior {
|
with ProducerBehavior with ConsumerBehavior {
|
||||||
|
|
||||||
def receive = producerBehavior orElse consumerBehavior
|
def receive = producerBehavior.orElse[Any, Unit](consumerBehavior)
|
||||||
}
|
}
|
||||||
|
|
||||||
// protocol
|
// protocol
|
||||||
|
|
|
||||||
|
|
@ -12,13 +12,10 @@ import akka.util.ByteString
|
||||||
import akka.actor.Props
|
import akka.actor.Props
|
||||||
import scala.collection.immutable
|
import scala.collection.immutable
|
||||||
|
|
||||||
class FSMDocSpec extends MyFavoriteTestFrameWorkPlusAkkaTestKit {
|
object FSMDocSpec {
|
||||||
|
// messages and data types
|
||||||
//#fsm-code-elided
|
//#test-code
|
||||||
//#simple-imports
|
import akka.actor.ActorRef
|
||||||
import akka.actor.{ ActorRef, FSM }
|
|
||||||
import scala.concurrent.duration._
|
|
||||||
//#simple-imports
|
|
||||||
//#simple-events
|
//#simple-events
|
||||||
// received events
|
// received events
|
||||||
final case class SetTarget(ref: ActorRef)
|
final case class SetTarget(ref: ActorRef)
|
||||||
|
|
@ -38,6 +35,17 @@ class FSMDocSpec extends MyFavoriteTestFrameWorkPlusAkkaTestKit {
|
||||||
case object Uninitialized extends Data
|
case object Uninitialized extends Data
|
||||||
final case class Todo(target: ActorRef, queue: immutable.Seq[Any]) extends Data
|
final case class Todo(target: ActorRef, queue: immutable.Seq[Any]) extends Data
|
||||||
//#simple-state
|
//#simple-state
|
||||||
|
//#test-code
|
||||||
|
}
|
||||||
|
|
||||||
|
class FSMDocSpec extends MyFavoriteTestFrameWorkPlusAkkaTestKit {
|
||||||
|
import FSMDocSpec._
|
||||||
|
|
||||||
|
//#fsm-code-elided
|
||||||
|
//#simple-imports
|
||||||
|
import akka.actor.{ ActorRef, FSM }
|
||||||
|
import scala.concurrent.duration._
|
||||||
|
//#simple-imports
|
||||||
//#simple-fsm
|
//#simple-fsm
|
||||||
class Buncher extends FSM[State, Data] {
|
class Buncher extends FSM[State, Data] {
|
||||||
|
|
||||||
|
|
@ -56,6 +64,7 @@ class FSMDocSpec extends MyFavoriteTestFrameWorkPlusAkkaTestKit {
|
||||||
case Active -> Idle =>
|
case Active -> Idle =>
|
||||||
stateData match {
|
stateData match {
|
||||||
case Todo(ref, queue) => ref ! Batch(queue)
|
case Todo(ref, queue) => ref ! Batch(queue)
|
||||||
|
case _ => // nothing to do
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
//#transition-elided
|
//#transition-elided
|
||||||
|
|
|
||||||
|
|
@ -113,7 +113,8 @@ class Worker extends Actor with ActorLogging {
|
||||||
//#messages
|
//#messages
|
||||||
object CounterService {
|
object CounterService {
|
||||||
final case class Increment(n: Int)
|
final case class Increment(n: Int)
|
||||||
case object GetCurrentCount
|
sealed abstract class GetCurrentCount
|
||||||
|
case object GetCurrentCount extends GetCurrentCount
|
||||||
final case class CurrentCount(key: String, count: Long)
|
final case class CurrentCount(key: String, count: Long)
|
||||||
class ServiceUnavailable(msg: String) extends RuntimeException(msg)
|
class ServiceUnavailable(msg: String) extends RuntimeException(msg)
|
||||||
|
|
||||||
|
|
@ -176,9 +177,9 @@ class CounterService extends Actor {
|
||||||
for ((replyTo, msg) <- backlog) c.tell(msg, sender = replyTo)
|
for ((replyTo, msg) <- backlog) c.tell(msg, sender = replyTo)
|
||||||
backlog = IndexedSeq.empty
|
backlog = IndexedSeq.empty
|
||||||
|
|
||||||
case msg @ Increment(n) => forwardOrPlaceInBacklog(msg)
|
case msg: Increment => forwardOrPlaceInBacklog(msg)
|
||||||
|
|
||||||
case msg @ GetCurrentCount => forwardOrPlaceInBacklog(msg)
|
case msg: GetCurrentCount => forwardOrPlaceInBacklog(msg)
|
||||||
|
|
||||||
case Terminated(actorRef) if Some(actorRef) == storage =>
|
case Terminated(actorRef) if Some(actorRef) == storage =>
|
||||||
// After 3 restarts the storage child is stopped.
|
// After 3 restarts the storage child is stopped.
|
||||||
|
|
|
||||||
|
|
@ -108,7 +108,7 @@ class FaultHandlingDocSpec extends TestKit(ActorSystem("FaultHandlingDocSpec", t
|
||||||
}
|
}
|
||||||
|
|
||||||
"A supervisor" must "apply the chosen strategy for its child" in {
|
"A supervisor" must "apply the chosen strategy for its child" in {
|
||||||
//#testkit
|
//#testkit
|
||||||
|
|
||||||
//#create
|
//#create
|
||||||
val supervisor = system.actorOf(Props[Supervisor], "supervisor")
|
val supervisor = system.actorOf(Props[Supervisor], "supervisor")
|
||||||
|
|
|
||||||
|
|
@ -143,7 +143,7 @@ class TypedActorDocSpec extends AkkaSpec(Map("akka.loglevel" -> "INFO")) {
|
||||||
//#typed-actor-call-strict
|
//#typed-actor-call-strict
|
||||||
//#typed-actor-calls
|
//#typed-actor-calls
|
||||||
|
|
||||||
Await.result(fSquare, 3 seconds) should be(100)
|
Await.result(fSquare, 3.seconds) should be(100)
|
||||||
|
|
||||||
oSquare should be(Some(100))
|
oSquare should be(Some(100))
|
||||||
|
|
||||||
|
|
@ -193,7 +193,7 @@ class TypedActorDocSpec extends AkkaSpec(Map("akka.loglevel" -> "INFO")) {
|
||||||
|
|
||||||
TypedActor(system).poisonPill(awesomeFooBar)
|
TypedActor(system).poisonPill(awesomeFooBar)
|
||||||
//#typed-actor-supercharge-usage
|
//#typed-actor-supercharge-usage
|
||||||
Await.result(f, 3 seconds) should be("YES")
|
Await.result(f, 3.seconds) should be("YES")
|
||||||
}
|
}
|
||||||
|
|
||||||
"typed router pattern" in {
|
"typed router pattern" in {
|
||||||
|
|
|
||||||
|
|
@ -70,18 +70,19 @@ class EchoManager(handlerClass: Class[_]) extends Actor with ActorLogging {
|
||||||
|
|
||||||
}
|
}
|
||||||
|
|
||||||
|
//#echo-handler
|
||||||
object EchoHandler {
|
object EchoHandler {
|
||||||
|
final case class Ack(offset: Int) extends Tcp.Event
|
||||||
|
|
||||||
def props(connection: ActorRef, remote: InetSocketAddress): Props =
|
def props(connection: ActorRef, remote: InetSocketAddress): Props =
|
||||||
Props(classOf[EchoHandler], connection, remote)
|
Props(classOf[EchoHandler], connection, remote)
|
||||||
}
|
}
|
||||||
|
|
||||||
//#echo-handler
|
|
||||||
class EchoHandler(connection: ActorRef, remote: InetSocketAddress)
|
class EchoHandler(connection: ActorRef, remote: InetSocketAddress)
|
||||||
extends Actor with ActorLogging {
|
extends Actor with ActorLogging {
|
||||||
|
|
||||||
import Tcp._
|
import Tcp._
|
||||||
|
import EchoHandler._
|
||||||
final case class Ack(offset: Int) extends Event
|
|
||||||
|
|
||||||
// sign death pact: this actor terminates when connection breaks
|
// sign death pact: this actor terminates when connection breaks
|
||||||
context watch connection
|
context watch connection
|
||||||
|
|
|
||||||
|
|
@ -87,8 +87,9 @@ object ScalaUdpDocSpec {
|
||||||
//#connected
|
//#connected
|
||||||
case msg: String =>
|
case msg: String =>
|
||||||
connection ! UdpConnected.Send(ByteString(msg))
|
connection ! UdpConnected.Send(ByteString(msg))
|
||||||
case d @ UdpConnected.Disconnect => connection ! d
|
case UdpConnected.Disconnect =>
|
||||||
case UdpConnected.Disconnected => context.stop(self)
|
connection ! UdpConnected.Disconnect
|
||||||
|
case UdpConnected.Disconnected => context.stop(self)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
//#connected
|
//#connected
|
||||||
|
|
|
||||||
|
|
@ -4,13 +4,19 @@
|
||||||
|
|
||||||
package docs.persistence
|
package docs.persistence
|
||||||
|
|
||||||
import akka.actor.{ Actor, ActorSystem, Props }
|
import akka.actor.{ Actor, ActorRef, ActorSystem, Props }
|
||||||
import akka.persistence._
|
import akka.persistence._
|
||||||
import com.typesafe.config.ConfigFactory
|
import com.typesafe.config.ConfigFactory
|
||||||
|
|
||||||
import scala.concurrent.duration._
|
import scala.concurrent.duration._
|
||||||
import scala.language.postfixOps
|
import scala.language.postfixOps
|
||||||
trait PersistenceDocSpec {
|
|
||||||
|
object PersistenceDocSpec {
|
||||||
|
|
||||||
|
trait SomeOtherMessage
|
||||||
|
|
||||||
|
val persistentActor: ActorRef = ???
|
||||||
|
|
||||||
val config =
|
val config =
|
||||||
"""
|
"""
|
||||||
//#auto-update-interval
|
//#auto-update-interval
|
||||||
|
|
@ -21,13 +27,7 @@ trait PersistenceDocSpec {
|
||||||
//#auto-update
|
//#auto-update
|
||||||
"""
|
"""
|
||||||
|
|
||||||
trait SomeOtherMessage
|
object Recovery {
|
||||||
|
|
||||||
implicit val system: ActorSystem
|
|
||||||
|
|
||||||
import system._
|
|
||||||
|
|
||||||
new AnyRef {
|
|
||||||
trait MyPersistentActor1 extends PersistentActor {
|
trait MyPersistentActor1 extends PersistentActor {
|
||||||
//#recover-on-start-disabled
|
//#recover-on-start-disabled
|
||||||
override def preStart() = ()
|
override def preStart() = ()
|
||||||
|
|
@ -45,7 +45,6 @@ trait PersistenceDocSpec {
|
||||||
//#recover-on-start-custom
|
//#recover-on-start-custom
|
||||||
}
|
}
|
||||||
|
|
||||||
val persistentActor = system.deadLetters
|
|
||||||
//#recover-explicit
|
//#recover-explicit
|
||||||
persistentActor ! Recover()
|
persistentActor ! Recover()
|
||||||
//#recover-explicit
|
//#recover-explicit
|
||||||
|
|
@ -69,7 +68,7 @@ trait PersistenceDocSpec {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object NoRecovery {
|
||||||
trait MyPersistentActor1 extends PersistentActor {
|
trait MyPersistentActor1 extends PersistentActor {
|
||||||
//#recover-fully-disabled
|
//#recover-fully-disabled
|
||||||
override def preStart() = self ! Recover(toSequenceNr = 0L)
|
override def preStart() = self ! Recover(toSequenceNr = 0L)
|
||||||
|
|
@ -77,7 +76,7 @@ trait PersistenceDocSpec {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object PersistenceId {
|
||||||
trait PersistentActorMethods {
|
trait PersistentActorMethods {
|
||||||
//#persistence-id
|
//#persistence-id
|
||||||
def persistenceId: String
|
def persistenceId: String
|
||||||
|
|
@ -101,7 +100,7 @@ trait PersistenceDocSpec {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object AtLeastOnce {
|
||||||
//#at-least-once-example
|
//#at-least-once-example
|
||||||
import akka.actor.{ Actor, ActorPath }
|
import akka.actor.{ Actor, ActorPath }
|
||||||
import akka.persistence.AtLeastOnceDelivery
|
import akka.persistence.AtLeastOnceDelivery
|
||||||
|
|
@ -145,7 +144,7 @@ trait PersistenceDocSpec {
|
||||||
//#at-least-once-example
|
//#at-least-once-example
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object SaveSnapshot {
|
||||||
|
|
||||||
class MyPersistentActor extends PersistentActor {
|
class MyPersistentActor extends PersistentActor {
|
||||||
override def persistenceId = "my-stable-persistence-id"
|
override def persistenceId = "my-stable-persistence-id"
|
||||||
|
|
@ -164,7 +163,7 @@ trait PersistenceDocSpec {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object OfferSnapshot {
|
||||||
class MyPersistentActor extends PersistentActor {
|
class MyPersistentActor extends PersistentActor {
|
||||||
override def persistenceId = "my-stable-persistence-id"
|
override def persistenceId = "my-stable-persistence-id"
|
||||||
|
|
||||||
|
|
@ -183,8 +182,6 @@ trait PersistenceDocSpec {
|
||||||
|
|
||||||
import akka.actor.Props
|
import akka.actor.Props
|
||||||
|
|
||||||
val persistentActor = system.actorOf(Props[MyPersistentActor])
|
|
||||||
|
|
||||||
//#snapshot-criteria
|
//#snapshot-criteria
|
||||||
persistentActor ! Recover(fromSnapshot = SnapshotSelectionCriteria(
|
persistentActor ! Recover(fromSnapshot = SnapshotSelectionCriteria(
|
||||||
maxSequenceNr = 457L,
|
maxSequenceNr = 457L,
|
||||||
|
|
@ -192,9 +189,7 @@ trait PersistenceDocSpec {
|
||||||
//#snapshot-criteria
|
//#snapshot-criteria
|
||||||
}
|
}
|
||||||
|
|
||||||
new AnyRef {
|
object PersistAsync {
|
||||||
|
|
||||||
val persistentActor = system.actorOf(Props[MyPersistentActor]())
|
|
||||||
|
|
||||||
//#persist-async
|
//#persist-async
|
||||||
class MyPersistentActor extends PersistentActor {
|
class MyPersistentActor extends PersistentActor {
|
||||||
|
|
@ -228,9 +223,8 @@ trait PersistenceDocSpec {
|
||||||
|
|
||||||
//#persist-async
|
//#persist-async
|
||||||
}
|
}
|
||||||
new AnyRef {
|
|
||||||
|
|
||||||
val persistentActor = system.actorOf(Props[MyPersistentActor]())
|
object Defer {
|
||||||
|
|
||||||
//#defer
|
//#defer
|
||||||
class MyPersistentActor extends PersistentActor {
|
class MyPersistentActor extends PersistentActor {
|
||||||
|
|
@ -268,9 +262,12 @@ trait PersistenceDocSpec {
|
||||||
|
|
||||||
//#defer-caller
|
//#defer-caller
|
||||||
}
|
}
|
||||||
new AnyRef {
|
|
||||||
|
object View {
|
||||||
import akka.actor.Props
|
import akka.actor.Props
|
||||||
|
|
||||||
|
val system: ActorSystem = ???
|
||||||
|
|
||||||
//#view
|
//#view
|
||||||
class MyView extends PersistentView {
|
class MyView extends PersistentView {
|
||||||
override def persistenceId: String = "some-persistence-id"
|
override def persistenceId: String = "some-persistence-id"
|
||||||
|
|
|
||||||
|
|
@ -17,7 +17,7 @@ class DefaultOSGiLogger extends DefaultLogger {
|
||||||
|
|
||||||
val messageFormat = " %s | %s | %s | %s"
|
val messageFormat = " %s | %s | %s | %s"
|
||||||
|
|
||||||
override def receive: Receive = uninitialisedReceive orElse super.receive
|
override def receive: Receive = uninitialisedReceive.orElse[Any, Unit](super.receive)
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Behaviour of the logger that waits for its LogService
|
* Behaviour of the logger that waits for its LogService
|
||||||
|
|
|
||||||
|
|
@ -14,8 +14,8 @@ object SnapshotSpec {
|
||||||
var state = List.empty[String]
|
var state = List.empty[String]
|
||||||
|
|
||||||
override def receiveRecover: Receive = {
|
override def receiveRecover: Receive = {
|
||||||
case payload: String ⇒ state = s"${payload}-${lastSequenceNr}" :: state
|
case payload: String ⇒ state = s"${payload}-${lastSequenceNr}" :: state
|
||||||
case SnapshotOffer(_, snapshot: List[String]) ⇒ state = snapshot
|
case SnapshotOffer(_, snapshot: List[_]) ⇒ state = snapshot.asInstanceOf[List[String]]
|
||||||
}
|
}
|
||||||
|
|
||||||
override def receiveCommand = {
|
override def receiveCommand = {
|
||||||
|
|
|
||||||
|
|
@ -26,6 +26,8 @@ object Slf4jLoggerSpec {
|
||||||
}
|
}
|
||||||
"""
|
"""
|
||||||
|
|
||||||
|
case class StringWithMDC(s: String, mdc: Map[String, Any])
|
||||||
|
|
||||||
class LogProducer extends Actor with DiagnosticActorLogging {
|
class LogProducer extends Actor with DiagnosticActorLogging {
|
||||||
|
|
||||||
def receive = {
|
def receive = {
|
||||||
|
|
@ -33,7 +35,7 @@ object Slf4jLoggerSpec {
|
||||||
log.error(e, e.getMessage)
|
log.error(e, e.getMessage)
|
||||||
case (s: String, x: Int, y: Int) ⇒
|
case (s: String, x: Int, y: Int) ⇒
|
||||||
log.info(s, x, y)
|
log.info(s, x, y)
|
||||||
case (s: String, mdc: Map[String, Any]) ⇒
|
case StringWithMDC(s, mdc) ⇒
|
||||||
log.mdc(mdc)
|
log.mdc(mdc)
|
||||||
log.info(s)
|
log.info(s)
|
||||||
log.clearMDC()
|
log.clearMDC()
|
||||||
|
|
@ -96,7 +98,7 @@ class Slf4jLoggerSpec extends AkkaSpec(Slf4jLoggerSpec.config) with BeforeAndAft
|
||||||
}
|
}
|
||||||
|
|
||||||
"put custom MDC values when specified" in {
|
"put custom MDC values when specified" in {
|
||||||
producer ! ("Message with custom MDC values", Map("ticketNumber" -> 3671, "ticketDesc" -> "Custom MDC Values"))
|
producer ! StringWithMDC("Message with custom MDC values", Map("ticketNumber" -> 3671, "ticketDesc" -> "Custom MDC Values"))
|
||||||
|
|
||||||
awaitCond(outputString.contains("----"), 5 seconds)
|
awaitCond(outputString.contains("----"), 5 seconds)
|
||||||
val s = outputString
|
val s = outputString
|
||||||
|
|
@ -109,7 +111,7 @@ class Slf4jLoggerSpec extends AkkaSpec(Slf4jLoggerSpec.config) with BeforeAndAft
|
||||||
}
|
}
|
||||||
|
|
||||||
"Support null values in custom MDC" in {
|
"Support null values in custom MDC" in {
|
||||||
producer ! ("Message with null custom MDC values", Map("ticketNumber" -> 3671, "ticketDesc" -> null))
|
producer ! StringWithMDC("Message with null custom MDC values", Map("ticketNumber" -> 3671, "ticketDesc" -> null))
|
||||||
|
|
||||||
awaitCond(outputString.contains("----"), 5 seconds)
|
awaitCond(outputString.contains("----"), 5 seconds)
|
||||||
val s = outputString
|
val s = outputString
|
||||||
|
|
|
||||||
|
|
@ -287,17 +287,17 @@ object AkkaBuild extends Build {
|
||||||
pomIncludeRepository := (_ => false) // do not leak internal repositories during staging
|
pomIncludeRepository := (_ => false) // do not leak internal repositories during staging
|
||||||
)
|
)
|
||||||
|
|
||||||
private def deprecation: Boolean = System.getProperty("akka.deprecation", "false").toBoolean
|
private def allWarnings: Boolean = System.getProperty("akka.allwarnings", "false").toBoolean
|
||||||
|
|
||||||
lazy val defaultSettings = baseSettings ++ resolverSettings ++ TestExtras.Filter.settings ++
|
lazy val defaultSettings = baseSettings ++ resolverSettings ++ TestExtras.Filter.settings ++
|
||||||
Protobuf.settings ++ Seq(
|
Protobuf.settings ++ Seq(
|
||||||
// compile options
|
// compile options
|
||||||
scalacOptions in Compile ++= Seq("-encoding", "UTF-8", "-target:jvm-1.6", "-feature", "-unchecked", "-Xlog-reflective-calls", "-Xlint"),
|
scalacOptions in Compile ++= Seq("-encoding", "UTF-8", "-target:jvm-1.6", "-feature", "-unchecked", "-Xlog-reflective-calls", "-Xlint"),
|
||||||
scalacOptions in Compile ++= (if (deprecation) Seq("-deprecation") else Nil),
|
scalacOptions in Compile ++= (if (allWarnings) Seq("-deprecation") else Nil),
|
||||||
scalacOptions in Test := (scalacOptions in Test).value.filterNot(_ == "-Xlog-reflective-calls"),
|
scalacOptions in Test := (scalacOptions in Test).value.filterNot(_ == "-Xlog-reflective-calls"),
|
||||||
// -XDignore.symbol.file suppresses sun.misc.Unsafe warnings
|
// -XDignore.symbol.file suppresses sun.misc.Unsafe warnings
|
||||||
javacOptions in compile ++= Seq("-encoding", "UTF-8", "-source", "1.6", "-target", "1.6", "-Xlint:unchecked", "-XDignore.symbol.file"),
|
javacOptions in compile ++= Seq("-encoding", "UTF-8", "-source", "1.6", "-target", "1.6", "-Xlint:unchecked", "-XDignore.symbol.file"),
|
||||||
javacOptions in compile ++= (if (deprecation) Seq("-Xlint:deprecation") else Nil),
|
javacOptions in compile ++= (if (allWarnings) Seq("-Xlint:deprecation") else Nil),
|
||||||
javacOptions in doc ++= Seq("-encoding", "UTF-8", "-source", "1.6"),
|
javacOptions in doc ++= Seq("-encoding", "UTF-8", "-source", "1.6"),
|
||||||
incOptions := incOptions.value.withNameHashing(true),
|
incOptions := incOptions.value.withNameHashing(true),
|
||||||
|
|
||||||
|
|
|
||||||
Loading…
Add table
Add a link
Reference in a new issue