2012-05-25 12:10:37 +02:00
|
|
|
/**
|
2013-01-09 01:47:48 +01:00
|
|
|
* Copyright (C) 2009-2013 Typesafe Inc. <http://www.typesafe.com>
|
2012-05-25 12:10:37 +02:00
|
|
|
*/
|
|
|
|
|
package akka.cluster
|
|
|
|
|
|
2012-07-26 14:47:21 +02:00
|
|
|
import language.implicitConversions
|
2012-10-30 15:08:41 +01:00
|
|
|
|
|
|
|
|
import org.scalatest.Suite
|
|
|
|
|
import org.scalatest.exceptions.TestFailedException
|
|
|
|
|
|
2012-05-25 12:10:37 +02:00
|
|
|
import com.typesafe.config.Config
|
|
|
|
|
import com.typesafe.config.ConfigFactory
|
|
|
|
|
import akka.remote.testconductor.RoleName
|
2012-09-20 15:23:18 +02:00
|
|
|
import akka.remote.testkit.{ STMultiNodeSpec, MultiNodeSpec }
|
2012-05-28 11:06:02 +02:00
|
|
|
import akka.testkit._
|
2012-10-01 20:08:21 +02:00
|
|
|
import akka.testkit.TestEvent._
|
2012-10-30 15:08:41 +01:00
|
|
|
import akka.actor.{ ActorSystem, Address }
|
|
|
|
|
import akka.event.Logging.ErrorLevel
|
2012-09-21 14:50:06 +02:00
|
|
|
import scala.concurrent.duration._
|
2012-10-30 15:08:41 +01:00
|
|
|
import scala.collection.immutable
|
2012-06-15 17:32:40 +02:00
|
|
|
import java.util.concurrent.ConcurrentHashMap
|
2012-05-25 12:10:37 +02:00
|
|
|
|
|
|
|
|
object MultiNodeClusterSpec {
|
2012-09-06 21:48:40 +02:00
|
|
|
|
|
|
|
|
def clusterConfigWithFailureDetectorPuppet: Config =
|
|
|
|
|
ConfigFactory.parseString("akka.cluster.failure-detector.implementation-class = akka.cluster.FailureDetectorPuppet").
|
|
|
|
|
withFallback(clusterConfig)
|
|
|
|
|
|
|
|
|
|
def clusterConfig(failureDetectorPuppet: Boolean): Config =
|
|
|
|
|
if (failureDetectorPuppet) clusterConfigWithFailureDetectorPuppet else clusterConfig
|
|
|
|
|
|
2012-05-25 12:10:37 +02:00
|
|
|
def clusterConfig: Config = ConfigFactory.parseString("""
|
2012-08-28 08:36:14 +02:00
|
|
|
akka.actor.provider = akka.cluster.ClusterActorRefProvider
|
2012-05-25 12:10:37 +02:00
|
|
|
akka.cluster {
|
2012-10-01 20:08:21 +02:00
|
|
|
auto-join = off
|
2012-09-20 15:23:18 +02:00
|
|
|
auto-down = off
|
|
|
|
|
jmx.enabled = off
|
|
|
|
|
gossip-interval = 200 ms
|
|
|
|
|
leader-actions-interval = 200 ms
|
|
|
|
|
unreachable-nodes-reaper-interval = 200 ms
|
|
|
|
|
periodic-tasks-initial-delay = 300 ms
|
|
|
|
|
publish-stats-interval = 0 s # always, when it happens
|
|
|
|
|
failure-detector.heartbeat-interval = 400 ms
|
2012-05-25 12:10:37 +02:00
|
|
|
}
|
2012-10-01 20:08:21 +02:00
|
|
|
akka.loglevel = INFO
|
2012-09-06 21:48:40 +02:00
|
|
|
akka.remote.log-remote-lifecycle-events = off
|
2012-10-01 20:08:21 +02:00
|
|
|
akka.event-handlers = ["akka.testkit.TestEventListener"]
|
2012-05-25 12:10:37 +02:00
|
|
|
akka.test {
|
|
|
|
|
single-expect-default = 5 s
|
|
|
|
|
}
|
|
|
|
|
""")
|
|
|
|
|
}
|
|
|
|
|
|
2012-09-12 15:12:13 +02:00
|
|
|
trait MultiNodeClusterSpec extends Suite with STMultiNodeSpec { self: MultiNodeSpec ⇒
|
2012-05-25 12:10:37 +02:00
|
|
|
|
2012-06-08 09:23:36 +02:00
|
|
|
override def initialParticipants = roles.size
|
|
|
|
|
|
2012-06-15 17:32:40 +02:00
|
|
|
private val cachedAddresses = new ConcurrentHashMap[RoleName, Address]
|
|
|
|
|
|
2012-10-01 20:08:21 +02:00
|
|
|
override def atStartup(): Unit = {
|
|
|
|
|
muteLog()
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def muteLog(sys: ActorSystem = system): Unit = {
|
|
|
|
|
if (!sys.log.isDebugEnabled) {
|
|
|
|
|
Seq(".*Metrics collection has started successfully.*",
|
2012-11-07 20:36:24 +01:00
|
|
|
".*Metrics will be retreived from MBeans.*",
|
2012-10-01 20:08:21 +02:00
|
|
|
".*Cluster Node.* - registered cluster JMX MBean.*",
|
|
|
|
|
".*Cluster Node.* - is starting up.*",
|
|
|
|
|
".*Shutting down cluster Node.*",
|
|
|
|
|
".*Cluster node successfully shut down.*",
|
2013-01-15 09:35:07 +01:00
|
|
|
".*Using a dedicated scheduler for cluster.*") foreach { s ⇒
|
2012-10-01 20:08:21 +02:00
|
|
|
sys.eventStream.publish(Mute(EventFilter.info(pattern = s)))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
Seq(".*received dead letter from.*ClientDisconnected",
|
|
|
|
|
".*received dead letter from.*deadLetters.*PoisonPill",
|
|
|
|
|
".*installing context org.jboss.netty.channel.DefaultChannelPipeline.*") foreach { s ⇒
|
|
|
|
|
sys.eventStream.publish(Mute(EventFilter.warning(pattern = s)))
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def muteMarkingAsUnreachable(sys: ActorSystem = system): Unit = if (!sys.log.isDebugEnabled) {
|
|
|
|
|
sys.eventStream.publish(Mute(EventFilter.error(pattern = ".*Marking.* as UNREACHABLE.*")))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def muteDeadLetters(sys: ActorSystem = system): Unit = if (!sys.log.isDebugEnabled) {
|
|
|
|
|
sys.eventStream.publish(Mute(EventFilter.warning(pattern = ".*received dead letter from.*")))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def afterAll(): Unit = {
|
|
|
|
|
if (!log.isDebugEnabled) {
|
|
|
|
|
muteDeadLetters()
|
|
|
|
|
system.eventStream.setLogLevel(ErrorLevel)
|
|
|
|
|
}
|
|
|
|
|
super.afterAll()
|
|
|
|
|
}
|
|
|
|
|
|
2012-06-15 17:32:40 +02:00
|
|
|
/**
|
|
|
|
|
* Lookup the Address for the role.
|
2012-06-18 11:16:30 +02:00
|
|
|
*
|
|
|
|
|
* Implicit conversion from RoleName to Address.
|
|
|
|
|
*
|
2012-06-15 17:32:40 +02:00
|
|
|
* It is cached, which has the implication that stopping
|
|
|
|
|
* and then restarting a role (jvm) with another address is not
|
|
|
|
|
* supported.
|
|
|
|
|
*/
|
2012-06-18 11:16:30 +02:00
|
|
|
implicit def address(role: RoleName): Address = {
|
2012-06-15 17:32:40 +02:00
|
|
|
cachedAddresses.get(role) match {
|
|
|
|
|
case null ⇒
|
|
|
|
|
val address = node(role).address
|
|
|
|
|
cachedAddresses.put(role, address)
|
|
|
|
|
address
|
|
|
|
|
case address ⇒ address
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-06-15 13:32:55 +02:00
|
|
|
// Cluster tests are written so that if previous step (test method) failed
|
|
|
|
|
// it will most likely not be possible to run next step. This ensures
|
|
|
|
|
// fail fast of steps after the first failure.
|
|
|
|
|
private var failed = false
|
|
|
|
|
override protected def withFixture(test: NoArgTest): Unit = try {
|
|
|
|
|
if (failed) {
|
|
|
|
|
val e = new TestFailedException("Previous step failed", 0)
|
|
|
|
|
// short stack trace
|
|
|
|
|
e.setStackTrace(e.getStackTrace.take(1))
|
|
|
|
|
throw e
|
|
|
|
|
}
|
|
|
|
|
super.withFixture(test)
|
|
|
|
|
} catch {
|
2012-07-26 14:47:21 +02:00
|
|
|
case t: Throwable ⇒
|
2012-06-15 13:32:55 +02:00
|
|
|
failed = true
|
|
|
|
|
throw t
|
|
|
|
|
}
|
|
|
|
|
|
2012-08-16 18:28:01 +02:00
|
|
|
def clusterView: ClusterReadView = cluster.readView
|
|
|
|
|
|
2012-06-11 14:32:17 +02:00
|
|
|
/**
|
|
|
|
|
* Get the cluster node to use.
|
|
|
|
|
*/
|
2012-09-06 21:48:40 +02:00
|
|
|
def cluster: Cluster = Cluster(system)
|
2012-05-25 12:10:37 +02:00
|
|
|
|
2012-06-04 23:21:28 +02:00
|
|
|
/**
|
2012-06-25 21:07:44 +02:00
|
|
|
* Use this method for the initial startup of the cluster node.
|
2012-06-04 23:21:28 +02:00
|
|
|
*/
|
2012-06-25 21:07:44 +02:00
|
|
|
def startClusterNode(): Unit = {
|
2012-08-16 18:28:01 +02:00
|
|
|
if (clusterView.members.isEmpty) {
|
2012-06-25 21:07:44 +02:00
|
|
|
cluster join myself
|
2012-08-16 18:28:01 +02:00
|
|
|
awaitCond(clusterView.members.exists(_.address == address(myself)))
|
2012-06-25 21:07:44 +02:00
|
|
|
} else
|
2012-08-16 18:28:01 +02:00
|
|
|
clusterView.self
|
2012-06-25 21:07:44 +02:00
|
|
|
}
|
2012-06-04 23:21:28 +02:00
|
|
|
|
2012-06-05 15:53:30 +02:00
|
|
|
/**
|
|
|
|
|
* Initialize the cluster with the specified member
|
|
|
|
|
* nodes (roles). First node will be started first
|
|
|
|
|
* and others will join the first.
|
|
|
|
|
*/
|
2012-10-30 15:08:41 +01:00
|
|
|
def startCluster(roles: RoleName*): Unit = awaitStartCluster(false, roles.to[immutable.Seq])
|
2012-06-05 14:13:44 +02:00
|
|
|
|
2012-06-05 15:53:30 +02:00
|
|
|
/**
|
|
|
|
|
* Initialize the cluster of the specified member
|
|
|
|
|
* nodes (roles) and wait until all joined and `Up`.
|
|
|
|
|
* First node will be started first and others will join
|
|
|
|
|
* the first.
|
|
|
|
|
*/
|
2012-10-30 15:08:41 +01:00
|
|
|
def awaitClusterUp(roles: RoleName*): Unit = awaitStartCluster(true, roles.to[immutable.Seq])
|
2012-06-05 14:13:44 +02:00
|
|
|
|
2012-10-30 15:08:41 +01:00
|
|
|
private def awaitStartCluster(upConvergence: Boolean = true, roles: immutable.Seq[RoleName]): Unit = {
|
2012-06-05 14:13:44 +02:00
|
|
|
runOn(roles.head) {
|
|
|
|
|
// make sure that the node-to-join is started before other join
|
|
|
|
|
startClusterNode()
|
|
|
|
|
}
|
2012-06-15 14:39:47 +02:00
|
|
|
enterBarrier(roles.head.name + "-started")
|
2012-06-05 14:13:44 +02:00
|
|
|
if (roles.tail.contains(myself)) {
|
2012-06-15 17:32:40 +02:00
|
|
|
cluster.join(roles.head)
|
2012-06-05 14:13:44 +02:00
|
|
|
}
|
|
|
|
|
if (upConvergence && roles.contains(myself)) {
|
|
|
|
|
awaitUpConvergence(numberOfMembers = roles.length)
|
|
|
|
|
}
|
2012-06-15 14:39:47 +02:00
|
|
|
enterBarrier(roles.map(_.name).mkString("-") + "-joined")
|
2012-06-05 14:13:44 +02:00
|
|
|
}
|
|
|
|
|
|
2012-05-25 12:10:37 +02:00
|
|
|
/**
|
|
|
|
|
* Assert that the member addresses match the expected addresses in the
|
|
|
|
|
* sort order used by the cluster.
|
|
|
|
|
*/
|
|
|
|
|
def assertMembers(gotMembers: Iterable[Member], expectedAddresses: Address*): Unit = {
|
|
|
|
|
import Member.addressOrdering
|
|
|
|
|
val members = gotMembers.toIndexedSeq
|
|
|
|
|
members.size must be(expectedAddresses.length)
|
|
|
|
|
expectedAddresses.sorted.zipWithIndex.foreach { case (a, i) ⇒ members(i).address must be(a) }
|
|
|
|
|
}
|
|
|
|
|
|
2012-10-30 15:08:41 +01:00
|
|
|
def assertLeader(nodesInCluster: RoleName*): Unit =
|
|
|
|
|
if (nodesInCluster.contains(myself)) assertLeaderIn(nodesInCluster.to[immutable.Seq])
|
2012-06-01 11:37:44 +02:00
|
|
|
|
2012-05-25 12:10:37 +02:00
|
|
|
/**
|
|
|
|
|
* Assert that the cluster has elected the correct leader
|
|
|
|
|
* out of all nodes in the cluster. First
|
|
|
|
|
* member in the cluster ring is expected leader.
|
|
|
|
|
*/
|
2012-10-30 15:08:41 +01:00
|
|
|
def assertLeaderIn(nodesInCluster: immutable.Seq[RoleName]): Unit = if (nodesInCluster.contains(myself)) {
|
2012-05-25 12:10:37 +02:00
|
|
|
nodesInCluster.length must not be (0)
|
2012-05-25 14:48:00 +02:00
|
|
|
val expectedLeader = roleOfLeader(nodesInCluster)
|
2012-10-19 17:28:20 +02:00
|
|
|
val leader = clusterView.leader
|
|
|
|
|
val isLeader = leader == Some(clusterView.selfAddress)
|
2012-10-31 16:37:03 +01:00
|
|
|
assert(isLeader == isNode(expectedLeader),
|
2012-10-19 17:28:20 +02:00
|
|
|
"expectedLeader [%s], got leader [%s], members [%s]".format(expectedLeader, leader, clusterView.members))
|
2012-08-16 18:28:01 +02:00
|
|
|
clusterView.status must (be(MemberStatus.Up) or be(MemberStatus.Leaving))
|
2012-05-25 12:10:37 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2012-05-28 11:06:02 +02:00
|
|
|
* Wait until the expected number of members has status Up and convergence has been reached.
|
|
|
|
|
* Also asserts that nodes in the 'canNotBePartOfMemberRing' are *not* part of the cluster ring.
|
2012-05-25 12:10:37 +02:00
|
|
|
*/
|
2012-05-28 11:06:02 +02:00
|
|
|
def awaitUpConvergence(
|
|
|
|
|
numberOfMembers: Int,
|
2012-12-12 11:49:20 +01:00
|
|
|
canNotBePartOfMemberRing: Set[Address] = Set.empty,
|
2012-09-18 09:58:30 +02:00
|
|
|
timeout: FiniteDuration = 20.seconds): Unit = {
|
2012-05-30 17:17:09 +02:00
|
|
|
within(timeout) {
|
2012-12-12 11:49:20 +01:00
|
|
|
if (!canNotBePartOfMemberRing.isEmpty) // don't run this on an empty set
|
|
|
|
|
awaitCond(
|
|
|
|
|
canNotBePartOfMemberRing forall (address ⇒ !(clusterView.members exists (_.address == address))))
|
2012-08-16 18:28:01 +02:00
|
|
|
awaitCond(clusterView.members.size == numberOfMembers)
|
|
|
|
|
awaitCond(clusterView.members.forall(_.status == MemberStatus.Up))
|
2012-12-04 17:07:08 +01:00
|
|
|
// clusterView.leader is updated by LeaderChanged, await that to be updated also
|
|
|
|
|
val expectedLeader = clusterView.members.headOption.map(_.address)
|
|
|
|
|
awaitCond(clusterView.leader == expectedLeader)
|
2012-05-30 17:17:09 +02:00
|
|
|
}
|
2012-05-25 12:10:37 +02:00
|
|
|
}
|
2012-05-28 14:15:44 +02:00
|
|
|
|
2012-06-01 15:15:53 +02:00
|
|
|
/**
|
|
|
|
|
* Wait until the specified nodes have seen the same gossip overview.
|
|
|
|
|
*/
|
2012-08-15 16:47:34 +02:00
|
|
|
def awaitSeenSameState(addresses: Address*): Unit =
|
2012-08-16 18:28:01 +02:00
|
|
|
awaitCond((addresses.toSet -- clusterView.seenBy).isEmpty)
|
2012-06-01 15:15:53 +02:00
|
|
|
|
2012-10-30 15:08:41 +01:00
|
|
|
def roleOfLeader(nodesInCluster: immutable.Seq[RoleName] = roles): RoleName = {
|
2012-05-25 14:48:00 +02:00
|
|
|
nodesInCluster.length must not be (0)
|
|
|
|
|
nodesInCluster.sorted.head
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Sort the roles in the order used by the cluster.
|
|
|
|
|
*/
|
|
|
|
|
implicit val clusterOrdering: Ordering[RoleName] = new Ordering[RoleName] {
|
|
|
|
|
import Member.addressOrdering
|
2012-06-15 17:32:40 +02:00
|
|
|
def compare(x: RoleName, y: RoleName) = addressOrdering.compare(address(x), address(y))
|
2012-05-25 14:48:00 +02:00
|
|
|
}
|
|
|
|
|
|
2012-06-18 11:16:30 +02:00
|
|
|
def roleName(addr: Address): Option[RoleName] = roles.find(address(_) == addr)
|
2012-06-15 14:37:51 +02:00
|
|
|
|
2012-09-06 21:48:40 +02:00
|
|
|
/**
|
|
|
|
|
* Marks a node as available in the failure detector if
|
|
|
|
|
* [[akka.cluster.FailureDetectorPuppet]] is used as
|
|
|
|
|
* failure detector.
|
|
|
|
|
*/
|
|
|
|
|
def markNodeAsAvailable(address: Address): Unit = cluster.failureDetector match {
|
|
|
|
|
case puppet: FailureDetectorPuppet ⇒ puppet.markNodeAsAvailable(address)
|
|
|
|
|
case _ ⇒
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Marks a node as unavailable in the failure detector if
|
|
|
|
|
* [[akka.cluster.FailureDetectorPuppet]] is used as
|
|
|
|
|
* failure detector.
|
|
|
|
|
*/
|
|
|
|
|
def markNodeAsUnavailable(address: Address): Unit = cluster.failureDetector match {
|
|
|
|
|
case puppet: FailureDetectorPuppet ⇒ puppet.markNodeAsUnavailable(address)
|
|
|
|
|
case _ ⇒
|
|
|
|
|
}
|
|
|
|
|
|
2012-05-28 11:06:02 +02:00
|
|
|
}
|
2012-08-28 16:38:05 +02:00
|
|
|
|