Skip to content

Commit 90fbcd3

Browse files
authored
Index trampoline payments by hash and secret (#1770)
We need to group incoming HTLCs together by payment_hash and payment_secret, otherwise we will reject valid payments that are split into multiple distinct trampoline parts (same payment_hash but different payment_secret). Fixes #1723
1 parent 9e4042f commit 90fbcd3

File tree

4 files changed

+216
-114
lines changed

4 files changed

+216
-114
lines changed

eclair-core/src/main/scala/fr/acinq/eclair/payment/relay/NodeRelay.scala

+36-42
Original file line numberDiff line numberDiff line change
@@ -24,6 +24,7 @@ import akka.actor.typed.scaladsl.{ActorContext, Behaviors}
2424
import fr.acinq.bitcoin.ByteVector32
2525
import fr.acinq.eclair.channel.{CMD_FAIL_HTLC, CMD_FULFILL_HTLC}
2626
import fr.acinq.eclair.db.PendingRelayDb
27+
import fr.acinq.eclair.payment.IncomingPacket.NodeRelayPacket
2728
import fr.acinq.eclair.payment.Monitoring.{Metrics, Tags}
2829
import fr.acinq.eclair.payment.OutgoingPacket.Upstream
2930
import fr.acinq.eclair.payment._
@@ -75,13 +76,29 @@ object NodeRelay {
7576
}
7677
}
7778

78-
def apply(nodeParams: NodeParams, parent: akka.actor.typed.ActorRef[NodeRelayer.Command], register: ActorRef, relayId: UUID, paymentHash: ByteVector32, outgoingPaymentFactory: OutgoingPaymentFactory): Behavior[Command] =
79+
def apply(nodeParams: NodeParams,
80+
parent: akka.actor.typed.ActorRef[NodeRelayer.Command],
81+
register: ActorRef,
82+
relayId: UUID,
83+
nodeRelayPacket: NodeRelayPacket,
84+
paymentSecret: ByteVector32,
85+
outgoingPaymentFactory: OutgoingPaymentFactory): Behavior[Command] =
7986
Behaviors.setup { context =>
87+
val paymentHash = nodeRelayPacket.add.paymentHash
88+
val totalAmountIn = nodeRelayPacket.outerPayload.totalAmount
8089
Behaviors.withMdc(Logs.mdc(
8190
category_opt = Some(Logs.LogCategory.PAYMENT),
8291
parentPaymentId_opt = Some(relayId), // for a node relay, we use the same identifier for the whole relay itself, and the outgoing payment
8392
paymentHash_opt = Some(paymentHash))) {
84-
new NodeRelay(nodeParams, parent, register, relayId, paymentHash, context, outgoingPaymentFactory)()
93+
context.log.info("relaying payment relayId={}", relayId)
94+
val mppFsmAdapters = {
95+
context.messageAdapter[MultiPartPaymentFSM.ExtraPaymentReceived[HtlcPart]](WrappedMultiPartExtraPaymentReceived)
96+
context.messageAdapter[MultiPartPaymentFSM.MultiPartPaymentFailed](WrappedMultiPartPaymentFailed)
97+
context.messageAdapter[MultiPartPaymentFSM.MultiPartPaymentSucceeded](WrappedMultiPartPaymentSucceeded)
98+
}.toClassic
99+
val incomingPaymentHandler = context.actorOf(MultiPartPaymentFSM.props(nodeParams, paymentHash, totalAmountIn, mppFsmAdapters))
100+
new NodeRelay(nodeParams, parent, register, relayId, paymentHash, paymentSecret, context, outgoingPaymentFactory)
101+
.receiving(Queue.empty, nodeRelayPacket.innerPayload, nodeRelayPacket.nextPacket, incomingPaymentHandler)
85102
}
86103
}
87104

