2013-01-15 18:08:45 +01:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2009-2013 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
|
|
|
|
|
|
|
|
|
package akka.io
|
|
|
|
|
|
|
|
|
|
import java.net.InetSocketAddress
|
|
|
|
|
import java.io.IOException
|
|
|
|
|
import java.nio.channels.SocketChannel
|
2013-01-22 14:10:36 +01:00
|
|
|
import java.nio.ByteBuffer
|
|
|
|
|
import scala.annotation.tailrec
|
2013-01-22 16:03:22 +01:00
|
|
|
import scala.collection.immutable
|
2013-01-15 18:08:45 +01:00
|
|
|
import scala.util.control.NonFatal
|
|
|
|
|
import scala.concurrent.duration._
|
|
|
|
|
import akka.actor._
|
|
|
|
|
import akka.util.ByteString
|
2013-02-05 11:48:47 +01:00
|
|
|
import akka.io.Inet.SocketOption
|
|
|
|
|
import akka.io.Tcp._
|
2013-02-01 13:11:17 +01:00
|
|
|
import akka.io.SelectionHandler._
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Base class for TcpIncomingConnection and TcpOutgoingConnection.
|
2013-02-15 11:59:01 +01:00
|
|
|
*
|
|
|
|
|
* INTERNAL API
|
2013-01-15 18:08:45 +01:00
|
|
|
*/
|
2013-01-22 15:51:21 +01:00
|
|
|
private[io] abstract class TcpConnection(val channel: SocketChannel,
|
2013-02-04 16:24:34 +01:00
|
|
|
val tcp: TcpExt) extends Actor with ActorLogging {
|
2013-01-17 17:29:44 +01:00
|
|
|
import tcp.Settings._
|
2013-02-04 16:24:34 +01:00
|
|
|
import tcp.bufferPool
|
2013-01-24 15:08:42 +01:00
|
|
|
import TcpConnection._
|
2013-01-17 14:31:35 +01:00
|
|
|
var pendingWrite: PendingWrite = null
|
2013-01-17 14:45:50 +01:00
|
|
|
|
|
|
|
|
// Needed to send the ConnectionClosed message in the postStop handler.
|
2013-01-21 14:45:19 +01:00
|
|
|
var closedMessage: CloseInformation = null
|
2013-01-17 14:45:50 +01:00
|
|
|
|
2013-04-07 17:05:22 +02:00
|
|
|
var keepOpenOnPeerClosed: Boolean = false
|
|
|
|
|
|
2013-01-17 14:31:35 +01:00
|
|
|
def writePending = pendingWrite ne null
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-01-22 15:51:21 +01:00
|
|
|
def selector = context.parent
|
|
|
|
|
|
2013-01-15 18:08:45 +01:00
|
|
|
// STATES
|
|
|
|
|
|
|
|
|
|
/** connection established, waiting for registration from user handler */
|
|
|
|
|
def waitingForRegistration(commander: ActorRef): Receive = {
|
2013-04-07 17:05:22 +02:00
|
|
|
case Register(handler, keepOpenOnPeerClosed) ⇒
|
2013-02-10 13:52:52 +01:00
|
|
|
if (TraceLogging) log.debug("[{}] registered as connection handler", handler)
|
2013-04-07 17:05:22 +02:00
|
|
|
this.keepOpenOnPeerClosed = keepOpenOnPeerClosed
|
|
|
|
|
|
2013-01-23 15:28:14 +01:00
|
|
|
doRead(handler, None) // immediately try reading
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
context.setReceiveTimeout(Duration.Undefined)
|
|
|
|
|
context.watch(handler) // sign death pact
|
|
|
|
|
|
|
|
|
|
context.become(connected(handler))
|
|
|
|
|
|
|
|
|
|
case cmd: CloseCommand ⇒
|
2013-04-02 16:39:21 +02:00
|
|
|
handleClose(commander, Some(sender), cmd.event)
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
case ReceiveTimeout ⇒
|
2013-01-17 14:45:50 +01:00
|
|
|
// after sending `Register` user should watch this actor to make sure
|
|
|
|
|
// it didn't die because of the timeout
|
2013-02-10 13:52:52 +01:00
|
|
|
log.warning("Configured registration timeout of [{}] expired, stopping", RegisterTimeout)
|
2013-01-15 18:08:45 +01:00
|
|
|
context.stop(self)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/** normal connected state */
|
2013-04-02 15:17:48 +02:00
|
|
|
def connected(handler: ActorRef): Receive = handleWriteMessages(handler) orElse {
|
|
|
|
|
case StopReading ⇒ selector ! DisableReadInterest
|
|
|
|
|
case ResumeReading ⇒ selector ! ReadInterest
|
|
|
|
|
case ChannelReadable ⇒ doRead(handler, None)
|
2013-01-23 15:19:03 +01:00
|
|
|
|
2013-04-02 15:17:48 +02:00
|
|
|
case cmd: CloseCommand ⇒ handleClose(handler, Some(sender), cmd.event)
|
|
|
|
|
}
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-04-02 15:17:48 +02:00
|
|
|
/** the peer sent EOF first, but we may still want to send */
|
|
|
|
|
def peerSentEOF(handler: ActorRef): Receive = handleWriteMessages(handler) orElse {
|
2013-04-02 16:39:21 +02:00
|
|
|
case cmd: CloseCommand ⇒ handleClose(handler, Some(sender), cmd.event)
|
2013-01-15 18:08:45 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/** connection is closing but a write has to be finished first */
|
2013-01-21 14:45:19 +01:00
|
|
|
def closingWithPendingWrite(handler: ActorRef, closeCommander: Option[ActorRef], closedEvent: ConnectionClosed): Receive = {
|
2013-02-05 13:38:27 +01:00
|
|
|
case StopReading ⇒ selector ! DisableReadInterest
|
2013-01-15 18:08:45 +01:00
|
|
|
case ResumeReading ⇒ selector ! ReadInterest
|
2013-01-21 14:45:19 +01:00
|
|
|
case ChannelReadable ⇒ doRead(handler, closeCommander)
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
case ChannelWritable ⇒
|
2013-01-17 14:45:50 +01:00
|
|
|
doWrite(handler)
|
2013-01-15 18:08:45 +01:00
|
|
|
if (!writePending) // writing is now finished
|
2013-01-21 14:45:19 +01:00
|
|
|
handleClose(handler, closeCommander, closedEvent)
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-01-21 14:45:19 +01:00
|
|
|
case Abort ⇒ handleClose(handler, Some(sender), Aborted)
|
2013-01-15 18:08:45 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/** connection is closed on our side and we're waiting from confirmation from the other side */
|
2013-01-21 14:45:19 +01:00
|
|
|
def closing(handler: ActorRef, closeCommander: Option[ActorRef]): Receive = {
|
2013-02-05 13:38:27 +01:00
|
|
|
case StopReading ⇒ selector ! DisableReadInterest
|
2013-01-15 18:08:45 +01:00
|
|
|
case ResumeReading ⇒ selector ! ReadInterest
|
2013-01-21 14:45:19 +01:00
|
|
|
case ChannelReadable ⇒ doRead(handler, closeCommander)
|
|
|
|
|
case Abort ⇒ handleClose(handler, Some(sender), Aborted)
|
2013-01-15 18:08:45 +01:00
|
|
|
}
|
|
|
|
|
|
2013-04-02 15:17:48 +02:00
|
|
|
def handleWriteMessages(handler: ActorRef): Receive = {
|
|
|
|
|
case ChannelWritable ⇒ if (writePending) doWrite(handler)
|
|
|
|
|
|
2013-04-09 12:25:43 +02:00
|
|
|
case write: WriteCommand if writePending ⇒
|
2013-04-02 15:17:48 +02:00
|
|
|
if (TraceLogging) log.debug("Dropping write because queue is full")
|
|
|
|
|
sender ! write.failureMessage
|
|
|
|
|
|
|
|
|
|
case write: Write if write.data.isEmpty ⇒
|
|
|
|
|
if (write.wantsAck)
|
|
|
|
|
sender ! write.ack
|
|
|
|
|
|
2013-04-09 12:25:43 +02:00
|
|
|
case write: WriteCommand ⇒
|
2013-04-02 15:17:48 +02:00
|
|
|
pendingWrite = createWrite(write)
|
|
|
|
|
doWrite(handler)
|
|
|
|
|
}
|
|
|
|
|
|
2013-01-15 18:08:45 +01:00
|
|
|
// AUXILIARIES and IMPLEMENTATION
|
|
|
|
|
|
2013-01-21 17:02:20 +01:00
|
|
|
/** used in subclasses to start the common machinery above once a channel is connected */
|
2013-01-22 16:03:22 +01:00
|
|
|
def completeConnect(commander: ActorRef, options: immutable.Traversable[SocketOption]): Unit = {
|
2013-03-05 14:22:21 +01:00
|
|
|
// Turn off Nagle's algorithm by default
|
|
|
|
|
channel.socket.setTcpNoDelay(true)
|
2013-01-15 18:08:45 +01:00
|
|
|
options.foreach(_.afterConnect(channel.socket))
|
|
|
|
|
|
|
|
|
|
commander ! Connected(
|
|
|
|
|
channel.socket.getRemoteSocketAddress.asInstanceOf[InetSocketAddress],
|
|
|
|
|
channel.socket.getLocalSocketAddress.asInstanceOf[InetSocketAddress])
|
|
|
|
|
|
2013-01-17 17:29:44 +01:00
|
|
|
context.setReceiveTimeout(RegisterTimeout)
|
2013-01-15 18:08:45 +01:00
|
|
|
context.become(waitingForRegistration(commander))
|
|
|
|
|
}
|
|
|
|
|
|
2013-01-21 14:45:19 +01:00
|
|
|
def doRead(handler: ActorRef, closeCommander: Option[ActorRef]): Unit = {
|
2013-04-03 16:10:49 +02:00
|
|
|
@tailrec def innerRead(buffer: ByteBuffer, remainingLimit: Int): ReadResult =
|
2013-01-24 15:08:42 +01:00
|
|
|
if (remainingLimit > 0) {
|
|
|
|
|
// never read more than the configured limit
|
|
|
|
|
buffer.clear()
|
2013-01-26 10:44:44 +01:00
|
|
|
val maxBufferSpace = math.min(DirectBufferSize, remainingLimit)
|
|
|
|
|
buffer.limit(maxBufferSpace)
|
2013-01-24 15:08:42 +01:00
|
|
|
val readBytes = channel.read(buffer)
|
|
|
|
|
buffer.flip()
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-04-03 16:10:49 +02:00
|
|
|
if (TraceLogging) log.debug("Read [{}] bytes.", readBytes)
|
|
|
|
|
if (readBytes > 0) handler ! Received(ByteString(buffer))
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-01-24 15:08:42 +01:00
|
|
|
readBytes match {
|
2013-04-03 16:10:49 +02:00
|
|
|
case `maxBufferSpace` ⇒ innerRead(buffer, remainingLimit - maxBufferSpace)
|
|
|
|
|
case x if x >= 0 ⇒ AllRead
|
|
|
|
|
case -1 ⇒ EndOfStream
|
2013-01-24 15:08:42 +01:00
|
|
|
case _ ⇒
|
|
|
|
|
throw new IllegalStateException("Unexpected value returned from read: " + readBytes)
|
|
|
|
|
}
|
2013-04-03 16:10:49 +02:00
|
|
|
} else MoreDataWaiting
|
2013-01-24 15:08:42 +01:00
|
|
|
|
2013-02-04 16:24:34 +01:00
|
|
|
val buffer = bufferPool.acquire()
|
2013-04-03 16:10:49 +02:00
|
|
|
try innerRead(buffer, ReceivedMessageSizeLimit) match {
|
|
|
|
|
case AllRead ⇒ selector ! ReadInterest
|
|
|
|
|
case MoreDataWaiting ⇒ self ! ChannelReadable
|
2013-04-02 15:17:48 +02:00
|
|
|
case EndOfStream if channel.socket.isOutputShutdown ⇒
|
|
|
|
|
if (TraceLogging) log.debug("Read returned end-of-stream, our side already closed")
|
|
|
|
|
doCloseConnection(handler, closeCommander, ConfirmedClosed)
|
2013-01-24 15:08:42 +01:00
|
|
|
case EndOfStream ⇒
|
2013-04-02 15:17:48 +02:00
|
|
|
if (TraceLogging) log.debug("Read returned end-of-stream, our side not yet closed")
|
|
|
|
|
handleClose(handler, closeCommander, PeerClosed)
|
2013-01-15 18:08:45 +01:00
|
|
|
} catch {
|
|
|
|
|
case e: IOException ⇒ handleError(handler, e)
|
2013-02-04 16:24:34 +01:00
|
|
|
} finally bufferPool.release(buffer)
|
2013-01-15 18:08:45 +01:00
|
|
|
}
|
|
|
|
|
|
2013-04-09 12:25:04 +02:00
|
|
|
def doWrite(handler: ActorRef): Unit =
|
|
|
|
|
pendingWrite = pendingWrite.doWrite(handler)
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
def closeReason =
|
|
|
|
|
if (channel.socket.isOutputShutdown) ConfirmedClosed
|
|
|
|
|
else PeerClosed
|
|
|
|
|
|
2013-04-02 15:22:11 +02:00
|
|
|
def handleClose(handler: ActorRef, closeCommander: Option[ActorRef], closedEvent: ConnectionClosed): Unit = closedEvent match {
|
|
|
|
|
case Aborted ⇒
|
2013-01-17 17:29:44 +01:00
|
|
|
if (TraceLogging) log.debug("Got Abort command. RESETing connection.")
|
2013-01-21 14:45:19 +01:00
|
|
|
doCloseConnection(handler, closeCommander, closedEvent)
|
2013-04-07 17:05:22 +02:00
|
|
|
case PeerClosed if keepOpenOnPeerClosed ⇒
|
2013-04-02 15:17:48 +02:00
|
|
|
// report that peer closed the connection
|
|
|
|
|
handler ! PeerClosed
|
|
|
|
|
// used to check if peer already closed its side later
|
|
|
|
|
channel.socket().shutdownInput()
|
|
|
|
|
context.become(peerSentEOF(handler))
|
2013-04-02 15:22:11 +02:00
|
|
|
case _ if writePending ⇒ // finish writing first
|
2013-01-17 17:29:44 +01:00
|
|
|
if (TraceLogging) log.debug("Got Close command but write is still pending.")
|
2013-01-21 14:45:19 +01:00
|
|
|
context.become(closingWithPendingWrite(handler, closeCommander, closedEvent))
|
2013-04-02 15:22:11 +02:00
|
|
|
case ConfirmedClosed ⇒ // shutdown output and wait for confirmation
|
2013-01-17 17:29:44 +01:00
|
|
|
if (TraceLogging) log.debug("Got ConfirmedClose command, sending FIN.")
|
2013-01-15 18:08:45 +01:00
|
|
|
channel.socket.shutdownOutput()
|
|
|
|
|
|
2013-04-02 15:17:48 +02:00
|
|
|
if (channel.socket().isInputShutdown) // if peer closed first, the socket is now fully closed
|
|
|
|
|
doCloseConnection(handler, closeCommander, closedEvent)
|
|
|
|
|
else context.become(closing(handler, closeCommander))
|
2013-04-02 15:22:11 +02:00
|
|
|
case _ ⇒ // close now
|
2013-01-17 17:29:44 +01:00
|
|
|
if (TraceLogging) log.debug("Got Close command, closing connection.")
|
2013-01-21 14:45:19 +01:00
|
|
|
doCloseConnection(handler, closeCommander, closedEvent)
|
2013-04-02 15:22:11 +02:00
|
|
|
}
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-01-21 14:45:19 +01:00
|
|
|
def doCloseConnection(handler: ActorRef, closeCommander: Option[ActorRef], closedEvent: ConnectionClosed): Unit = {
|
2013-01-15 18:08:45 +01:00
|
|
|
if (closedEvent == Aborted) abort()
|
|
|
|
|
else channel.close()
|
|
|
|
|
|
2013-01-21 14:45:19 +01:00
|
|
|
closedMessage = CloseInformation(Set(handler) ++ closeCommander, closedEvent)
|
2013-01-17 14:45:50 +01:00
|
|
|
|
2013-01-15 18:08:45 +01:00
|
|
|
context.stop(self)
|
|
|
|
|
}
|
|
|
|
|
|
2013-04-09 12:25:04 +02:00
|
|
|
def handleError(handler: ActorRef, exception: IOException): Nothing = {
|
2013-01-23 11:47:12 +01:00
|
|
|
closedMessage = CloseInformation(Set(handler), ErrorClosed(extractMsg(exception)))
|
2013-01-17 14:45:50 +01:00
|
|
|
|
2013-01-15 18:08:45 +01:00
|
|
|
throw exception
|
|
|
|
|
}
|
2013-01-17 14:45:50 +01:00
|
|
|
@tailrec private[this] def extractMsg(t: Throwable): String =
|
|
|
|
|
if (t == null) "unknown"
|
|
|
|
|
else {
|
|
|
|
|
t.getMessage match {
|
|
|
|
|
case null | "" ⇒ extractMsg(t.getCause)
|
|
|
|
|
case msg ⇒ msg
|
|
|
|
|
}
|
|
|
|
|
}
|
2013-01-15 18:08:45 +01:00
|
|
|
|
|
|
|
|
def abort(): Unit = {
|
|
|
|
|
try channel.socket.setSoLinger(true, 0) // causes the following close() to send TCP RST
|
|
|
|
|
catch {
|
|
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
// setSoLinger can fail due to http://bugs.sun.com/view_bug.do?bug_id=6799574
|
|
|
|
|
// (also affected: OS/X Java 1.6.0_37)
|
2013-02-10 13:52:52 +01:00
|
|
|
if (TraceLogging) log.debug("setSoLinger(true, 0) failed with [{}]", e)
|
2013-01-15 18:08:45 +01:00
|
|
|
}
|
|
|
|
|
channel.close()
|
|
|
|
|
}
|
|
|
|
|
|
2013-01-17 14:45:50 +01:00
|
|
|
override def postStop(): Unit = {
|
2013-01-22 17:32:46 +01:00
|
|
|
if (channel.isOpen)
|
|
|
|
|
abort()
|
|
|
|
|
|
2013-04-09 12:25:04 +02:00
|
|
|
if (writePending) pendingWrite.release()
|
2013-01-22 17:32:46 +01:00
|
|
|
|
2013-01-17 14:45:50 +01:00
|
|
|
if (closedMessage != null) {
|
2013-01-21 14:45:19 +01:00
|
|
|
val interestedInClose =
|
|
|
|
|
if (writePending) closedMessage.notificationsTo + pendingWrite.commander
|
|
|
|
|
else closedMessage.notificationsTo
|
2013-01-17 14:45:50 +01:00
|
|
|
|
2013-01-21 14:45:19 +01:00
|
|
|
interestedInClose.foreach(_ ! closedMessage.closedEvent)
|
2013-01-17 14:45:50 +01:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def postRestart(reason: Throwable): Unit =
|
|
|
|
|
throw new IllegalStateException("Restarting not supported for connection actors.")
|
2013-01-15 18:08:45 +01:00
|
|
|
|
2013-04-09 12:25:04 +02:00
|
|
|
private[io] case class PendingBufferWrite(
|
2013-01-21 14:45:19 +01:00
|
|
|
commander: ActorRef,
|
|
|
|
|
ack: Any,
|
|
|
|
|
remainingData: ByteString,
|
2013-04-09 12:25:04 +02:00
|
|
|
buffer: ByteBuffer) extends PendingWrite {
|
|
|
|
|
|
|
|
|
|
def release(): Unit = bufferPool.release(buffer)
|
|
|
|
|
|
|
|
|
|
def doWrite(handler: ActorRef): PendingWrite = {
|
|
|
|
|
@tailrec def innerWrite(pendingWrite: PendingBufferWrite): PendingWrite = {
|
|
|
|
|
val toWrite = pendingWrite.buffer.remaining()
|
|
|
|
|
require(toWrite != 0)
|
|
|
|
|
val writtenBytes = channel.write(pendingWrite.buffer)
|
|
|
|
|
if (TraceLogging) log.debug("Wrote [{}] bytes to channel", writtenBytes)
|
|
|
|
|
|
|
|
|
|
val nextWrite = pendingWrite.consume(writtenBytes)
|
|
|
|
|
|
|
|
|
|
if (pendingWrite.hasData)
|
|
|
|
|
if (writtenBytes == toWrite) innerWrite(nextWrite) // wrote complete buffer, try again now
|
|
|
|
|
else {
|
|
|
|
|
selector ! WriteInterest
|
|
|
|
|
nextWrite
|
|
|
|
|
} // try again later
|
|
|
|
|
else { // everything written
|
|
|
|
|
if (pendingWrite.wantsAck)
|
|
|
|
|
pendingWrite.commander ! pendingWrite.ack
|
|
|
|
|
|
|
|
|
|
pendingWrite.release()
|
|
|
|
|
null
|
|
|
|
|
}
|
|
|
|
|
}
|
2013-01-21 14:45:19 +01:00
|
|
|
|
2013-04-09 12:25:04 +02:00
|
|
|
try innerWrite(this)
|
|
|
|
|
catch { case e: IOException ⇒ handleError(handler, e) }
|
|
|
|
|
}
|
|
|
|
|
def hasData = buffer.remaining() > 0 || remainingData.size > 0
|
|
|
|
|
def consume(writtenBytes: Int): PendingBufferWrite =
|
2013-01-21 14:45:19 +01:00
|
|
|
if (buffer.remaining() == 0) {
|
|
|
|
|
buffer.clear()
|
|
|
|
|
val copied = remainingData.copyToBuffer(buffer)
|
|
|
|
|
buffer.flip()
|
|
|
|
|
copy(remainingData = remainingData.drop(copied))
|
|
|
|
|
} else this
|
2013-01-17 14:31:35 +01:00
|
|
|
}
|
2013-04-09 12:25:43 +02:00
|
|
|
def createWrite(write: WriteCommand): PendingWrite = write match {
|
|
|
|
|
case write: Write ⇒
|
|
|
|
|
val buffer = bufferPool.acquire()
|
|
|
|
|
val copied = write.data.copyToBuffer(buffer)
|
|
|
|
|
buffer.flip()
|
2013-01-17 14:31:35 +01:00
|
|
|
|
2013-04-09 12:25:43 +02:00
|
|
|
PendingBufferWrite(sender, write.ack, write.data.drop(copied), buffer)
|
2013-01-17 14:31:35 +01:00
|
|
|
}
|
2013-01-24 15:08:42 +01:00
|
|
|
}
|
|
|
|
|
|
2013-02-15 11:59:01 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2013-01-24 15:08:42 +01:00
|
|
|
private[io] object TcpConnection {
|
|
|
|
|
sealed trait ReadResult
|
|
|
|
|
object EndOfStream extends ReadResult
|
2013-04-03 16:10:49 +02:00
|
|
|
object AllRead extends ReadResult
|
|
|
|
|
object MoreDataWaiting extends ReadResult
|
2013-01-21 14:45:19 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Used to transport information to the postStop method to notify
|
|
|
|
|
* interested party about a connection close.
|
|
|
|
|
*/
|
2013-01-24 15:08:42 +01:00
|
|
|
case class CloseInformation(
|
2013-01-21 14:45:19 +01:00
|
|
|
notificationsTo: Set[ActorRef],
|
2013-03-27 15:29:24 +01:00
|
|
|
closedEvent: Event)
|
2013-04-09 12:25:04 +02:00
|
|
|
|
|
|
|
|
/** Abstraction over pending writes */
|
|
|
|
|
trait PendingWrite {
|
|
|
|
|
def commander: ActorRef
|
|
|
|
|
def ack: Any
|
|
|
|
|
|
|
|
|
|
def wantsAck = !ack.isInstanceOf[NoAck]
|
|
|
|
|
def doWrite(handler: ActorRef): PendingWrite
|
|
|
|
|
|
|
|
|
|
/** Release any open resources */
|
|
|
|
|
def release(): Unit
|
|
|
|
|
}
|
2013-01-22 15:51:21 +01:00
|
|
|
}
|