2014-04-08 10:28:27 +02:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2009-2014 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
|
|
|
|
package akka.stream.io
|
|
|
|
|
|
|
|
|
|
import akka.io.{ IO, Tcp }
|
|
|
|
|
import scala.util.control.NoStackTrace
|
2014-04-14 15:35:15 +02:00
|
|
|
import akka.actor.{ ActorRefFactory, Actor, Props, ActorRef, Status }
|
2014-04-08 10:28:27 +02:00
|
|
|
import akka.stream.impl._
|
|
|
|
|
import akka.util.ByteString
|
|
|
|
|
import akka.io.Tcp._
|
|
|
|
|
import akka.stream.MaterializerSettings
|
|
|
|
|
import org.reactivestreams.api.Processor
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] object TcpStreamActor {
|
|
|
|
|
case object WriteAck extends Tcp.Event
|
|
|
|
|
class TcpStreamException(msg: String) extends RuntimeException(msg) with NoStackTrace
|
|
|
|
|
|
|
|
|
|
def outboundProps(connectCmd: Connect, requester: ActorRef, settings: MaterializerSettings): Props =
|
|
|
|
|
Props(new OutboundTcpStreamActor(connectCmd, requester, settings))
|
|
|
|
|
def inboundProps(connection: ActorRef, settings: MaterializerSettings): Props =
|
|
|
|
|
Props(new InboundTcpStreamActor(connection, settings))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2014-04-14 15:35:15 +02:00
|
|
|
private[akka] abstract class TcpStreamActor(val settings: MaterializerSettings) extends Actor
|
2014-04-08 10:28:27 +02:00
|
|
|
with PrimaryInputs
|
|
|
|
|
with PrimaryOutputs {
|
|
|
|
|
|
|
|
|
|
import TcpStreamActor._
|
|
|
|
|
def connection: ActorRef
|
|
|
|
|
|
2014-04-14 15:35:15 +02:00
|
|
|
val tcpInputs = new DefaultInputTransferStates {
|
2014-04-08 10:28:27 +02:00
|
|
|
private var closed: Boolean = false
|
|
|
|
|
private var pendingElement: ByteString = null
|
|
|
|
|
|
|
|
|
|
override def inputsAvailable: Boolean = pendingElement ne null
|
|
|
|
|
override def inputsDepleted: Boolean = closed && !inputsAvailable
|
|
|
|
|
override def prefetch(): Unit = connection ! ResumeReading
|
|
|
|
|
override def isClosed: Boolean = closed
|
|
|
|
|
override def complete(): Unit = closed = true
|
|
|
|
|
override def cancel(): Unit = {
|
|
|
|
|
closed = true
|
|
|
|
|
pendingElement = null
|
|
|
|
|
}
|
|
|
|
|
override def dequeueInputElement(): Any = {
|
|
|
|
|
val elem = pendingElement
|
|
|
|
|
pendingElement = null
|
|
|
|
|
connection ! ResumeReading
|
|
|
|
|
elem
|
|
|
|
|
}
|
|
|
|
|
override def enqueueInputElement(elem: Any): Unit = pendingElement = elem.asInstanceOf[ByteString]
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
2014-04-14 15:35:15 +02:00
|
|
|
val tcpOutputs = new DefaultOutputTransferStates {
|
2014-04-08 10:28:27 +02:00
|
|
|
private var closed: Boolean = false
|
|
|
|
|
private var pendingDemand = true
|
|
|
|
|
override def isClosed: Boolean = closed
|
|
|
|
|
override def cancel(e: Throwable): Unit = {
|
|
|
|
|
if (!closed) connection ! Abort
|
|
|
|
|
closed = true
|
|
|
|
|
}
|
|
|
|
|
override def complete(): Unit = {
|
|
|
|
|
if (!closed) connection ! ConfirmedClose
|
|
|
|
|
closed = true
|
|
|
|
|
}
|
|
|
|
|
override def enqueueOutputElement(elem: Any): Unit = {
|
|
|
|
|
connection ! Write(elem.asInstanceOf[ByteString], WriteAck)
|
|
|
|
|
pendingDemand = false
|
|
|
|
|
}
|
|
|
|
|
def enqueueDemand(): Unit = pendingDemand = true
|
|
|
|
|
|
|
|
|
|
override def demandAvailable: Boolean = pendingDemand
|
|
|
|
|
}
|
|
|
|
|
|
2014-04-14 15:35:15 +02:00
|
|
|
val writePump = new Pump {
|
|
|
|
|
lazy val NeedsInputAndDemand = primaryInputs.NeedsInput && tcpOutputs.NeedsDemand
|
2014-04-08 10:28:27 +02:00
|
|
|
override protected def transfer(): TransferState = {
|
|
|
|
|
var batch = ByteString.empty
|
|
|
|
|
while (primaryInputs.inputsAvailable) batch ++= primaryInputs.dequeueInputElement().asInstanceOf[ByteString]
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpOutputs.enqueueOutputElement(batch)
|
2014-04-08 10:28:27 +02:00
|
|
|
NeedsInputAndDemand
|
|
|
|
|
}
|
2014-04-14 15:35:15 +02:00
|
|
|
override protected def pumpFinished(): Unit = tcpOutputs.complete()
|
2014-04-08 10:28:27 +02:00
|
|
|
override protected def pumpFailed(e: Throwable): Unit = fail(e)
|
|
|
|
|
override protected def pumpContext: ActorRefFactory = context
|
|
|
|
|
}
|
|
|
|
|
|
2014-04-14 15:35:15 +02:00
|
|
|
val readPump = new Pump {
|
|
|
|
|
lazy val NeedsInputAndDemand = tcpInputs.NeedsInput && primaryOutputs.NeedsDemand
|
2014-04-08 10:28:27 +02:00
|
|
|
override protected def transfer(): TransferState = {
|
2014-04-14 15:35:15 +02:00
|
|
|
primaryOutputs.enqueueOutputElement(tcpInputs.dequeueInputElement())
|
2014-04-08 10:28:27 +02:00
|
|
|
NeedsInputAndDemand
|
|
|
|
|
}
|
2014-04-14 15:35:15 +02:00
|
|
|
override protected def pumpFinished(): Unit = primaryOutputs.complete()
|
2014-04-08 10:28:27 +02:00
|
|
|
override protected def pumpFailed(e: Throwable): Unit = fail(e)
|
|
|
|
|
override protected def pumpContext: ActorRefFactory = context
|
|
|
|
|
}
|
|
|
|
|
|
2014-04-14 15:35:15 +02:00
|
|
|
override def pumpInputs(): Unit = writePump.pump()
|
|
|
|
|
override def pumpOutputs(): Unit = readPump.pump()
|
2014-04-08 10:28:27 +02:00
|
|
|
|
|
|
|
|
override def receive = waitingExposedPublisher
|
|
|
|
|
|
|
|
|
|
override def primaryInputOnError(e: Throwable): Unit = fail(e)
|
|
|
|
|
override def primaryInputOnComplete(): Unit = shutdown()
|
|
|
|
|
override def primaryInputsReady(): Unit = {
|
|
|
|
|
connection ! Register(self, keepOpenOnPeerClosed = true, useResumeWriting = false)
|
2014-04-14 15:35:15 +02:00
|
|
|
readPump.setTransferState(readPump.NeedsInputAndDemand)
|
|
|
|
|
writePump.setTransferState(writePump.NeedsInputAndDemand)
|
|
|
|
|
tcpInputs.prefetch()
|
2014-04-08 10:28:27 +02:00
|
|
|
context.become(running)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def primaryOutputsReady(): Unit = context.become(downstreamManagement orElse waitingForUpstream)
|
|
|
|
|
override def primaryOutputsFinished(completed: Boolean): Unit = shutdown()
|
|
|
|
|
|
|
|
|
|
val running: Receive = upstreamManagement orElse downstreamManagement orElse {
|
|
|
|
|
case WriteAck ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpOutputs.enqueueDemand()
|
2014-04-08 10:28:27 +02:00
|
|
|
pumpInputs()
|
|
|
|
|
case Received(data) ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpInputs.enqueueInputElement(data)
|
2014-04-08 10:28:27 +02:00
|
|
|
pumpOutputs()
|
|
|
|
|
case Closed ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpInputs.complete()
|
|
|
|
|
tcpOutputs.complete()
|
|
|
|
|
writePump.pump()
|
|
|
|
|
readPump.pump()
|
2014-04-08 10:28:27 +02:00
|
|
|
case ConfirmedClosed ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpInputs.complete()
|
2014-04-08 10:28:27 +02:00
|
|
|
pumpOutputs()
|
|
|
|
|
case PeerClosed ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpInputs.complete()
|
2014-04-08 10:28:27 +02:00
|
|
|
pumpOutputs()
|
|
|
|
|
case ErrorClosed(cause) ⇒ fail(new TcpStreamException(s"The connection closed with error $cause"))
|
|
|
|
|
case CommandFailed(cmd) ⇒ fail(new TcpStreamException(s"Tcp command [$cmd] failed"))
|
|
|
|
|
case Aborted ⇒ fail(new TcpStreamException("The connection has been aborted"))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def fail(e: Throwable): Unit = {
|
2014-04-14 15:35:15 +02:00
|
|
|
tcpInputs.cancel()
|
|
|
|
|
tcpOutputs.cancel(e)
|
2014-04-08 10:28:27 +02:00
|
|
|
if (primaryInputs ne null) primaryInputs.cancel()
|
2014-04-14 15:35:15 +02:00
|
|
|
primaryOutputs.cancel(e)
|
2014-04-08 10:28:27 +02:00
|
|
|
exposedPublisher.shutdown(Some(e))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def shutdown(): Unit = {
|
2014-04-14 15:35:15 +02:00
|
|
|
if (tcpOutputs.isClosed && primaryOutputs.isClosed) {
|
2014-04-08 10:28:27 +02:00
|
|
|
context.stop(self)
|
|
|
|
|
exposedPublisher.shutdown(None)
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] class InboundTcpStreamActor(
|
|
|
|
|
val connection: ActorRef, _settings: MaterializerSettings)
|
|
|
|
|
extends TcpStreamActor(_settings) {
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] class OutboundTcpStreamActor(val connectCmd: Connect, val requester: ActorRef, _settings: MaterializerSettings)
|
|
|
|
|
extends TcpStreamActor(_settings) {
|
|
|
|
|
import TcpStreamActor._
|
|
|
|
|
var connection: ActorRef = _
|
|
|
|
|
import context.system
|
|
|
|
|
|
|
|
|
|
override def primaryOutputsReady(): Unit = context.become(waitingExposedProcessor)
|
|
|
|
|
|
|
|
|
|
val waitingExposedProcessor: Receive = {
|
|
|
|
|
case StreamTcpManager.ExposedProcessor(processor) ⇒
|
|
|
|
|
IO(Tcp) ! connectCmd
|
|
|
|
|
context.become(waitConnection(processor))
|
|
|
|
|
case _ ⇒ throw new IllegalStateException("The second message must be ExposedProcessor")
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def waitConnection(exposedProcessor: Processor[ByteString, ByteString]): Receive = {
|
|
|
|
|
case Connected(remoteAddress, localAddress) ⇒
|
|
|
|
|
connection = sender()
|
|
|
|
|
requester ! StreamTcp.OutgoingTcpConnection(remoteAddress, localAddress, exposedProcessor)
|
|
|
|
|
context.become(downstreamManagement orElse waitingForUpstream)
|
|
|
|
|
case f: CommandFailed ⇒
|
2014-04-14 15:35:15 +02:00
|
|
|
val ex = new TcpStreamException("Connection failed.")
|
|
|
|
|
requester ! Status.Failure(ex)
|
|
|
|
|
fail(ex)
|
2014-04-08 10:28:27 +02:00
|
|
|
}
|
|
|
|
|
}
|