@@ -144,66 +161,37 @@ class NodeRelay private(nodeParams: NodeParams,
144161
register: ActorRef,
145162
relayId: UUID,
146163
paymentHash: ByteVector32,
164+
paymentSecret: ByteVector32,
147165
context: ActorContext[NodeRelay.Command],
148166
outgoingPaymentFactory: NodeRelay.OutgoingPaymentFactory) {
149167

150168
import NodeRelay._
151169

152-
private val mppFsmAdapters = {
153-
context.messageAdapter[MultiPartPaymentFSM.ExtraPaymentReceived[HtlcPart]](WrappedMultiPartExtraPaymentReceived)
154-
context.messageAdapter[MultiPartPaymentFSM.MultiPartPaymentFailed](WrappedMultiPartPaymentFailed)
155-
context.messageAdapter[MultiPartPaymentFSM.MultiPartPaymentSucceeded](WrappedMultiPartPaymentSucceeded)
156-
}.toClassic
157-
private val payFsmAdapters = {
158-
context.messageAdapter[PreimageReceived](WrappedPreimageReceived)
159-
context.messageAdapter[PaymentSent](WrappedPaymentSent)
160-
context.messageAdapter[PaymentFailed](WrappedPaymentFailed)
161-
}.toClassic
162-
163-
def apply(): Behavior[Command] =
164-
Behaviors.receiveMessagePartial {
165-
// We make sure we receive all payment parts before forwarding to the next trampoline node.
166-
case Relay(IncomingPacket.NodeRelayPacket(add, outer, inner, next)) => outer.paymentSecret match {
167-
case None =>
168-
// TODO: @pm: maybe those checks should be done later in the flow (by the mpp FSM?)
169-
context.log.warn("rejecting htlcId={}: missing payment secret", add.id)
170-
rejectHtlc(add.id, add.channelId, add.amountMsat)
171-
stopping()
172-
case Some(secret) =>
173-
import akka.actor.typed.scaladsl.adapter._
174-
context.log.info("relaying payment relayId={}", relayId)
175-
val mppFsm = context.actorOf(MultiPartPaymentFSM.props(nodeParams, add.paymentHash, outer.totalAmount, mppFsmAdapters))
176-
context.log.debug("forwarding incoming htlc to the payment FSM")
177-
mppFsm ! MultiPartPaymentFSM.HtlcPart(outer.totalAmount, add)
178-
receiving(Queue(add), secret, inner, next, mppFsm)
179-
}
180-
}
181-
182170
/**
183171
* We start by aggregating an incoming HTLC set. Once we received the whole set, we will compute a route to the next
184172
* trampoline node and forward the payment.
185173
*
186174
* @param htlcs received incoming HTLCs for this set.
187-
* @param secret all incoming HTLCs in this set must have the same secret to protect against probing / fee theft.
188175
* @param nextPayload relay instructions (should be identical across HTLCs in this set).
189176
* @param nextPacket trampoline onion to relay to the next trampoline node.
190177
* @param handler actor handling the aggregation of the incoming HTLC set.
191178
*/
192-
private def receiving(htlcs: Queue[UpdateAddHtlc], secret: ByteVector32, nextPayload: Onion.NodeRelayPayload, nextPacket: OnionRoutingPacket, handler: ActorRef): Behavior[Command] =
179+
private def receiving(htlcs: Queue[UpdateAddHtlc], nextPayload: Onion.NodeRelayPayload, nextPacket: OnionRoutingPacket, handler: ActorRef): Behavior[Command] =
193180
Behaviors.receiveMessagePartial {
194181
case Relay(IncomingPacket.NodeRelayPacket(add, outer, _, _)) => outer.paymentSecret match {
182+
// TODO: @pm: maybe those checks should be done by the mpp FSM?
195183
case None =>
196-
context.log.warn("rejecting htlcId={}: missing payment secret", add.id)
184+
context.log.warn("rejecting htlc #{} from channel {}: missing payment secret", add.id, add.channelId)
197185
rejectHtlc(add.id, add.channelId, add.amountMsat)
198186
Behaviors.same
199-
case Some(incomingSecret) if incomingSecret != secret =>
200-
context.log.warn("rejecting htlcId={}: payment secret doesn't match other HTLCs in the set", add.id)
187+
case Some(incomingSecret) if incomingSecret != paymentSecret =>
188+
context.log.warn("rejecting htlc #{} from channel {}: payment secret doesn't match other HTLCs in the set", add.id, add.channelId)
201189
rejectHtlc(add.id, add.channelId, add.amountMsat)
202190
Behaviors.same
203-
case Some(incomingSecret) if incomingSecret == secret =>
204-
context.log.debug("forwarding incoming htlc to the payment FSM")
191+
case Some(incomingSecret) if incomingSecret == paymentSecret =>
192+
context.log.debug("forwarding incoming htlc #{} from channel {} to the payment FSM", add.id, add.channelId)
205193
handler ! MultiPartPaymentFSM.HtlcPart(outer.totalAmount, add)
206-
receiving(htlcs :+ add, secret, nextPayload, nextPacket, handler)
194+
receiving(htlcs :+ add, nextPayload, nextPacket, handler)
207195
}
208196
case WrappedMultiPartPaymentFailed(MultiPartPaymentFSM.MultiPartPaymentFailed(_, failure, parts)) =>
209197
context.log.warn("could not complete incoming multi-part payment (parts={} paidAmount={} failure={})", parts.size, parts.map(_.amount).sum, failure)
@@ -267,14 +255,20 @@ class NodeRelay private(nodeParams: NodeParams,
267255
* Once the downstream payment is settled (fulfilled or failed), we reject new upstream payments while we wait for our parent to stop us.
268256
*/
269257
private def stopping(): Behavior[Command] = {
270-
parent ! NodeRelayer.RelayComplete(context.self, paymentHash)
258+
parent ! NodeRelayer.RelayComplete(context.self, paymentHash, paymentSecret)
271259
Behaviors.receiveMessagePartial {
272260
rejectExtraHtlcPartialFunction orElse {
273261
case Stop => Behaviors.stopped
274262
}
275263
}
276264
}
277265

266+
private val payFsmAdapters = {
267+
context.messageAdapter[PreimageReceived](WrappedPreimageReceived)
268+
context.messageAdapter[PaymentSent](WrappedPaymentSent)
269+
context.messageAdapter[PaymentFailed](WrappedPaymentFailed)
270+
}.toClassic
271+
278272
private def relay(upstream: Upstream.Trampoline, payloadOut: Onion.NodeRelayPayload, packetOut: OnionRoutingPacket): ActorRef = {
279273
val paymentCfg = SendPaymentConfig(relayId, relayId, None, paymentHash, payloadOut.amountToForward, payloadOut.outgoingNodeId, upstream, None, storeInDb = false, publishEvent = false, Nil)
280274
val routeParams = computeRouteParams(nodeParams, upstream.amountIn, upstream.expiryIn, payloadOut.amountToForward, payloadOut.outgoingCltv)
@@ -322,7 +316,7 @@ class NodeRelay private(nodeParams: NodeParams,
322316
}
323317

324318
private def rejectExtraHtlc(add: UpdateAddHtlc): Unit = {
325-
context.log.warn("rejecting extra htlcId={}", add.id)
319+
context.log.warn("rejecting extra htlc #{} from channel {}", add.id, add.channelId)
326320
rejectHtlc(add.id, add.channelId, add.amountMsat)
327321
}
328322

eclair-core/src/main/scala/fr/acinq/eclair/payment/relay/NodeRelayer.scala

+38-22
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,10 @@ package fr.acinq.eclair.payment.relay
1919
import akka.actor.typed.scaladsl.Behaviors
2020
import akka.actor.typed.{ActorRef, Behavior}
2121
import fr.acinq.bitcoin.ByteVector32
22+
import fr.acinq.eclair.channel.CMD_FAIL_HTLC
23+
import fr.acinq.eclair.db.PendingRelayDb
2224
import fr.acinq.eclair.payment._
25+
import fr.acinq.eclair.wire.protocol.IncorrectOrUnknownPaymentDetails
2326
import fr.acinq.eclair.{Logs, NodeParams}
2427

2528
import java.util.UUID
@@ -29,16 +32,16 @@ import java.util.UUID
2932
*/
3033

3134
/**
32-
* The [[NodeRelayer]] relays an upstream payment to a downstream remote node (which is not necessarily a direct peer). It
33-
* doesn't do the job itself, instead it dispatches each individual payment (which can be multi-in, multi-out) to a child
34-
* actor of type [[NodeRelay]].
35+
* The [[NodeRelayer]] relays an upstream payment to a downstream remote node (which is not necessarily a direct peer).
36+
* It doesn't do the job itself, instead it dispatches each individual payment (which can be multi-in, multi-out) to a
37+
* child actor of type [[NodeRelay]].
3538
*/
3639
object NodeRelayer {
3740

3841
// @formatter:off
3942
sealed trait Command
4043
case class Relay(nodeRelayPacket: IncomingPacket.NodeRelayPacket) extends Command
41-
case class RelayComplete(childHandler: ActorRef[NodeRelay.Command], paymentHash: ByteVector32) extends Command
44+
case class RelayComplete(childHandler: ActorRef[NodeRelay.Command], paymentHash: ByteVector32, paymentSecret: ByteVector32) extends Command
4245
private[relay] case class GetPendingPayments(replyTo: akka.actor.ActorRef) extends Command
4346
// @formatter:on
4447

@@ -48,34 +51,47 @@ object NodeRelayer {
4851
case _: GetPendingPayments => Logs.mdc()
4952
}
5053

54+
case class PaymentKey(paymentHash: ByteVector32, paymentSecret: ByteVector32)
55+
5156
/**
52-
* @param children a map of current in-process payments, indexed by payment hash and purposefully *not* by payment id,
53-
* because that is how we aggregate payment parts (when the incoming payment uses MPP).
57+
* @param children a map of pending payments. We must index by both payment hash and payment secret because we may
58+
* need to independently relay multiple parts of the same payment using distinct payment secrets.
59+
* NB: the payment secret used here is different from the invoice's payment secret and ensures we can
60+
* group together HTLCs that the previous trampoline node sent in the same MPP.
5461
*/
55-
def apply(nodeParams: NodeParams, router: akka.actor.ActorRef, register: akka.actor.ActorRef, children: Map[ByteVector32, ActorRef[NodeRelay.Command]] = Map.empty): Behavior[Command] =
62+
def apply(nodeParams: NodeParams, register: akka.actor.ActorRef, outgoingPaymentFactory: NodeRelay.OutgoingPaymentFactory, children: Map[PaymentKey, ActorRef[NodeRelay.Command]] = Map.empty): Behavior[Command] =
5663
Behaviors.setup { context =>
5764
Behaviors.withMdc(Logs.mdc(category_opt = Some(Logs.LogCategory.PAYMENT)), mdc) {
5865
Behaviors.receiveMessage {
5966
case Relay(nodeRelayPacket) =>
60-
import nodeRelayPacket.add.paymentHash
61-
children.get(paymentHash) match {
62-
case Some(handler) =>
63-
context.log.debug("forwarding incoming htlc to existing handler")
64-
handler ! NodeRelay.Relay(nodeRelayPacket)
65-
Behaviors.same
67+
val htlcIn = nodeRelayPacket.add
68+
nodeRelayPacket.outerPayload.paymentSecret match {
69+
case Some(paymentSecret) =>
70+
val childKey = PaymentKey(htlcIn.paymentHash, paymentSecret)
71+
children.get(childKey) match {
72+
case Some(handler) =>
73+
context.log.debug("forwarding incoming htlc #{} from channel {} to existing handler", htlcIn.id, htlcIn.channelId)
74+
handler ! NodeRelay.Relay(nodeRelayPacket)
75+
Behaviors.same
76+
case None =>
77+
val relayId = UUID.randomUUID()
78+
context.log.debug(s"spawning a new handler with relayId=$relayId")
79+
val handler = context.spawn(NodeRelay.apply(nodeParams, context.self, register, relayId, nodeRelayPacket, childKey.paymentSecret, outgoingPaymentFactory), relayId.toString)
80+
context.log.debug("forwarding incoming htlc #{} from channel {} to new handler", htlcIn.id, htlcIn.channelId)
81+
handler ! NodeRelay.Relay(nodeRelayPacket)
82+
apply(nodeParams, register, outgoingPaymentFactory, children + (childKey -> handler))
83+
}
6684
case None =>
67-
val relayId = UUID.randomUUID()
68-
context.log.debug(s"spawning a new handler with relayId=$relayId")
69-
val outgoingPaymentFactory = NodeRelay.SimpleOutgoingPaymentFactory(nodeParams, router, register)
70-
val handler = context.spawn(NodeRelay.apply(nodeParams, context.self, register, relayId, paymentHash, outgoingPaymentFactory), relayId.toString)
71-
context.log.debug("forwarding incoming htlc to new handler")
72-
handler ! NodeRelay.Relay(nodeRelayPacket)
73-
apply(nodeParams, router, register, children + (paymentHash -> handler))
85+
context.log.warn("rejecting htlc #{} from channel {}: missing payment secret", htlcIn.id, htlcIn.channelId)
86+
val failureMessage = IncorrectOrUnknownPaymentDetails(htlcIn.amountMsat, nodeParams.currentBlockHeight)
87+
val cmd = CMD_FAIL_HTLC(htlcIn.id, Right(failureMessage), commit = true)
88+
PendingRelayDb.safeSend(register, nodeParams.db.pendingRelay, htlcIn.channelId, cmd)
89+
Behaviors.same
7490
}
75-
case RelayComplete(childHandler, paymentHash) =>
91+
case RelayComplete(childHandler, paymentHash, paymentSecret) =>
7692
// we do a back-and-forth between parent and child before stopping the child to prevent a race condition
7793
childHandler ! NodeRelay.Stop
78-
apply(nodeParams, router, register, children - paymentHash)
94+
apply(nodeParams, register, outgoingPaymentFactory, children - PaymentKey(paymentHash, paymentSecret))
7995
case GetPendingPayments(replyTo) =>
8096
replyTo ! children
8197
Behaviors.same

eclair-core/src/main/scala/fr/acinq/eclair/payment/relay/Relayer.scala

+1-1
Original file line numberDiff line numberDiff line change
@@ -55,7 +55,7 @@ class Relayer(nodeParams: NodeParams, router: ActorRef, register: ActorRef, paym
5555

5656
private val postRestartCleaner = context.actorOf(PostRestartHtlcCleaner.props(nodeParams, register, initialized), "post-restart-htlc-cleaner")
5757
private val channelRelayer = context.spawn(Behaviors.supervise(ChannelRelayer(nodeParams, register)).onFailure(SupervisorStrategy.resume), "channel-relayer")
58-
private val nodeRelayer = context.spawn(Behaviors.supervise(NodeRelayer(nodeParams, router, register)).onFailure(SupervisorStrategy.resume), name = "node-relayer")
58+
private val nodeRelayer = context.spawn(Behaviors.supervise(NodeRelayer(nodeParams, register, NodeRelay.SimpleOutgoingPaymentFactory(nodeParams, router, register))).onFailure(SupervisorStrategy.resume), name = "node-relayer")
5959

6060
def receive: Receive = {
6161
case RelayForward(add) =>

0 commit comments

Comments
 (0)