2017-05-26 04:48:44 +08:00
/*
* FlowTransport . actor . cpp
*
* This source file is part of the FoundationDB open source project
*
* Copyright 2013 - 2018 Apple Inc . and the FoundationDB project authors
2018-02-22 02:25:11 +08:00
*
2017-05-26 04:48:44 +08:00
* Licensed under the Apache License , Version 2.0 ( the " License " ) ;
* you may not use this file except in compliance with the License .
* You may obtain a copy of the License at
2018-02-22 02:25:11 +08:00
*
2017-05-26 04:48:44 +08:00
* http : //www.apache.org/licenses/LICENSE-2.0
2018-02-22 02:25:11 +08:00
*
2017-05-26 04:48:44 +08:00
* Unless required by applicable law or agreed to in writing , software
* distributed under the License is distributed on an " AS IS " BASIS ,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND , either express or implied .
* See the License for the specific language governing permissions and
* limitations under the License .
*/
2018-10-20 01:30:13 +08:00
# include "fdbrpc/FlowTransport.h"
2017-05-26 04:48:44 +08:00
2019-06-25 05:40:43 +08:00
# include <unordered_map>
2017-10-28 07:54:44 +08:00
# if VALGRIND
# include <memcheck.h>
# endif
2019-06-25 05:40:43 +08:00
# include "fdbrpc/crc32c.h"
# include "fdbrpc/fdbrpc.h"
# include "fdbrpc/FailureMonitor.h"
# include "fdbrpc/genericactors.actor.h"
# include "fdbrpc/simulator.h"
# include "flow/ActorCollection.h"
# include "flow/Error.h"
# include "flow/flow.h"
# include "flow/Net2Packet.h"
# include "flow/TDMetric.actor.h"
# include "flow/ObjectSerializer.h"
# include "flow/ProtocolVersion.h"
2018-08-11 07:05:40 +08:00
# include "flow/actorcompiler.h" // This must be the last #include.
2017-10-28 07:54:44 +08:00
2019-03-24 08:54:46 +08:00
static NetworkAddressList g_currentDeliveryPeerAddress = NetworkAddressList ( ) ;
2017-05-26 04:48:44 +08:00
const UID WLTOKEN_ENDPOINT_NOT_FOUND ( - 1 , 0 ) ;
const UID WLTOKEN_PING_PACKET ( - 1 , 1 ) ;
const UID TOKEN_IGNORE_PACKET ( 0 , 2 ) ;
const uint64_t TOKEN_STREAM_FLAG = 1 ;
class EndpointMap : NonCopyable {
public :
EndpointMap ( ) ;
2019-07-04 12:03:58 +08:00
void insert ( NetworkMessageReceiver * r , Endpoint : : Token & token , TaskPriority priority ) ;
2017-05-26 04:48:44 +08:00
NetworkMessageReceiver * get ( Endpoint : : Token const & token ) ;
2019-06-25 17:47:35 +08:00
TaskPriority getPriority ( Endpoint : : Token const & token ) ;
2017-05-26 04:48:44 +08:00
void remove ( Endpoint : : Token const & token , NetworkMessageReceiver * r ) ;
private :
void realloc ( ) ;
struct Entry {
union {
uint64_t uid [ 2 ] ; // priority packed into lower 32 bits; actual lower 32 bits of token are the index in data[]
uint32_t nextFree ;
} ;
NetworkMessageReceiver * receiver ;
Endpoint : : Token & token ( ) { return * ( Endpoint : : Token * ) uid ; }
} ;
std : : vector < Entry > data ;
uint32_t firstFree ;
} ;
2019-01-25 09:28:26 +08:00
EndpointMap : : EndpointMap ( )
: firstFree ( - 1 )
2017-05-26 04:48:44 +08:00
{
}
void EndpointMap : : realloc ( ) {
int oldSize = data . size ( ) ;
data . resize ( std : : max ( 128 , oldSize * 2 ) ) ;
for ( int i = oldSize ; i < data . size ( ) ; i + + ) {
data [ i ] . receiver = 0 ;
data [ i ] . nextFree = i + 1 ;
}
data [ data . size ( ) - 1 ] . nextFree = firstFree ;
firstFree = oldSize ;
}
2019-07-04 12:03:58 +08:00
void EndpointMap : : insert ( NetworkMessageReceiver * r , Endpoint : : Token & token , TaskPriority priority ) {
2017-05-26 04:48:44 +08:00
if ( firstFree = = uint32_t ( - 1 ) ) realloc ( ) ;
int index = firstFree ;
firstFree = data [ index ] . nextFree ;
token = Endpoint : : Token ( token . first ( ) , ( token . second ( ) & 0xffffffff00000000LL ) | index ) ;
2019-07-04 12:03:58 +08:00
data [ index ] . token ( ) = Endpoint : : Token ( token . first ( ) , ( token . second ( ) & 0xffffffff00000000LL ) | static_cast < uint32_t > ( priority ) ) ;
2017-05-26 04:48:44 +08:00
data [ index ] . receiver = r ;
}
NetworkMessageReceiver * EndpointMap : : get ( Endpoint : : Token const & token ) {
uint32_t index = token . second ( ) ;
if ( index < data . size ( ) & & data [ index ] . token ( ) . first ( ) = = token . first ( ) & & ( ( data [ index ] . token ( ) . second ( ) & 0xffffffff00000000LL ) | index ) = = token . second ( ) )
return data [ index ] . receiver ;
return 0 ;
}
2019-06-25 17:47:35 +08:00
TaskPriority EndpointMap : : getPriority ( Endpoint : : Token const & token ) {
2017-05-26 04:48:44 +08:00
uint32_t index = token . second ( ) ;
if ( index < data . size ( ) & & data [ index ] . token ( ) . first ( ) = = token . first ( ) & & ( ( data [ index ] . token ( ) . second ( ) & 0xffffffff00000000LL ) | index ) = = token . second ( ) )
2019-06-25 17:47:35 +08:00
return static_cast < TaskPriority > ( data [ index ] . token ( ) . second ( ) ) ;
return TaskPriority : : UnknownEndpoint ;
2017-05-26 04:48:44 +08:00
}
void EndpointMap : : remove ( Endpoint : : Token const & token , NetworkMessageReceiver * r ) {
uint32_t index = token . second ( ) ;
if ( index < data . size ( ) & & data [ index ] . token ( ) . first ( ) = = token . first ( ) & & ( ( data [ index ] . token ( ) . second ( ) & 0xffffffff00000000LL ) | index ) = = token . second ( ) & & data [ index ] . receiver = = r ) {
data [ index ] . receiver = 0 ;
data [ index ] . nextFree = firstFree ;
firstFree = index ;
}
}
struct EndpointNotFoundReceiver : NetworkMessageReceiver {
EndpointNotFoundReceiver ( EndpointMap & endpoints ) {
//endpoints[WLTOKEN_ENDPOINT_NOT_FOUND] = this;
Endpoint : : Token e = WLTOKEN_ENDPOINT_NOT_FOUND ;
2019-07-04 12:03:58 +08:00
endpoints . insert ( this , e , TaskPriority : : DefaultEndpoint ) ;
2017-05-26 04:48:44 +08:00
ASSERT ( e = = WLTOKEN_ENDPOINT_NOT_FOUND ) ;
}
virtual void receive ( ArenaReader & reader ) {
// Remote machine tells us it doesn't have endpoint e
Endpoint e ; reader > > e ;
IFailureMonitor : : failureMonitor ( ) . endpointNotFound ( e ) ;
}
2019-01-29 11:38:13 +08:00
virtual void receive ( ArenaObjectReader & reader ) {
Endpoint e ;
reader . deserialize ( e ) ;
IFailureMonitor : : failureMonitor ( ) . endpointNotFound ( e ) ;
}
2017-05-26 04:48:44 +08:00
} ;
struct PingReceiver : NetworkMessageReceiver {
PingReceiver ( EndpointMap & endpoints ) {
Endpoint : : Token e = WLTOKEN_PING_PACKET ;
2019-07-04 12:03:58 +08:00
endpoints . insert ( this , e , TaskPriority : : ReadSocket ) ;
2017-05-26 04:48:44 +08:00
ASSERT ( e = = WLTOKEN_PING_PACKET ) ;
}
virtual void receive ( ArenaReader & reader ) {
ReplyPromise < Void > reply ; reader > > reply ;
reply . send ( Void ( ) ) ;
}
2019-01-29 11:38:13 +08:00
virtual void receive ( ArenaObjectReader & reader ) {
ReplyPromise < Void > reply ;
reader . deserialize ( reply ) ;
reply . send ( Void ( ) ) ;
}
2017-05-26 04:48:44 +08:00
} ;
class TransportData {
public :
2019-01-25 09:28:26 +08:00
TransportData ( uint64_t transportId )
2017-05-26 04:48:44 +08:00
: endpointNotFoundReceiver ( endpoints ) ,
2017-07-18 01:40:36 +08:00
pingReceiver ( endpoints ) ,
2017-05-26 04:48:44 +08:00
warnAlwaysForLargePacket ( true ) ,
lastIncompatibleMessage ( 0 ) ,
2018-04-12 02:15:12 +08:00
transportId ( transportId ) ,
numIncompatibleConnections ( 0 )
2019-04-05 05:11:12 +08:00
{
degraded = Reference < AsyncVar < bool > > ( new AsyncVar < bool > ( false ) ) ;
}
2017-05-26 04:48:44 +08:00
2018-07-28 11:46:54 +08:00
~ TransportData ( ) ;
2017-05-26 04:48:44 +08:00
void initMetrics ( ) {
bytesSent . init ( LiteralStringRef ( " Net2.BytesSent " ) ) ;
countPacketsReceived . init ( LiteralStringRef ( " Net2.CountPacketsReceived " ) ) ;
countPacketsGenerated . init ( LiteralStringRef ( " Net2.CountPacketsGenerated " ) ) ;
countConnEstablished . init ( LiteralStringRef ( " Net2.CountConnEstablished " ) ) ;
countConnClosedWithError . init ( LiteralStringRef ( " Net2.CountConnClosedWithError " ) ) ;
countConnClosedWithoutError . init ( LiteralStringRef ( " Net2.CountConnClosedWithoutError " ) ) ;
}
2018-07-09 01:26:41 +08:00
struct Peer * getPeer ( NetworkAddress const & address , bool openConnection = true ) ;
2018-10-31 04:44:37 +08:00
// Returns true if given network address 'address' is one of the address we are listening on.
bool isLocalAddress ( const NetworkAddress & address ) const ;
2018-12-18 03:25:44 +08:00
NetworkAddressList localAddresses ;
2019-02-01 10:20:14 +08:00
std : : vector < Future < Void > > listeners ;
2019-03-28 09:21:13 +08:00
std : : unordered_map < NetworkAddress , struct Peer * > peers ;
2019-04-05 05:11:12 +08:00
std : : unordered_map < NetworkAddress , std : : pair < double , double > > closedPeers ;
Reference < AsyncVar < bool > > degraded ;
2017-05-26 04:48:44 +08:00
bool warnAlwaysForLargePacket ;
// These declarations must be in exactly this order
EndpointMap endpoints ;
EndpointNotFoundReceiver endpointNotFoundReceiver ;
PingReceiver pingReceiver ;
// End ordered declarations
Int64MetricHandle bytesSent ;
Int64MetricHandle countPacketsReceived ;
Int64MetricHandle countPacketsGenerated ;
Int64MetricHandle countConnEstablished ;
Int64MetricHandle countConnClosedWithError ;
Int64MetricHandle countConnClosedWithoutError ;
std : : map < NetworkAddress , std : : pair < uint64_t , double > > incompatiblePeers ;
2017-11-15 10:37:29 +08:00
uint32_t numIncompatibleConnections ;
2017-05-26 04:48:44 +08:00
std : : map < uint64_t , double > multiVersionConnections ;
double lastIncompatibleMessage ;
uint64_t transportId ;
Future < Void > multiVersionCleanup ;
} ;
# define CONNECT_PACKET_V0 0x0FDB00A444020001LL
# define CONNECT_PACKET_V0_SIZE 14
# pragma pack( push, 1 )
struct ConnectPacket {
2019-03-01 08:07:49 +08:00
// The value does not inclueds the size of `connectPacketLength` itself,
2019-03-01 04:24:56 +08:00
// but only the other fields of this structure.
uint32_t connectPacketLength ;
2019-06-19 05:49:04 +08:00
ProtocolVersion protocolVersion ; // Expect currentProtocolVersion
2019-03-01 08:07:49 +08:00
2017-05-26 04:48:44 +08:00
uint16_t canonicalRemotePort ; // Port number to reconnect to the originating process
uint64_t connectionId ; // Multi-version clients will use the same Id for both connections, other connections will set this to zero. Added at protocol Version 0x0FDB00A444020001.
2019-03-01 08:07:49 +08:00
// IP Address to reconnect to the originating process. Only one of these must be populated.
uint32_t canonicalRemoteIp4 ;
enum ConnectPacketFlags {
FLAG_IPV6 = 1
} ;
uint16_t flags ;
uint8_t canonicalRemoteIp6 [ 16 ] ;
2019-02-27 10:04:03 +08:00
2019-03-09 03:40:32 +08:00
ConnectPacket ( ) {
memset ( this , 0 , sizeof ( * this ) ) ;
}
2019-02-27 10:04:03 +08:00
IPAddress canonicalRemoteIp ( ) const {
if ( isIPv6 ( ) ) {
2019-03-01 08:07:49 +08:00
IPAddress : : IPAddressStore store ;
memcpy ( store . data ( ) , canonicalRemoteIp6 , sizeof ( canonicalRemoteIp6 ) ) ;
return IPAddress ( store ) ;
2019-02-27 10:04:03 +08:00
} else {
2019-03-01 08:07:49 +08:00
return IPAddress ( canonicalRemoteIp4 ) ;
2019-02-27 10:04:03 +08:00
}
}
2017-05-26 04:48:44 +08:00
2019-02-27 10:04:03 +08:00
void setCanonicalRemoteIp ( const IPAddress & ip ) {
if ( ip . isV6 ( ) ) {
2019-03-01 08:07:49 +08:00
flags = flags | FLAG_IPV6 ;
memcpy ( & canonicalRemoteIp6 , ip . toV6 ( ) . data ( ) , 16 ) ;
2019-02-27 10:04:03 +08:00
} else {
2019-03-01 08:07:49 +08:00
flags = flags & ~ FLAG_IPV6 ;
canonicalRemoteIp4 = ip . toV4 ( ) ;
2019-02-27 10:04:03 +08:00
}
}
2019-03-01 08:07:49 +08:00
bool isIPv6 ( ) const { return flags & FLAG_IPV6 ; }
2019-02-27 10:04:03 +08:00
uint32_t totalPacketSize ( ) const { return connectPacketLength + sizeof ( connectPacketLength ) ; }
template < class Ar >
void serialize ( Ar & ar ) {
2019-03-01 08:07:49 +08:00
serializer ( ar , connectPacketLength ) ;
2019-04-11 08:41:02 +08:00
if ( connectPacketLength > sizeof ( ConnectPacket ) - sizeof ( connectPacketLength ) ) {
ASSERT ( ! g_network - > isSimulated ( ) ) ;
throw serialization_failed ( ) ;
}
2019-03-01 08:07:49 +08:00
serializer ( ar , protocolVersion , canonicalRemotePort , connectionId , canonicalRemoteIp4 ) ;
2019-06-19 08:55:27 +08:00
if ( ar . isDeserializing & & ! ar . protocolVersion ( ) . hasIPv6 ( ) ) {
2019-03-01 08:07:49 +08:00
flags = 0 ;
2019-02-27 10:04:03 +08:00
} else {
2019-03-01 08:07:49 +08:00
// We can send everything in serialized packet, since the current version of ConnectPacket
// is backward compatible with CONNECT_PACKET_V0.
serializer ( ar , flags ) ;
ar . serializeBytes ( & canonicalRemoteIp6 , sizeof ( canonicalRemoteIp6 ) ) ;
2019-02-27 10:04:03 +08:00
}
2017-05-26 04:48:44 +08:00
}
} ;
# pragma pack( pop )
2019-02-18 06:46:09 +08:00
ACTOR static Future < Void > connectionReader ( TransportData * transport , Reference < IConnection > conn , Peer * peer ,
Promise < Peer * > onConnected ) ;
2017-07-18 01:40:36 +08:00
2018-07-09 01:26:41 +08:00
static PacketID sendPacket ( TransportData * self , ISerializeSource const & what , const Endpoint & destination , bool reliable , bool openConnection ) ;
2017-05-26 04:48:44 +08:00
struct Peer : NonCopyable {
TransportData * transport ;
NetworkAddress destination ;
UnsentPacketQueue unsent ;
ReliablePacketList reliable ;
AsyncTrigger dataToSend ; // Triggered when unsent.empty() becomes false
Future < Void > connect ;
2019-05-17 08:26:48 +08:00
AsyncTrigger resetPing ;
2017-07-18 01:40:36 +08:00
bool compatible ;
2017-05-26 04:48:44 +08:00
bool outgoingConnectionIdle ; // We don't actually have a connection open and aren't trying to open one because we don't have anything to send
double lastConnectTime ;
double reconnectionDelay ;
2018-07-11 04:10:29 +08:00
int peerReferences ;
2018-07-28 11:42:06 +08:00
bool incompatibleProtocolVersionNewer ;
2019-05-17 08:26:48 +08:00
int64_t bytesReceived ;
2019-06-29 04:43:22 +08:00
double lastDataPacketSentTime ;
2017-05-26 04:48:44 +08:00
2019-06-29 04:43:22 +08:00
explicit Peer ( TransportData * transport , NetworkAddress const & destination )
: transport ( transport ) , destination ( destination ) , outgoingConnectionIdle ( false ) , lastConnectTime ( 0.0 ) ,
reconnectionDelay ( FLOW_KNOBS - > INITIAL_RECONNECTION_TIME ) , compatible ( true ) ,
incompatibleProtocolVersionNewer ( false ) , peerReferences ( - 1 ) , bytesReceived ( 0 ) , lastDataPacketSentTime ( now ( ) ) {
2018-07-09 01:26:41 +08:00
connect = connectionKeeper ( this ) ;
2017-05-26 04:48:44 +08:00
}
void send ( PacketBuffer * pb , ReliablePacket * rp , bool firstUnsent ) {
unsent . setWriteBuffer ( pb ) ;
if ( rp ) reliable . insert ( rp ) ;
if ( firstUnsent ) dataToSend . trigger ( ) ;
}
void prependConnectPacket ( ) {
// Send the ConnectPacket expected at the beginning of a new connection
ConnectPacket pkt ;
2019-03-24 08:54:46 +08:00
if ( transport - > localAddresses . address . isTLS ( ) = = destination . isTLS ( ) ) {
pkt . canonicalRemotePort = transport - > localAddresses . address . port ;
pkt . setCanonicalRemoteIp ( transport - > localAddresses . address . ip ) ;
} else if ( transport - > localAddresses . secondaryAddress . present ( ) ) {
pkt . canonicalRemotePort = transport - > localAddresses . secondaryAddress . get ( ) . port ;
pkt . setCanonicalRemoteIp ( transport - > localAddresses . secondaryAddress . get ( ) . ip ) ;
} else {
2019-06-19 05:49:04 +08:00
// a "mixed" TLS/non-TLS connection is like a client/server connection - there's no way to reverse it
pkt . canonicalRemotePort = 0 ;
2019-02-27 10:04:03 +08:00
pkt . setCanonicalRemoteIp ( IPAddress ( 0 ) ) ;
2017-05-26 04:48:44 +08:00
}
2019-02-01 10:20:14 +08:00
2019-03-01 08:07:49 +08:00
pkt . connectPacketLength = sizeof ( pkt ) - sizeof ( pkt . connectPacketLength ) ;
2019-06-19 05:49:04 +08:00
pkt . protocolVersion = currentProtocolVersion ;
if ( g_network - > useObjectSerializer ( ) ) {
pkt . protocolVersion . addObjectSerializerFlag ( ) ;
}
2017-05-26 04:48:44 +08:00
pkt . connectionId = transport - > transportId ;
PacketBuffer * pb_first = new PacketBuffer ;
2019-06-12 07:44:00 +08:00
PacketWriter wr ( pb_first , nullptr , Unversioned ( ) ) ;
2019-02-27 10:04:03 +08:00
pkt . serialize ( wr ) ;
2017-05-26 04:48:44 +08:00
unsent . prependWriteBuffer ( pb_first , wr . finish ( ) ) ;
}
void discardUnreliablePackets ( ) {
// Throw away the current unsent list, dropping the reference count on each PacketBuffer that accounts for presence in the unsent list
unsent . discardAll ( ) ;
2018-07-07 10:44:30 +08:00
// If there are reliable packets, compact reliable packets into a new unsent range
if ( ! reliable . empty ( ) ) {
PacketBuffer * pb = unsent . getWriteBuffer ( ) ;
2019-06-12 07:44:00 +08:00
pb = reliable . compact ( pb , nullptr ) ;
2018-07-07 10:44:30 +08:00
unsent . setWriteBuffer ( pb ) ;
}
2017-05-26 04:48:44 +08:00
}
void onIncomingConnection ( Reference < IConnection > conn , Future < Void > reader ) {
// In case two processes are trying to connect to each other simultaneously, the process with the larger canonical NetworkAddress
// gets to keep its outgoing connection.
if ( ! destination . isPublic ( ) & & ! outgoingConnectionIdle ) throw address_in_use ( ) ;
2019-03-24 08:54:46 +08:00
NetworkAddress compatibleAddr = transport - > localAddresses . address ;
if ( transport - > localAddresses . secondaryAddress . present ( ) & & transport - > localAddresses . secondaryAddress . get ( ) . isTLS ( ) = = destination . isTLS ( ) ) {
compatibleAddr = transport - > localAddresses . secondaryAddress . get ( ) ;
2019-02-01 10:20:14 +08:00
}
2019-02-27 08:20:05 +08:00
2019-02-01 10:20:14 +08:00
if ( ! destination . isPublic ( ) | | outgoingConnectionIdle | | destination > compatibleAddr ) {
2017-05-26 04:48:44 +08:00
// Keep the new connection
TraceEvent ( " IncomingConnection " , conn - > getDebugID ( ) )
2018-09-05 08:12:26 +08:00
. suppressFor ( 1.0 )
2017-05-26 04:48:44 +08:00
. detail ( " FromAddr " , conn - > getPeerAddress ( ) )
. detail ( " CanonicalAddr " , destination )
. detail ( " IsPublic " , destination . isPublic ( ) ) ;
connect . cancel ( ) ;
prependConnectPacket ( ) ;
connect = connectionKeeper ( this , conn , reader ) ;
} else {
TraceEvent ( " RedundantConnection " , conn - > getDebugID ( ) )
2018-09-05 08:12:26 +08:00
. suppressFor ( 1.0 )
2017-05-26 04:48:44 +08:00
. detail ( " FromAddr " , conn - > getPeerAddress ( ) . toString ( ) )
2019-02-01 10:20:14 +08:00
. detail ( " CanonicalAddr " , destination )
. detail ( " LocalAddr " , compatibleAddr ) ;
2017-05-26 04:48:44 +08:00
// Keep our prior connection
reader . cancel ( ) ;
conn - > close ( ) ;
// Send an (ignored) packet to make sure that, if our outgoing connection died before the peer made this connection attempt,
// we eventually find out that our connection is dead, close it, and then respond to the next connection reattempt from peer.
}
}
2017-07-18 01:40:36 +08:00
ACTOR static Future < Void > connectionMonitor ( Peer * peer ) {
2019-07-06 07:27:17 +08:00
state Endpoint remotePingEndpoint ( { peer - > destination } , WLTOKEN_PING_PACKET ) ;
loop {
if ( ! FlowTransport : : transport ( ) . isClient ( ) & & ! peer - > destination . isPublic ( ) ) {
// Don't send ping messages to clients unless necessary. Instead monitor incoming client pings.
state double lastRefreshed = now ( ) ;
state int64_t lastBytesReceived = peer - > bytesReceived ;
loop {
wait ( delay ( FLOW_KNOBS - > CONNECTION_MONITOR_LOOP_TIME ) ) ;
if ( lastBytesReceived < peer - > bytesReceived ) {
lastRefreshed = now ( ) ;
lastBytesReceived = peer - > bytesReceived ;
} else if ( lastRefreshed < now ( ) - FLOW_KNOBS - > CONNECTION_MONITOR_IDLE_TIMEOUT *
FLOW_KNOBS - > CONNECTION_MONITOR_INCOMING_IDLE_MULTIPLIER ) {
// If we have not received anything in this period, client must have closed
// connection by now. Break loop to check if it is still alive by sending a ping.
break ;
}
2019-06-28 15:39:51 +08:00
}
}
2017-05-26 04:48:44 +08:00
2019-07-09 10:13:53 +08:00
if ( peer - > reliable . empty ( ) & & peer - > unsent . empty ( ) ) {
if ( peer - > peerReferences = = 0 & &
( peer - > lastDataPacketSentTime < now ( ) - FLOW_KNOBS - > CONNECTION_MONITOR_UNREFERENCED_CLOSE_DELAY ) ) {
// TODO: What about when peerReference == -1?
2019-06-29 04:43:22 +08:00
throw connection_unreferenced ( ) ;
2019-07-09 10:13:53 +08:00
} else if ( FlowTransport : : transport ( ) . isClient ( ) & & peer - > destination . isPublic ( ) & &
( peer - > lastConnectTime < now ( ) - FLOW_KNOBS - > CONNECTION_MONITOR_IDLE_TIMEOUT ) & &
( peer - > lastDataPacketSentTime < now ( ) - FLOW_KNOBS - > CONNECTION_MONITOR_IDLE_TIMEOUT ) ) {
2019-07-06 07:27:17 +08:00
// First condition is necessary because we may get here if we are server.
2019-06-29 04:43:22 +08:00
throw connection_idle ( ) ;
2019-07-06 07:27:17 +08:00
}
2019-06-28 15:39:51 +08:00
}
2017-05-26 04:48:44 +08:00
2019-07-06 07:27:17 +08:00
wait ( delayJittered ( FLOW_KNOBS - > CONNECTION_MONITOR_LOOP_TIME ) ) ;
2019-06-28 15:39:51 +08:00
// TODO: Stop monitoring and close the connection with no onDisconnect requests outstanding
2017-05-26 04:48:44 +08:00
state ReplyPromise < Void > reply ;
2019-06-28 15:38:28 +08:00
FlowTransport : : transport ( ) . sendUnreliable ( SerializeSource < ReplyPromise < Void > > ( reply ) , remotePingEndpoint ) ;
2019-05-17 08:26:48 +08:00
state int64_t startingBytes = peer - > bytesReceived ;
2019-05-31 08:25:42 +08:00
state int timeouts = 0 ;
2019-05-17 08:26:48 +08:00
loop {
choose {
when ( wait ( delay ( FLOW_KNOBS - > CONNECTION_MONITOR_TIMEOUT ) ) ) {
if ( startingBytes = = peer - > bytesReceived ) {
TraceEvent ( " ConnectionTimeout " ) . suppressFor ( 1.0 ) . detail ( " WithAddr " , peer - > destination ) ;
throw connection_failed ( ) ;
}
2019-05-31 08:25:42 +08:00
if ( timeouts > 1 ) {
2019-06-28 15:39:51 +08:00
TraceEvent ( SevWarnAlways , " ConnectionSlowPing " )
. suppressFor ( 1.0 )
. detail ( " WithAddr " , peer - > destination )
. detail ( " Timeouts " , timeouts ) ;
2019-05-31 08:25:42 +08:00
}
2019-05-17 08:26:48 +08:00
startingBytes = peer - > bytesReceived ;
2019-05-31 08:25:42 +08:00
timeouts + + ;
2019-05-17 08:26:48 +08:00
}
when ( wait ( reply . getFuture ( ) ) ) {
break ;
}
when ( wait ( peer - > resetPing . onTrigger ( ) ) ) {
break ;
}
}
2017-05-26 04:48:44 +08:00
}
}
}
ACTOR static Future < Void > connectionWriter ( Peer * self , Reference < IConnection > conn ) {
state double lastWriteTime = now ( ) ;
loop {
2019-06-25 17:47:35 +08:00
//wait( delay(0, TaskPriority::WriteSocket) );
wait ( delayJittered ( std : : max < double > ( FLOW_KNOBS - > MIN_COALESCE_DELAY , FLOW_KNOBS - > MAX_COALESCE_DELAY - ( now ( ) - lastWriteTime ) ) , TaskPriority : : WriteSocket ) ) ;
//wait( delay(500e-6, TaskPriority::WriteSocket) );
//wait( yield(TaskPriority::WriteSocket) );
2017-05-26 04:48:44 +08:00
// Send until there is nothing left to send
loop {
lastWriteTime = now ( ) ;
2019-06-12 08:52:25 +08:00
int sent = conn - > write ( self - > unsent . getUnsent ( ) , /* limit= */ FLOW_KNOBS - > MAX_PACKET_SEND_BYTES ) ;
2017-05-26 04:48:44 +08:00
if ( sent ) {
self - > transport - > bytesSent + = sent ;
self - > unsent . sent ( sent ) ;
}
if ( self - > unsent . empty ( ) ) break ;
TEST ( true ) ; // We didn't write everything, so apparently the write buffer is full. Wait for it to be nonfull.
2018-08-11 04:57:10 +08:00
wait ( conn - > onWritable ( ) ) ;
2019-06-25 17:47:35 +08:00
wait ( yield ( TaskPriority : : WriteSocket ) ) ;
2017-05-26 04:48:44 +08:00
}
// Wait until there is something to send
while ( self - > unsent . empty ( ) )
2018-08-11 04:57:10 +08:00
wait ( self - > dataToSend . onTrigger ( ) ) ;
2017-05-26 04:48:44 +08:00
}
}
2019-01-25 09:28:26 +08:00
ACTOR static Future < Void > connectionKeeper ( Peer * self ,
Reference < IConnection > conn = Reference < IConnection > ( ) ,
2017-05-26 04:48:44 +08:00
Future < Void > reader = Void ( ) ) {
2018-12-18 03:25:44 +08:00
TraceEvent ( SevDebug , " ConnectionKeeper " , conn ? conn - > getDebugID ( ) : UID ( ) )
2017-05-26 04:48:44 +08:00
. detail ( " PeerAddr " , self - > destination )
. detail ( " ConnSet " , ( bool ) conn ) ;
2019-05-22 07:39:08 +08:00
2019-05-30 04:43:21 +08:00
// This is used only at client side and is used to override waiting for unsent data to update failure monitoring
// status. At client, if an existing connection fails, we retry making a connection and if that fails, then only
// we report that address as failed.
2019-05-22 07:39:08 +08:00
state bool clientReconnectDelay = false ;
2017-05-26 04:48:44 +08:00
loop {
try {
if ( ! conn ) { // Always, except for the first loop with an incoming connection
self - > outgoingConnectionIdle = true ;
2019-05-21 02:54:46 +08:00
2019-05-22 07:39:08 +08:00
// Wait until there is something to send.
2019-05-30 04:43:21 +08:00
while ( self - > unsent . empty ( ) ) {
2019-06-12 09:58:00 +08:00
if ( FlowTransport : : transport ( ) . isClient ( ) & & self - > destination . isPublic ( ) & &
clientReconnectDelay ) {
break ;
2019-05-21 02:54:46 +08:00
}
2019-06-12 09:58:00 +08:00
wait ( self - > dataToSend . onTrigger ( ) ) ;
2019-05-21 02:54:46 +08:00
}
2017-05-26 04:48:44 +08:00
ASSERT ( self - > destination . isPublic ( ) ) ;
self - > outgoingConnectionIdle = false ;
2019-06-12 09:58:00 +08:00
wait ( delayJittered (
std : : max ( 0.0 , self - > lastConnectTime + self - > reconnectionDelay -
now ( ) ) ) ) ; // Don't connect() to the same peer more than once per 2 sec
2017-05-26 04:48:44 +08:00
self - > lastConnectTime = now ( ) ;
2018-08-02 05:30:57 +08:00
TraceEvent ( " ConnectingTo " , conn ? conn - > getDebugID ( ) : UID ( ) ) . suppressFor ( 1.0 ) . detail ( " PeerAddr " , self - > destination ) ;
2017-05-26 04:48:44 +08:00
Reference < IConnection > _conn = wait ( timeout ( INetworkConnections : : net ( ) - > connect ( self - > destination ) , FLOW_KNOBS - > CONNECTION_MONITOR_TIMEOUT , Reference < IConnection > ( ) ) ) ;
if ( _conn ) {
2019-05-21 02:54:46 +08:00
if ( FlowTransport : : transport ( ) . isClient ( ) ) {
IFailureMonitor : : failureMonitor ( ) . setStatus ( self - > destination , FailureStatus ( false ) ) ;
}
if ( self - > unsent . empty ( ) ) {
_conn - > close ( ) ;
2019-05-22 07:39:08 +08:00
clientReconnectDelay = false ;
2019-05-21 02:54:46 +08:00
continue ;
} else {
conn = _conn ;
2019-05-30 04:43:21 +08:00
TraceEvent ( " ConnectionExchangingConnectPacket " , conn - > getDebugID ( ) )
. suppressFor ( 1.0 )
. detail ( " PeerAddr " , self - > destination ) ;
2019-05-21 02:54:46 +08:00
self - > prependConnectPacket ( ) ;
}
2017-05-26 04:48:44 +08:00
} else {
2018-08-02 05:30:57 +08:00
TraceEvent ( " ConnectionTimedOut " , conn ? conn - > getDebugID ( ) : UID ( ) ) . suppressFor ( 1.0 ) . detail ( " PeerAddr " , self - > destination ) ;
2019-05-22 07:39:08 +08:00
if ( FlowTransport : : transport ( ) . isClient ( ) ) {
IFailureMonitor : : failureMonitor ( ) . setStatus ( self - > destination , FailureStatus ( true ) ) ;
}
2017-05-26 04:48:44 +08:00
throw connection_failed ( ) ;
}
2017-07-18 01:40:36 +08:00
reader = connectionReader ( self - > transport , conn , self , Promise < Peer * > ( ) ) ;
2017-05-26 04:48:44 +08:00
} else {
self - > outgoingConnectionIdle = false ;
}
2017-10-04 01:53:24 +08:00
try {
self - > transport - > countConnEstablished + + ;
2018-08-11 04:57:10 +08:00
wait ( connectionWriter ( self , conn ) | | reader | | connectionMonitor ( self ) ) ;
2017-10-04 01:53:24 +08:00
} catch ( Error & e ) {
2019-05-30 04:43:21 +08:00
if ( e . code ( ) = = error_code_connection_failed | | e . code ( ) = = error_code_actor_cancelled | |
e . code ( ) = = error_code_connection_unreferenced | |
( g_network - > isSimulated ( ) & & e . code ( ) = = error_code_checksum_failed ) )
2017-10-04 01:53:24 +08:00
self - > transport - > countConnClosedWithoutError + + ;
else
self - > transport - > countConnClosedWithError + + ;
throw e ;
}
2017-05-26 04:48:44 +08:00
ASSERT ( false ) ;
} catch ( Error & e ) {
if ( now ( ) - self - > lastConnectTime > FLOW_KNOBS - > RECONNECTION_RESET_TIME ) {
self - > reconnectionDelay = FLOW_KNOBS - > INITIAL_RECONNECTION_TIME ;
} else {
self - > reconnectionDelay = std : : min ( FLOW_KNOBS - > MAX_RECONNECTION_TIME , self - > reconnectionDelay * FLOW_KNOBS - > RECONNECTION_TIME_GROWTH_RATE ) ;
}
self - > discardUnreliablePackets ( ) ;
reader = Future < Void > ( ) ;
2019-05-30 04:43:21 +08:00
bool ok = e . code ( ) = = error_code_connection_failed | | e . code ( ) = = error_code_actor_cancelled | |
2019-06-28 15:39:51 +08:00
e . code ( ) = = error_code_connection_unreferenced | | e . code ( ) = = error_code_connection_idle | |
2019-05-30 04:43:21 +08:00
( g_network - > isSimulated ( ) & & e . code ( ) = = error_code_checksum_failed ) ;
2017-05-26 04:48:44 +08:00
2017-07-18 01:40:36 +08:00
if ( self - > compatible ) {
2019-06-28 15:39:51 +08:00
TraceEvent ( ok ? SevInfo : SevWarnAlways , " ConnectionClosed " , conn ? conn - > getDebugID ( ) : UID ( ) )
. error ( e , true )
. suppressFor ( 1.0 )
. detail ( " PeerAddr " , self - > destination ) ;
2017-07-18 01:40:36 +08:00
}
2017-07-18 07:15:18 +08:00
else {
2019-06-28 15:39:51 +08:00
TraceEvent ( ok ? SevInfo : SevWarnAlways , " IncompatibleConnectionClosed " ,
conn ? conn - > getDebugID ( ) : UID ( ) )
. error ( e , true )
. suppressFor ( 1.0 )
. detail ( " PeerAddr " , self - > destination ) ;
2017-07-18 07:15:18 +08:00
}
2017-05-26 04:48:44 +08:00
2019-04-05 05:11:12 +08:00
if ( self - > destination . isPublic ( ) & & IFailureMonitor : : failureMonitor ( ) . getState ( self - > destination ) . isAvailable ( ) ) {
auto & it = self - > transport - > closedPeers [ self - > destination ] ;
if ( now ( ) - it . second > FLOW_KNOBS - > TOO_MANY_CONNECTIONS_CLOSED_RESET_DELAY ) {
it . first = now ( ) ;
} else if ( now ( ) - it . first > FLOW_KNOBS - > TOO_MANY_CONNECTIONS_CLOSED_TIMEOUT ) {
2019-06-28 15:39:51 +08:00
TraceEvent ( SevWarnAlways , " TooManyConnectionsClosed " , conn ? conn - > getDebugID ( ) : UID ( ) )
. suppressFor ( 5.0 )
. detail ( " PeerAddr " , self - > destination ) ;
2019-04-05 05:11:12 +08:00
self - > transport - > degraded - > set ( true ) ;
}
it . second = now ( ) ;
}
2017-05-26 04:48:44 +08:00
if ( conn ) {
2019-06-28 15:39:51 +08:00
if ( FlowTransport : : transport ( ) . isClient ( ) & & e . code ( ) ! = error_code_connection_idle ) {
2019-05-22 07:39:08 +08:00
clientReconnectDelay = true ;
}
2017-05-26 04:48:44 +08:00
conn - > close ( ) ;
conn = Reference < IConnection > ( ) ;
}
2019-06-28 15:39:51 +08:00
// Clients might send more packets in response, which needs to go out on the next connection
IFailureMonitor : : failureMonitor ( ) . notifyDisconnect ( self - > destination ) ;
2017-05-26 04:48:44 +08:00
if ( e . code ( ) = = error_code_actor_cancelled ) throw ;
// Try to recover, even from serious errors, by retrying
2018-07-09 01:26:41 +08:00
2018-07-11 04:10:29 +08:00
if ( self - > peerReferences < = 0 & & self - > reliable . empty ( ) & & self - > unsent . empty ( ) ) {
2018-08-02 05:30:57 +08:00
TraceEvent ( " PeerDestroy " ) . error ( e ) . suppressFor ( 1.0 ) . detail ( " PeerAddr " , self - > destination ) ;
2018-07-09 01:26:41 +08:00
self - > connect . cancel ( ) ;
self - > transport - > peers . erase ( self - > destination ) ;
delete self ;
return Void ( ) ;
2018-07-09 01:26:41 +08:00
}
2017-05-26 04:48:44 +08:00
}
}
}
} ;
2018-07-28 11:46:54 +08:00
TransportData : : ~ TransportData ( ) {
for ( auto & p : peers ) {
p . second - > connect . cancel ( ) ;
delete p . second ;
}
}
2019-04-12 04:24:00 +08:00
ACTOR static void deliver ( TransportData * self , Endpoint destination , ArenaReader reader , bool inReadSocket ) {
2019-06-25 17:47:35 +08:00
TaskPriority priority = self - > endpoints . getPriority ( destination . token ) ;
if ( priority < TaskPriority : : ReadSocket | | ! inReadSocket ) {
2018-08-11 04:57:10 +08:00
wait ( delay ( 0 , priority ) ) ;
2017-05-26 04:48:44 +08:00
} else {
g_network - > setCurrentTask ( priority ) ;
}
auto receiver = self - > endpoints . get ( destination . token ) ;
if ( receiver ) {
try {
2019-01-07 19:38:35 +08:00
g_currentDeliveryPeerAddress = destination . addresses ;
2019-04-12 04:24:00 +08:00
if ( g_network - > useObjectSerializer ( ) ) {
StringRef data = reader . arenaReadAll ( ) ;
ASSERT ( data . size ( ) > 8 ) ;
2019-01-26 07:10:20 +08:00
ArenaObjectReader objReader ( reader . arena ( ) , reader . arenaReadAll ( ) ) ;
receiver - > receive ( objReader ) ;
} else {
2019-02-23 01:33:54 +08:00
receiver - > receive ( reader ) ;
}
g_currentDeliveryPeerAddress = { NetworkAddress ( ) } ;
2017-05-26 04:48:44 +08:00
} catch ( Error & e ) {
2019-01-07 19:38:35 +08:00
g_currentDeliveryPeerAddress = { NetworkAddress ( ) } ;
2018-10-31 04:44:37 +08:00
TraceEvent ( SevError , " ReceiverError " ) . error ( e ) . detail ( " Token " , destination . token . toString ( ) ) . detail ( " Peer " , destination . getPrimaryAddress ( ) ) ;
2017-05-26 04:48:44 +08:00
throw ;
}
} else if ( destination . token . first ( ) & TOKEN_STREAM_FLAG ) {
// We don't have the (stream) endpoint 'token', notify the remote machine
2019-03-02 03:45:46 +08:00
if ( destination . token . first ( ) ! = - 1 ) {
sendPacket ( self ,
2019-03-24 08:54:46 +08:00
SerializeSource < Endpoint > ( Endpoint ( self - > localAddresses , destination . token ) ) ,
2019-03-02 03:45:46 +08:00
Endpoint ( destination . addresses , WLTOKEN_ENDPOINT_NOT_FOUND ) , false , true ) ;
}
2017-05-26 04:48:44 +08:00
}
if ( inReadSocket )
2019-06-25 17:47:35 +08:00
g_network - > setCurrentTask ( TaskPriority : : ReadSocket ) ;
2017-05-26 04:48:44 +08:00
}
2019-06-12 07:44:00 +08:00
static void scanPackets ( TransportData * transport , uint8_t * & unprocessed_begin , const uint8_t * e , Arena & arena ,
2019-06-19 08:55:27 +08:00
NetworkAddress const & peerAddress , ProtocolVersion peerProtocolVersion ) {
2017-05-26 04:48:44 +08:00
// Find each complete packet in the given byte range and queue a ready task to deliver it.
// Remove the complete packets from the range by increasing unprocessed_begin.
// There won't be more than 64K of data plus one packet, so this shouldn't take a long time.
uint8_t * p = unprocessed_begin ;
2017-05-27 06:28:52 +08:00
2019-06-12 07:44:00 +08:00
const bool checksumEnabled = ! peerAddress . isTLS ( ) ;
2017-05-26 04:48:44 +08:00
loop {
2017-05-27 06:28:52 +08:00
uint32_t packetLen , packetChecksum ;
2017-05-26 04:48:44 +08:00
//Retrieve packet length and checksum
2017-05-27 06:28:52 +08:00
if ( checksumEnabled ) {
if ( e - p < sizeof ( uint32_t ) * 2 ) break ;
packetLen = * ( uint32_t * ) p ; p + = sizeof ( uint32_t ) ;
packetChecksum = * ( uint32_t * ) p ; p + = sizeof ( uint32_t ) ;
} else {
if ( e - p < sizeof ( uint32_t ) ) break ;
packetLen = * ( uint32_t * ) p ; p + = sizeof ( uint32_t ) ;
}
2017-05-26 04:48:44 +08:00
if ( packetLen > FLOW_KNOBS - > PACKET_LIMIT ) {
TraceEvent ( SevError , " Net2_PacketLimitExceeded " ) . detail ( " FromPeer " , peerAddress . toString ( ) ) . detail ( " Length " , ( int ) packetLen ) ;
throw platform_error ( ) ;
}
if ( e - p < packetLen ) break ;
ASSERT ( packetLen > = sizeof ( UID ) ) ;
2017-05-27 06:28:52 +08:00
if ( checksumEnabled ) {
bool isBuggifyEnabled = false ;
2017-09-19 07:54:49 +08:00
if ( g_network - > isSimulated ( ) & & g_network - > now ( ) - g_simulator . lastConnectionFailure > g_simulator . connectionFailuresDisableDuration & & BUGGIFY_WITH_PROB ( 0.0001 ) ) {
2017-09-19 03:46:29 +08:00
g_simulator . lastConnectionFailure = g_network - > now ( ) ;
2017-05-27 06:28:52 +08:00
isBuggifyEnabled = true ;
TraceEvent ( SevInfo , " BitsFlip " ) ;
2019-05-11 05:01:52 +08:00
int flipBits = 32 - ( int ) floor ( log2 ( deterministicRandom ( ) - > randomUInt32 ( ) ) ) ;
2017-05-27 06:28:52 +08:00
2019-05-11 05:01:52 +08:00
uint32_t firstFlipByteLocation = deterministicRandom ( ) - > randomUInt32 ( ) % packetLen ;
int firstFlipBitLocation = deterministicRandom ( ) - > randomInt ( 0 , 8 ) ;
2017-05-27 06:28:52 +08:00
* ( p + firstFlipByteLocation ) ^ = 1 < < firstFlipBitLocation ;
flipBits - - ;
for ( int i = 0 ; i < flipBits ; i + + ) {
2019-05-11 05:01:52 +08:00
uint32_t byteLocation = deterministicRandom ( ) - > randomUInt32 ( ) % packetLen ;
int bitLocation = deterministicRandom ( ) - > randomInt ( 0 , 8 ) ;
2017-05-27 06:28:52 +08:00
if ( byteLocation ! = firstFlipByteLocation | | bitLocation ! = firstFlipBitLocation ) {
* ( p + byteLocation ) ^ = 1 < < bitLocation ;
}
2017-05-26 04:48:44 +08:00
}
}
2017-05-27 06:28:52 +08:00
uint32_t calculatedChecksum = crc32c_append ( 0 , p , packetLen ) ;
if ( calculatedChecksum ! = packetChecksum ) {
if ( isBuggifyEnabled ) {
2018-06-09 02:11:08 +08:00
TraceEvent ( SevInfo , " ChecksumMismatchExp " ) . detail ( " PacketChecksum " , ( int ) packetChecksum ) . detail ( " CalculatedChecksum " , ( int ) calculatedChecksum ) ;
2017-05-27 06:28:52 +08:00
} else {
2018-06-09 02:11:08 +08:00
TraceEvent ( SevWarnAlways , " ChecksumMismatchUnexp " ) . detail ( " PacketChecksum " , ( int ) packetChecksum ) . detail ( " CalculatedChecksum " , ( int ) calculatedChecksum ) ;
2017-05-27 06:28:52 +08:00
}
throw checksum_failed ( ) ;
2017-05-26 04:48:44 +08:00
} else {
2017-05-27 06:28:52 +08:00
if ( isBuggifyEnabled ) {
2018-06-09 02:11:08 +08:00
TraceEvent ( SevError , " ChecksumMatchUnexp " ) . detail ( " PacketChecksum " , ( int ) packetChecksum ) . detail ( " CalculatedChecksum " , ( int ) calculatedChecksum ) ;
2017-05-27 06:28:52 +08:00
}
2017-05-26 04:48:44 +08:00
}
}
2017-10-28 07:54:44 +08:00
# if VALGRIND
VALGRIND_CHECK_MEM_IS_DEFINED ( p , packetLen ) ;
# endif
2019-04-12 04:24:00 +08:00
ArenaReader reader ( arena , StringRef ( p , packetLen ) , AssumeVersion ( currentProtocolVersion ) ) ;
2019-01-26 07:10:20 +08:00
UID token ;
reader > > token ;
2017-05-26 04:48:44 +08:00
+ + transport - > countPacketsReceived ;
2018-02-16 06:54:35 +08:00
if ( packetLen > FLOW_KNOBS - > PACKET_WARNING ) {
TraceEvent ( transport - > warnAlwaysForLargePacket ? SevWarnAlways : SevWarn , " Net2_LargePacket " )
2018-08-02 05:30:57 +08:00
. suppressFor ( 1.0 )
2018-02-16 06:54:35 +08:00
. detail ( " FromPeer " , peerAddress . toString ( ) )
. detail ( " Length " , ( int ) packetLen )
2018-08-02 05:30:57 +08:00
. detail ( " Token " , token ) ;
2018-02-16 06:54:35 +08:00
if ( g_network - > isSimulated ( ) )
transport - > warnAlwaysForLargePacket = false ;
}
2019-04-12 04:24:00 +08:00
ASSERT ( ! reader . empty ( ) ) ;
deliver ( transport , Endpoint ( { peerAddress } , token ) , std : : move ( reader ) , true ) ;
2017-05-26 04:48:44 +08:00
unprocessed_begin = p = p + packetLen ;
}
}
2019-06-12 07:44:00 +08:00
// Given unprocessed buffer [begin, end), check if next packet size is known and return
2019-06-26 00:52:05 +08:00
// enough size for the next packet, whose format is: {size, optional_checksum, data} +
// next_packet_size.
2019-06-14 02:56:06 +08:00
static int getNewBufferSize ( const uint8_t * begin , const uint8_t * end , const NetworkAddress & peerAddress ) {
2019-06-12 07:44:00 +08:00
const int len = end - begin ;
if ( len < sizeof ( uint32_t ) ) {
2019-06-22 01:51:20 +08:00
return FLOW_KNOBS - > MIN_PACKET_BUFFER_BYTES ;
2019-06-12 07:44:00 +08:00
}
const uint32_t packetLen = * ( uint32_t * ) begin ;
if ( packetLen > FLOW_KNOBS - > PACKET_LIMIT ) {
TraceEvent ( SevError , " Net2_PacketLimitExceeded " ) . detail ( " FromPeer " , peerAddress . toString ( ) ) . detail ( " Length " , ( int ) packetLen ) ;
throw platform_error ( ) ;
}
2019-06-12 08:52:25 +08:00
return std : : max < uint32_t > ( FLOW_KNOBS - > MIN_PACKET_BUFFER_BYTES ,
2019-06-26 00:52:05 +08:00
packetLen + sizeof ( uint32_t ) * ( peerAddress . isTLS ( ) ? 2 : 3 ) ) ;
2019-06-12 07:44:00 +08:00
}
2017-05-26 04:48:44 +08:00
ACTOR static Future < Void > connectionReader (
TransportData * transport ,
2019-01-25 09:28:26 +08:00
Reference < IConnection > conn ,
2017-07-18 01:40:36 +08:00
Peer * peer ,
2019-01-25 09:28:26 +08:00
Promise < Peer * > onConnected )
2017-07-18 01:40:36 +08:00
{
2017-05-26 04:48:44 +08:00
// This actor exists whenever there is an open or opening connection, whether incoming or outgoing
2019-06-12 07:44:00 +08:00
// For incoming connections conn is set and peer is initially nullptr; for outgoing connections it is the reverse
2017-05-26 04:48:44 +08:00
state Arena arena ;
2019-06-12 07:44:00 +08:00
state uint8_t * unprocessed_begin = nullptr ;
state uint8_t * unprocessed_end = nullptr ;
state uint8_t * buffer_end = nullptr ;
2017-05-26 04:48:44 +08:00
state bool expectConnectPacket = true ;
2017-07-18 01:40:36 +08:00
state bool compatible = false ;
2018-07-18 02:36:05 +08:00
state bool incompatiblePeerCounted = false ;
2018-07-28 11:42:06 +08:00
state bool incompatibleProtocolVersionNewer = false ;
2017-05-26 04:48:44 +08:00
state NetworkAddress peerAddress ;
2019-06-19 08:55:27 +08:00
state ProtocolVersion peerProtocolVersion ;
2017-05-26 04:48:44 +08:00
peerAddress = conn - > getPeerAddress ( ) ;
2018-10-31 04:44:37 +08:00
if ( peer = = nullptr ) {
2017-05-26 04:48:44 +08:00
ASSERT ( ! peerAddress . isPublic ( ) ) ;
2017-07-18 01:40:36 +08:00
}
2017-11-14 07:07:39 +08:00
try {
2017-05-26 04:48:44 +08:00
loop {
2017-11-14 07:07:39 +08:00
loop {
2019-06-15 06:07:57 +08:00
state int readAllBytes = buffer_end - unprocessed_end ;
2019-06-26 00:52:05 +08:00
if ( readAllBytes < FLOW_KNOBS - > MIN_PACKET_BUFFER_FREE_BYTES ) {
2017-11-14 07:07:39 +08:00
Arena newArena ;
2019-06-12 07:44:00 +08:00
const int unproc_len = unprocessed_end - unprocessed_begin ;
const int len = getNewBufferSize ( unprocessed_begin , unprocessed_end , peerAddress ) ;
uint8_t * const newBuffer = new ( newArena ) uint8_t [ len ] ;
2017-11-14 07:07:39 +08:00
memcpy ( newBuffer , unprocessed_begin , unproc_len ) ;
arena = newArena ;
unprocessed_begin = newBuffer ;
unprocessed_end = newBuffer + unproc_len ;
buffer_end = newBuffer + len ;
readAllBytes = buffer_end - unprocessed_end ;
}
2017-05-26 04:48:44 +08:00
2019-06-15 06:07:57 +08:00
state int totalReadBytes = 0 ;
while ( true ) {
const int len = std : : min < int > ( buffer_end - unprocessed_end , FLOW_KNOBS - > MAX_PACKET_SEND_BYTES ) ;
if ( len = = 0 ) break ;
state int readBytes = conn - > read ( unprocessed_end , unprocessed_end + len ) ;
if ( readBytes = = 0 ) break ;
2019-06-29 08:32:54 +08:00
wait ( yield ( TaskPriority : : ReadSocket ) ) ;
2019-06-15 06:07:57 +08:00
totalReadBytes + = readBytes ;
unprocessed_end + = readBytes ;
2019-05-17 08:26:48 +08:00
}
2019-06-15 06:07:57 +08:00
if ( peer ) {
peer - > bytesReceived + = totalReadBytes ;
2019-05-17 08:26:48 +08:00
}
2019-06-15 06:07:57 +08:00
if ( totalReadBytes = = 0 ) break ;
state bool readWillBlock = totalReadBytes ! = readAllBytes ;
2018-10-31 04:44:37 +08:00
2017-11-14 07:07:39 +08:00
if ( expectConnectPacket & & unprocessed_end - unprocessed_begin > = CONNECT_PACKET_V0_SIZE ) {
// At the beginning of a connection, we expect to receive a packet containing the protocol version and the listening port of the remote process
2019-03-01 08:07:49 +08:00
int32_t connectPacketSize = ( ( ConnectPacket * ) unprocessed_begin ) - > totalPacketSize ( ) ;
2017-11-14 07:07:39 +08:00
if ( unprocessed_end - unprocessed_begin > = connectPacketSize ) {
2019-06-19 05:49:04 +08:00
auto protocolVersion = ( ( ConnectPacket * ) unprocessed_begin ) - > protocolVersion ;
2019-03-01 08:07:49 +08:00
BinaryReader pktReader ( unprocessed_begin , connectPacketSize , AssumeVersion ( protocolVersion ) ) ;
ConnectPacket pkt ;
serializer ( pktReader , pkt ) ;
uint64_t connectionId = pkt . connectionId ;
2019-06-19 05:49:04 +08:00
if ( g_network - > useObjectSerializer ( ) ! = pkt . protocolVersion . hasObjectSerializerFlag ( ) | |
! pkt . protocolVersion . isCompatible ( currentProtocolVersion ) ) {
2019-03-01 08:07:49 +08:00
incompatibleProtocolVersionNewer = pkt . protocolVersion > currentProtocolVersion ;
NetworkAddress addr = pkt . canonicalRemotePort
? NetworkAddress ( pkt . canonicalRemoteIp ( ) , pkt . canonicalRemotePort )
2019-02-27 10:04:03 +08:00
: conn - > getPeerAddress ( ) ;
2017-11-14 07:07:39 +08:00
if ( connectionId ! = 1 ) addr . port = 0 ;
2018-10-31 04:44:37 +08:00
2017-11-14 07:07:39 +08:00
if ( ! transport - > multiVersionConnections . count ( connectionId ) ) {
if ( now ( ) - transport - > lastIncompatibleMessage > FLOW_KNOBS - > CONNECTION_REJECTED_MESSAGE_DELAY ) {
TraceEvent ( SevWarn , " ConnectionRejected " , conn - > getDebugID ( ) )
2019-02-27 10:04:03 +08:00
. detail ( " Reason " , " IncompatibleProtocolVersion " )
2019-06-19 05:49:04 +08:00
. detail ( " LocalVersion " , currentProtocolVersion . version ( ) )
. detail ( " RejectedVersion " , pkt . protocolVersion . version ( ) )
. detail ( " VersionMask " , ProtocolVersion : : compatibleProtocolVersionMask )
2019-03-01 08:07:49 +08:00
. detail ( " Peer " , pkt . canonicalRemotePort ? NetworkAddress ( pkt . canonicalRemoteIp ( ) , pkt . canonicalRemotePort )
: conn - > getPeerAddress ( ) )
2019-02-27 10:04:03 +08:00
. detail ( " ConnectionId " , connectionId ) ;
2017-11-14 07:07:39 +08:00
transport - > lastIncompatibleMessage = now ( ) ;
}
if ( ! transport - > incompatiblePeers . count ( addr ) ) {
transport - > incompatiblePeers [ addr ] = std : : make_pair ( connectionId , now ( ) ) ;
}
} else if ( connectionId > 1 ) {
transport - > multiVersionConnections [ connectionId ] = now ( ) + FLOW_KNOBS - > CONNECTION_ID_TIMEOUT ;
2017-05-26 04:48:44 +08:00
}
2017-11-14 07:07:39 +08:00
compatible = false ;
2019-06-19 05:49:04 +08:00
if ( ! protocolVersion . hasMultiVersionClient ( ) ) {
2017-11-14 07:07:39 +08:00
// Older versions expected us to hang up. It may work even if we don't hang up here, but it's safer to keep the old behavior.
throw incompatible_protocol_version ( ) ;
2017-05-26 04:48:44 +08:00
}
}
2017-11-14 07:07:39 +08:00
else {
compatible = true ;
TraceEvent ( " ConnectionEstablished " , conn - > getDebugID ( ) )
2018-08-02 05:30:57 +08:00
. suppressFor ( 1.0 )
2017-11-14 07:07:39 +08:00
. detail ( " Peer " , conn - > getPeerAddress ( ) )
2019-01-26 07:10:20 +08:00
. detail ( " ConnectionId " , connectionId )
. detail ( " UseObjectSerializer " , false ) ;
2017-07-18 01:40:36 +08:00
}
2017-05-26 04:48:44 +08:00
2017-11-14 07:07:39 +08:00
if ( connectionId > 1 ) {
transport - > multiVersionConnections [ connectionId ] = now ( ) + FLOW_KNOBS - > CONNECTION_ID_TIMEOUT ;
}
unprocessed_begin + = connectPacketSize ;
expectConnectPacket = false ;
if ( peer ! = nullptr ) {
2019-04-12 04:24:00 +08:00
peerProtocolVersion = protocolVersion ;
2017-11-14 07:07:39 +08:00
// Outgoing connection; port information should be what we expect
2019-02-27 10:04:03 +08:00
TraceEvent ( " ConnectedOutgoing " )
. suppressFor ( 1.0 )
2019-03-01 08:07:49 +08:00
. detail ( " PeerAddr " , NetworkAddress ( pkt . canonicalRemoteIp ( ) , pkt . canonicalRemotePort ) ) ;
2017-11-14 07:07:39 +08:00
peer - > compatible = compatible ;
2018-07-28 11:42:06 +08:00
peer - > incompatibleProtocolVersionNewer = incompatibleProtocolVersionNewer ;
2018-07-18 02:36:05 +08:00
if ( ! compatible ) {
2017-11-15 10:37:29 +08:00
peer - > transport - > numIncompatibleConnections + + ;
2018-07-18 02:36:05 +08:00
incompatiblePeerCounted = true ;
}
2019-03-01 08:07:49 +08:00
ASSERT ( pkt . canonicalRemotePort = = peerAddress . port ) ;
2019-04-12 04:24:00 +08:00
onConnected . send ( peer ) ;
2017-11-14 07:07:39 +08:00
} else {
2019-04-12 04:24:00 +08:00
peerProtocolVersion = protocolVersion ;
2019-03-01 08:07:49 +08:00
if ( pkt . canonicalRemotePort ) {
peerAddress = NetworkAddress ( pkt . canonicalRemoteIp ( ) , pkt . canonicalRemotePort , true ,
2019-02-27 10:04:03 +08:00
peerAddress . isTLS ( ) ) ;
2017-11-14 07:07:39 +08:00
}
peer = transport - > getPeer ( peerAddress ) ;
peer - > compatible = compatible ;
2018-07-28 11:42:06 +08:00
peer - > incompatibleProtocolVersionNewer = incompatibleProtocolVersionNewer ;
2018-07-18 02:36:05 +08:00
if ( ! compatible ) {
2017-11-15 10:37:29 +08:00
peer - > transport - > numIncompatibleConnections + + ;
2018-07-18 02:36:05 +08:00
incompatiblePeerCounted = true ;
}
2017-11-14 07:07:39 +08:00
onConnected . send ( peer ) ;
2018-08-11 04:57:10 +08:00
wait ( delay ( 0 ) ) ; // Check for cancellation
2017-05-26 04:48:44 +08:00
}
}
}
2017-11-14 07:07:39 +08:00
if ( compatible ) {
scanPackets ( transport , unprocessed_begin , unprocessed_end , arena , peerAddress , peerProtocolVersion ) ;
}
else if ( ! expectConnectPacket ) {
unprocessed_begin = unprocessed_end ;
2019-05-17 08:26:48 +08:00
peer - > resetPing . trigger ( ) ;
2017-11-14 07:07:39 +08:00
}
2017-05-26 04:48:44 +08:00
2017-11-14 07:07:39 +08:00
if ( readWillBlock )
break ;
2017-05-26 04:48:44 +08:00
2019-06-25 17:47:35 +08:00
wait ( yield ( TaskPriority : : ReadSocket ) ) ;
2017-11-14 07:07:39 +08:00
}
2017-05-26 04:48:44 +08:00
2018-08-11 04:57:10 +08:00
wait ( conn - > onReadable ( ) ) ;
2019-06-25 17:47:35 +08:00
wait ( delay ( 0 , TaskPriority : : ReadSocket ) ) ; // We don't want to call conn->read directly from the reactor - we could get stuck in the reactor reading 1 packet at a time
2017-11-14 07:07:39 +08:00
}
}
catch ( Error & e ) {
2018-07-18 02:36:05 +08:00
if ( incompatiblePeerCounted ) {
ASSERT ( peer & & peer - > transport - > numIncompatibleConnections > 0 ) ;
2017-11-15 10:37:29 +08:00
peer - > transport - > numIncompatibleConnections - - ;
2017-11-14 07:07:39 +08:00
}
throw ;
2017-05-26 04:48:44 +08:00
}
}
ACTOR static Future < Void > connectionIncoming ( TransportData * self , Reference < IConnection > conn ) {
try {
2017-07-18 01:40:36 +08:00
state Promise < Peer * > onConnected ;
state Future < Void > reader = connectionReader ( self , conn , nullptr , onConnected ) ;
2017-05-26 04:48:44 +08:00
choose {
2018-08-11 04:57:10 +08:00
when ( wait ( reader ) ) { ASSERT ( false ) ; return Void ( ) ; }
2017-07-18 01:40:36 +08:00
when ( Peer * p = wait ( onConnected . getFuture ( ) ) ) {
2017-05-26 04:48:44 +08:00
p - > onIncomingConnection ( conn , reader ) ;
}
2018-08-11 04:57:10 +08:00
when ( wait ( delayJittered ( FLOW_KNOBS - > CONNECTION_MONITOR_TIMEOUT ) ) ) {
2017-05-26 04:48:44 +08:00
TEST ( true ) ; // Incoming connection timed out
throw timed_out ( ) ;
}
}
return Void ( ) ;
} catch ( Error & e ) {
2018-08-02 05:30:57 +08:00
TraceEvent ( " IncomingConnectionError " , conn - > getDebugID ( ) ) . error ( e ) . suppressFor ( 1.0 ) . detail ( " FromAddress " , conn - > getPeerAddress ( ) ) ;
2017-05-26 04:48:44 +08:00
conn - > close ( ) ;
return Void ( ) ;
}
}
ACTOR static Future < Void > listen ( TransportData * self , NetworkAddress listenAddr ) {
state ActorCollectionNoErrors incoming ; // Actors monitoring incoming connections that haven't yet been associated with a peer
state Reference < IListener > listener = INetworkConnections : : net ( ) - > listen ( listenAddr ) ;
try {
loop {
Reference < IConnection > conn = wait ( listener - > accept ( ) ) ;
2018-12-18 03:25:44 +08:00
TraceEvent ( " ConnectionFrom " , conn - > getDebugID ( ) ) . suppressFor ( 1.0 )
. detail ( " FromAddress " , conn - > getPeerAddress ( ) )
. detail ( " ListenAddress " , listenAddr . toString ( ) ) ;
2017-05-26 04:48:44 +08:00
incoming . add ( connectionIncoming ( self , conn ) ) ;
2019-06-25 17:47:35 +08:00
wait ( delay ( 0 ) | | delay ( FLOW_KNOBS - > CONNECTION_ACCEPT_DELAY , TaskPriority : : WriteSocket ) ) ;
2017-05-26 04:48:44 +08:00
}
} catch ( Error & e ) {
TraceEvent ( SevError , " ListenError " ) . error ( e ) ;
throw ;
}
}
2018-07-09 01:26:41 +08:00
Peer * TransportData : : getPeer ( NetworkAddress const & address , bool openConnection ) {
auto peer = peers . find ( address ) ;
if ( peer ! = peers . end ( ) ) {
return peer - > second ;
}
if ( ! openConnection ) {
2019-06-12 07:44:00 +08:00
return nullptr ;
2018-07-09 01:26:41 +08:00
}
Peer * newPeer = new Peer ( this , address ) ;
peers [ address ] = newPeer ;
return newPeer ;
2017-05-26 04:48:44 +08:00
}
2018-10-31 04:44:37 +08:00
bool TransportData : : isLocalAddress ( const NetworkAddress & address ) const {
2019-03-24 08:54:46 +08:00
return address = = localAddresses . address | | ( localAddresses . secondaryAddress . present ( ) & & address = = localAddresses . secondaryAddress . get ( ) ) ;
2018-10-31 04:44:37 +08:00
}
2017-05-26 04:48:44 +08:00
ACTOR static Future < Void > multiVersionCleanupWorker ( TransportData * self ) {
loop {
2018-08-11 04:57:10 +08:00
wait ( delay ( FLOW_KNOBS - > CONNECTION_CLEANUP_DELAY ) ) ;
2017-05-26 04:48:44 +08:00
for ( auto it = self - > incompatiblePeers . begin ( ) ; it ! = self - > incompatiblePeers . end ( ) ; ) {
if ( self - > multiVersionConnections . count ( it - > second . first ) ) {
it = self - > incompatiblePeers . erase ( it ) ;
} else {
it + + ;
}
}
for ( auto it = self - > multiVersionConnections . begin ( ) ; it ! = self - > multiVersionConnections . end ( ) ; ) {
if ( it - > second < now ( ) ) {
it = self - > multiVersionConnections . erase ( it ) ;
} else {
it + + ;
}
}
}
}
FlowTransport : : FlowTransport ( uint64_t transportId ) : self ( new TransportData ( transportId ) ) {
self - > multiVersionCleanup = multiVersionCleanupWorker ( self ) ;
}
FlowTransport : : ~ FlowTransport ( ) { delete self ; }
void FlowTransport : : initMetrics ( ) { self - > initMetrics ( ) ; }
2019-02-01 10:20:14 +08:00
NetworkAddressList FlowTransport : : getLocalAddresses ( ) const {
return self - > localAddresses ;
}
2018-10-31 04:44:37 +08:00
NetworkAddress FlowTransport : : getLocalAddress ( ) const {
2019-03-24 08:54:46 +08:00
return self - > localAddresses . address ;
2018-10-31 04:44:37 +08:00
}
2017-05-26 04:48:44 +08:00
2018-10-31 04:44:37 +08:00
std : : map < NetworkAddress , std : : pair < uint64_t , double > > * FlowTransport : : getIncompatiblePeers ( ) {
2017-05-26 04:48:44 +08:00
for ( auto it = self - > incompatiblePeers . begin ( ) ; it ! = self - > incompatiblePeers . end ( ) ; ) {
if ( self - > multiVersionConnections . count ( it - > second . first ) ) {
it = self - > incompatiblePeers . erase ( it ) ;
} else {
it + + ;
}
}
2018-10-31 04:44:37 +08:00
return & self - > incompatiblePeers ;
2017-05-26 04:48:44 +08:00
}
Future < Void > FlowTransport : : bind ( NetworkAddress publicAddress , NetworkAddress listenAddress ) {
ASSERT ( publicAddress . isPublic ( ) ) ;
2019-03-24 08:54:46 +08:00
if ( self - > localAddresses . address = = NetworkAddress ( ) ) {
self - > localAddresses . address = publicAddress ;
} else {
self - > localAddresses . secondaryAddress = publicAddress ;
}
2017-05-26 04:48:44 +08:00
TraceEvent ( " Binding " ) . detail ( " PublicAddress " , publicAddress ) . detail ( " ListenAddress " , listenAddress ) ;
2018-10-31 04:44:37 +08:00
Future < Void > listenF = listen ( self , listenAddress ) ;
2019-02-01 10:20:14 +08:00
self - > listeners . push_back ( listenF ) ;
2018-10-31 04:44:37 +08:00
return listenF ;
2017-05-26 04:48:44 +08:00
}
2019-03-27 03:05:43 +08:00
Endpoint FlowTransport : : loadedEndpoint ( const UID & token ) {
return Endpoint ( g_currentDeliveryPeerAddress , token ) ;
2017-05-26 04:48:44 +08:00
}
2019-06-25 08:37:57 +08:00
void FlowTransport : : addPeerReference ( const Endpoint & endpoint , bool isStream ) {
2019-05-22 07:39:08 +08:00
if ( FlowTransport : : transport ( ) . isClient ( ) ) {
IFailureMonitor : : failureMonitor ( ) . setStatus ( endpoint . getPrimaryAddress ( ) , FailureStatus ( false ) ) ;
}
2019-06-25 08:37:57 +08:00
if ( ! isStream | | ! endpoint . getPrimaryAddress ( ) . isValid ( ) ) return ;
2019-02-01 10:20:14 +08:00
Peer * peer = self - > getPeer ( endpoint . getPrimaryAddress ( ) ) ;
2019-01-09 23:41:02 +08:00
if ( peer - > peerReferences = = - 1 ) {
peer - > peerReferences = 1 ;
} else {
peer - > peerReferences + + ;
2018-07-11 04:10:29 +08:00
}
}
2019-06-25 08:37:57 +08:00
void FlowTransport : : removePeerReference ( const Endpoint & endpoint , bool isStream ) {
if ( ! isStream | | ! endpoint . getPrimaryAddress ( ) . isValid ( ) ) return ;
2019-02-01 10:20:14 +08:00
Peer * peer = self - > getPeer ( endpoint . getPrimaryAddress ( ) , false ) ;
2019-01-09 23:41:02 +08:00
if ( peer ) {
peer - > peerReferences - - ;
if ( peer - > peerReferences < 0 ) {
TraceEvent ( SevError , " InvalidPeerReferences " )
. detail ( " References " , peer - > peerReferences )
. detail ( " Address " , endpoint . getPrimaryAddress ( ) )
. detail ( " Token " , endpoint . token ) ;
}
if ( peer - > peerReferences = = 0 & & peer - > reliable . empty ( ) & & peer - > unsent . empty ( ) ) {
2019-05-17 08:26:48 +08:00
peer - > resetPing . trigger ( ) ;
2018-07-11 04:10:29 +08:00
}
}
}
2019-06-25 17:47:35 +08:00
void FlowTransport : : addEndpoint ( Endpoint & endpoint , NetworkMessageReceiver * receiver , TaskPriority taskID ) {
2019-05-11 05:01:52 +08:00
endpoint . token = deterministicRandom ( ) - > randomUniqueID ( ) ;
2017-05-26 04:48:44 +08:00
if ( receiver - > isStream ( ) ) {
2019-03-24 08:54:46 +08:00
endpoint . addresses = self - > localAddresses ;
2017-05-26 04:48:44 +08:00
endpoint . token = UID ( endpoint . token . first ( ) | TOKEN_STREAM_FLAG , endpoint . token . second ( ) ) ;
} else {
2019-03-24 08:54:46 +08:00
endpoint . addresses = NetworkAddressList ( ) ;
2017-05-26 04:48:44 +08:00
endpoint . token = UID ( endpoint . token . first ( ) & ~ TOKEN_STREAM_FLAG , endpoint . token . second ( ) ) ;
}
2019-07-04 12:03:58 +08:00
self - > endpoints . insert ( receiver , endpoint . token , taskID ) ;
2017-05-26 04:48:44 +08:00
}
void FlowTransport : : removeEndpoint ( const Endpoint & endpoint , NetworkMessageReceiver * receiver ) {
self - > endpoints . remove ( endpoint . token , receiver ) ;
}
2019-06-25 17:47:35 +08:00
void FlowTransport : : addWellKnownEndpoint ( Endpoint & endpoint , NetworkMessageReceiver * receiver , TaskPriority taskID ) {
2019-03-24 08:54:46 +08:00
endpoint . addresses = self - > localAddresses ;
2017-05-26 04:48:44 +08:00
ASSERT ( ( ( endpoint . token . first ( ) & TOKEN_STREAM_FLAG ) ! = 0 ) = = receiver - > isStream ( ) ) ;
Endpoint : : Token otoken = endpoint . token ;
2019-07-04 12:03:58 +08:00
self - > endpoints . insert ( receiver , endpoint . token , taskID ) ;
2017-05-26 04:48:44 +08:00
ASSERT ( endpoint . token = = otoken ) ;
}
2018-07-09 01:26:41 +08:00
static PacketID sendPacket ( TransportData * self , ISerializeSource const & what , const Endpoint & destination , bool reliable , bool openConnection ) {
2018-10-31 04:44:37 +08:00
if ( self - > isLocalAddress ( destination . getPrimaryAddress ( ) ) ) {
2017-05-26 04:48:44 +08:00
TEST ( true ) ; // "Loopback" delivery
// SOMEDAY: Would it be better to avoid (de)serialization by doing this check in flow?
2019-01-31 05:53:23 +08:00
Standalone < StringRef > copy ;
if ( g_network - > useObjectSerializer ( ) ) {
ObjectWriter wr ;
2019-04-18 01:49:19 +08:00
what . serializeObjectWriter ( wr ) ;
2019-01-31 05:53:23 +08:00
copy = wr . toStringRef ( ) ;
} else {
2019-04-10 05:29:21 +08:00
BinaryWriter wr ( AssumeVersion ( currentProtocolVersion ) ) ;
what . serializeBinaryWriter ( wr ) ;
copy = wr . toValue ( ) ;
2019-01-31 05:53:23 +08:00
}
2017-10-28 07:54:44 +08:00
# if VALGRIND
VALGRIND_CHECK_MEM_IS_DEFINED ( copy . begin ( ) , copy . size ( ) ) ;
# endif
2017-05-26 04:48:44 +08:00
2019-04-12 04:24:00 +08:00
ASSERT ( copy . size ( ) > 0 ) ;
deliver ( self , destination , ArenaReader ( copy . arena ( ) , copy , AssumeVersion ( currentProtocolVersion ) ) , false ) ;
2017-05-26 04:48:44 +08:00
2019-06-12 07:44:00 +08:00
return ( PacketID ) nullptr ;
2017-05-26 04:48:44 +08:00
} else {
2019-06-12 07:44:00 +08:00
const bool checksumEnabled = ! destination . getPrimaryAddress ( ) . isTLS ( ) ;
2017-05-26 04:48:44 +08:00
+ + self - > countPacketsGenerated ;
2019-02-01 10:20:14 +08:00
Peer * peer = self - > getPeer ( destination . getPrimaryAddress ( ) , openConnection ) ;
2017-05-26 04:48:44 +08:00
2017-07-18 01:40:36 +08:00
// If there isn't an open connection, a public address, or the peer isn't compatible, we can't send
2018-10-31 04:44:37 +08:00
if ( ! peer | | ( peer - > outgoingConnectionIdle & & ! destination . getPrimaryAddress ( ) . isPublic ( ) ) | | ( peer - > incompatibleProtocolVersionNewer & & destination . token ! = WLTOKEN_PING_PACKET ) ) {
2017-07-18 01:40:36 +08:00
TEST ( true ) ; // Can't send to private address without a compatible open connection
2019-06-12 07:44:00 +08:00
return ( PacketID ) nullptr ;
2017-05-26 04:48:44 +08:00
}
bool firstUnsent = peer - > unsent . empty ( ) ;
PacketBuffer * pb = peer - > unsent . getWriteBuffer ( ) ;
ReliablePacket * rp = reliable ? new ReliablePacket : 0 ;
int prevBytesWritten = pb - > bytes_written ;
PacketBuffer * checksumPb = pb ;
PacketWriter wr ( pb , rp , AssumeVersion ( currentProtocolVersion ) ) ; // SOMEDAY: Can we downgrade to talk to older peers?
// Reserve some space for packet length and checksum, write them after serializing data
SplitBuffer packetInfoBuffer ;
uint32_t len , checksum = 0 ;
2017-05-27 06:28:52 +08:00
int packetInfoSize = sizeof ( len ) ;
if ( checksumEnabled ) {
packetInfoSize + = sizeof ( checksum ) ;
}
2017-05-26 04:48:44 +08:00
wr . writeAhead ( packetInfoSize , & packetInfoBuffer ) ;
wr < < destination . token ;
2019-01-26 07:10:20 +08:00
what . serializePacketWriter ( wr , g_network - > useObjectSerializer ( ) ) ;
2017-05-26 04:48:44 +08:00
pb = wr . finish ( ) ;
len = wr . size ( ) - packetInfoSize ;
2017-05-27 06:28:52 +08:00
if ( checksumEnabled ) {
// Find the correct place to start calculating checksum
uint32_t checksumUnprocessedLength = len ;
prevBytesWritten + = packetInfoSize ;
if ( prevBytesWritten > = PacketBuffer : : DATA_SIZE ) {
prevBytesWritten - = PacketBuffer : : DATA_SIZE ;
checksumPb = checksumPb - > nextPacketBuffer ( ) ;
}
2017-05-26 04:48:44 +08:00
2017-05-27 06:28:52 +08:00
// Checksum calculation
while ( checksumUnprocessedLength > 0 ) {
uint32_t processLength = std : : min ( checksumUnprocessedLength , ( uint32_t ) ( PacketBuffer : : DATA_SIZE - prevBytesWritten ) ) ;
checksum = crc32c_append ( checksum , checksumPb - > data + prevBytesWritten , processLength ) ;
checksumUnprocessedLength - = processLength ;
checksumPb = checksumPb - > nextPacketBuffer ( ) ;
prevBytesWritten = 0 ;
}
2017-05-26 04:48:44 +08:00
}
// Write packet length and checksum into packet buffer
packetInfoBuffer . write ( & len , sizeof ( len ) ) ;
2017-05-27 06:28:52 +08:00
if ( checksumEnabled ) {
packetInfoBuffer . write ( & checksum , sizeof ( checksum ) , sizeof ( len ) ) ;
}
2017-05-26 04:48:44 +08:00
if ( len > FLOW_KNOBS - > PACKET_LIMIT ) {
2018-10-31 04:44:37 +08:00
TraceEvent ( SevError , " Net2_PacketLimitExceeded " ) . detail ( " ToPeer " , destination . getPrimaryAddress ( ) ) . detail ( " Length " , ( int ) len ) ;
2017-05-26 04:48:44 +08:00
// throw platform_error(); // FIXME: How to recover from this situation?
2019-01-25 09:28:26 +08:00
}
2017-05-26 04:48:44 +08:00
else if ( len > FLOW_KNOBS - > PACKET_WARNING ) {
TraceEvent ( self - > warnAlwaysForLargePacket ? SevWarnAlways : SevWarn , " Net2_LargePacket " )
2018-08-02 05:30:57 +08:00
. suppressFor ( 1.0 )
2018-10-31 04:44:37 +08:00
. detail ( " ToPeer " , destination . getPrimaryAddress ( ) )
2017-05-26 04:48:44 +08:00
. detail ( " Length " , ( int ) len )
2018-02-16 06:54:35 +08:00
. detail ( " Token " , destination . token )
2018-08-02 05:30:57 +08:00
. backtrace ( ) ;
2017-05-26 04:48:44 +08:00
if ( g_network - > isSimulated ( ) )
self - > warnAlwaysForLargePacket = false ;
}
2017-10-28 07:54:44 +08:00
# if VALGRIND
SendBuffer * checkbuf = pb ;
while ( checkbuf ) {
int size = checkbuf - > bytes_written ;
const uint8_t * data = checkbuf - > data ;
VALGRIND_CHECK_MEM_IS_DEFINED ( data , size ) ;
checkbuf = checkbuf - > next ;
}
# endif
2017-05-26 04:48:44 +08:00
peer - > send ( pb , rp , firstUnsent ) ;
2019-06-29 04:43:22 +08:00
if ( destination . token ! = WLTOKEN_PING_PACKET ) {
peer - > lastDataPacketSentTime = now ( ) ;
}
2017-05-26 04:48:44 +08:00
return ( PacketID ) rp ;
}
}
PacketID FlowTransport : : sendReliable ( ISerializeSource const & what , const Endpoint & destination ) {
2018-07-09 01:26:41 +08:00
return sendPacket ( self , what , destination , true , true ) ;
2017-05-26 04:48:44 +08:00
}
void FlowTransport : : cancelReliable ( PacketID pid ) {
ReliablePacket * p = ( ReliablePacket * ) pid ;
if ( p ) p - > remove ( ) ;
// SOMEDAY: Call reliable.compact() if a lot of memory is wasted in PacketBuffers by formerly reliable packets mixed with a few reliable ones. Don't forget to delref the new PacketBuffers since they are unsent.
}
2018-07-09 01:26:41 +08:00
void FlowTransport : : sendUnreliable ( ISerializeSource const & what , const Endpoint & destination , bool openConnection ) {
sendPacket ( self , what , destination , false , openConnection ) ;
2017-05-26 04:48:44 +08:00
}
2019-01-25 09:28:26 +08:00
int FlowTransport : : getEndpointCount ( ) {
return - 1 ;
2017-05-26 04:48:44 +08:00
}
2019-04-05 05:11:12 +08:00
Reference < AsyncVar < bool > > FlowTransport : : getDegraded ( ) {
return self - > degraded ;
}
2017-11-10 03:20:35 +08:00
bool FlowTransport : : incompatibleOutgoingConnectionsPresent ( ) {
2019-03-16 01:34:57 +08:00
return self - > numIncompatibleConnections > 0 ;
2017-11-10 03:20:35 +08:00
}
2019-05-30 04:43:21 +08:00
void FlowTransport : : createInstance ( bool isClient , uint64_t transportId ) {
2019-06-12 09:58:00 +08:00
g_network - > setGlobal ( INetwork : : enFailureMonitor , ( flowGlobalType ) new SimpleFailureMonitor ( ) ) ;
g_network - > setGlobal ( INetwork : : enClientFailureMonitor , isClient ? ( flowGlobalType ) 1 : nullptr ) ;
2017-05-26 04:48:44 +08:00
g_network - > setGlobal ( INetwork : : enFlowTransport , ( flowGlobalType ) new FlowTransport ( transportId ) ) ;
g_network - > setGlobal ( INetwork : : enNetworkAddressFunc , ( flowGlobalType ) & FlowTransport : : getGlobalLocalAddress ) ;
2019-02-01 10:20:14 +08:00
g_network - > setGlobal ( INetwork : : enNetworkAddressesFunc , ( flowGlobalType ) & FlowTransport : : getGlobalLocalAddresses ) ;
2017-05-26 04:48:44 +08:00
}