Skip to content

Commit 48c0c4c

Browse files
authored
Extract tx publishing from watchers (#1749)
Introduce a `TxPublisher` actor to publish channel txs. Move logic from watcher to this new actor. Remove the `TxSigningKit` abstraction that was introduced a bit too early. The `TxPublisher` will hold all the logic so we'll start by providing the full commitments, and we'll extract more compact objects later. We also now publish the commit-tx and its anchor-tx independently.
1 parent 3da0b80 commit 48c0c4c

27 files changed

+1561
-1308
lines changed

eclair-core/src/main/scala/fr/acinq/eclair/Setup.scala

+6-4
Original file line numberDiff line numberDiff line change
@@ -17,7 +17,7 @@
1717
package fr.acinq.eclair
1818

1919
import akka.Done
20-
import akka.actor.{ActorRef, ActorSystem, Props, SupervisorStrategy}
20+
import akka.actor.{ActorContext, ActorRef, ActorSystem, Props, SupervisorStrategy}
2121
import akka.pattern.after
2222
import akka.util.Timeout
2323
import com.softwaremill.sttp.okhttp.OkHttpFutureBackend
@@ -28,7 +28,7 @@ import fr.acinq.eclair.blockchain.bitcoind.rpc.{BasicBitcoinJsonRPCClient, Batch
2828
import fr.acinq.eclair.blockchain.bitcoind.zmq.ZMQActor
2929
import fr.acinq.eclair.blockchain.bitcoind.{BitcoinCoreWallet, ZmqWatcher}
3030
import fr.acinq.eclair.blockchain.fee._
31-
import fr.acinq.eclair.channel.Register
31+
import fr.acinq.eclair.channel.{Channel, Register, TxPublisher}
3232
import fr.acinq.eclair.crypto.keymanager.{LocalChannelKeyManager, LocalNodeKeyManager}
3333
import fr.acinq.eclair.db.Databases.FileBackup
3434
import fr.acinq.eclair.db.{Databases, DbEventHandler, FileBackupHandler}
@@ -230,10 +230,11 @@ class Setup(datadir: File,
230230
})
231231
_ <- feeratesRetrieved.future
232232

233+
extendedBitcoinClient = new ExtendedBitcoinClient(new BatchingBitcoinJsonRPCClient(bitcoin))
233234
watcher = {
234235
system.actorOf(SimpleSupervisor.props(Props(new ZMQActor(config.getString("bitcoind.zmqblock"), Some(zmqBlockConnected))), "zmqblock", SupervisorStrategy.Restart))
235236
system.actorOf(SimpleSupervisor.props(Props(new ZMQActor(config.getString("bitcoind.zmqtx"), Some(zmqTxConnected))), "zmqtx", SupervisorStrategy.Restart))
236-
system.actorOf(SimpleSupervisor.props(ZmqWatcher.props(nodeParams.chainHash, blockCount, new ExtendedBitcoinClient(new BatchingBitcoinJsonRPCClient(bitcoin))), "watcher", SupervisorStrategy.Resume))
237+
system.actorOf(SimpleSupervisor.props(ZmqWatcher.props(nodeParams.chainHash, blockCount, extendedBitcoinClient), "watcher", SupervisorStrategy.Resume))
237238
}
238239

239240
router = system.actorOf(SimpleSupervisor.props(Router.props(nodeParams, watcher, Some(routerInitialized)), "router", SupervisorStrategy.Resume))
@@ -267,7 +268,8 @@ class Setup(datadir: File,
267268
// we want to make sure the handler for post-restart broken HTLCs has finished initializing.
268269
_ <- postRestartCleanUpInitialized.future
269270

270-
channelFactory = Peer.SimpleChannelFactory(nodeParams, watcher, relayer, wallet)
271+
txPublisherFactory = Channel.SimpleTxPublisherFactory(nodeParams, watcher, extendedBitcoinClient)
272+
channelFactory = Peer.SimpleChannelFactory(nodeParams, watcher, relayer, wallet, txPublisherFactory)
271273
peerFactory = Switchboard.SimplePeerFactory(nodeParams, wallet, channelFactory)
272274

273275
switchboard = system.actorOf(SimpleSupervisor.props(Switchboard.props(nodeParams, peerFactory), "switchboard", SupervisorStrategy.Resume))

eclair-core/src/main/scala/fr/acinq/eclair/blockchain/WatcherTypes.scala

+1-16
Original file line numberDiff line numberDiff line change
@@ -17,10 +17,8 @@
1717
package fr.acinq.eclair.blockchain
1818

1919
import akka.actor.ActorRef
20-
import fr.acinq.bitcoin.{ByteVector32, Satoshi, Transaction}
21-
import fr.acinq.eclair.blockchain.fee.FeeratePerKw
20+
import fr.acinq.bitcoin.{ByteVector32, Transaction}
2221
import fr.acinq.eclair.channel.BitcoinEvent
23-
import fr.acinq.eclair.transactions.Transactions.TransactionSigningKit
2422
import fr.acinq.eclair.wire.protocol.ChannelAnnouncement
2523

2624
/**
@@ -110,19 +108,6 @@ final case class WatchEventSpentBasic(event: BitcoinEvent) extends WatchEvent
110108
// TODO: not implemented yet.
111109
final case class WatchEventLost(event: BitcoinEvent) extends WatchEvent
112110

113-
// @formatter:off
114-
sealed trait PublishStrategy
115-
object PublishStrategy {
116-
case object JustPublish extends PublishStrategy
117-
case class SetFeerate(currentFeerate: FeeratePerKw, targetFeerate: FeeratePerKw, dustLimit: Satoshi, signingKit: TransactionSigningKit) extends PublishStrategy {
118-
override def toString = s"SetFeerate(target=$targetFeerate)"
119-
}
120-
}
121-
// @formatter:on
122-
123-
/** Publish the provided tx as soon as possible depending on lock time, csv and publishing strategy. */
124-
final case class PublishAsap(tx: Transaction, strategy: PublishStrategy)
125-
126111
// @formatter:off
127112
sealed trait UtxoStatus
128113
object UtxoStatus {

eclair-core/src/main/scala/fr/acinq/eclair/blockchain/bitcoind/ZmqWatcher.scala

+10-262
Large diffs are not rendered by default.

eclair-core/src/main/scala/fr/acinq/eclair/channel/Channel.scala

+39-16
Original file line numberDiff line numberDiff line change
@@ -16,16 +16,20 @@
1616

1717
package fr.acinq.eclair.channel
1818

19-
import akka.actor.{ActorRef, FSM, OneForOneStrategy, Props, Status, SupervisorStrategy}
19+
import akka.actor.typed.scaladsl.Behaviors
20+
import akka.actor.typed.scaladsl.adapter.ClassicActorContextOps
21+
import akka.actor.{ActorContext, ActorRef, FSM, OneForOneStrategy, Props, Status, SupervisorStrategy}
2022
import akka.event.Logging.MDC
2123
import akka.pattern.pipe
2224
import fr.acinq.bitcoin.Crypto.{PrivateKey, PublicKey}
2325
import fr.acinq.bitcoin.{ByteVector32, OutPoint, Satoshi, SatoshiLong, Script, ScriptFlags, Transaction}
2426
import fr.acinq.eclair.Logs.LogCategory
2527
import fr.acinq.eclair._
2628
import fr.acinq.eclair.blockchain._
29+
import fr.acinq.eclair.blockchain.bitcoind.rpc.ExtendedBitcoinClient
2730
import fr.acinq.eclair.channel.Helpers.{Closing, Funding}
2831
import fr.acinq.eclair.channel.Monitoring.{Metrics, Tags}
32+
import fr.acinq.eclair.channel.TxPublisher.{PublishRawTx, PublishTx, SetChannelId, SignAndPublishTx}
2933
import fr.acinq.eclair.crypto.ShaChain
3034
import fr.acinq.eclair.crypto.keymanager.ChannelKeyManager
3135
import fr.acinq.eclair.db.PendingRelayDb
@@ -47,7 +51,19 @@ import scala.util.{Failure, Success, Try}
4751
*/
4852

4953
object Channel {
50-
def props(nodeParams: NodeParams, wallet: EclairWallet, remoteNodeId: PublicKey, blockchain: ActorRef, relayer: ActorRef, origin_opt: Option[ActorRef]): Props = Props(new Channel(nodeParams, wallet, remoteNodeId, blockchain, relayer, origin_opt))
54+
55+
trait TxPublisherFactory {
56+
def spawnTxPublisher(context: ActorContext, remoteNodeId: PublicKey): akka.actor.typed.ActorRef[TxPublisher.Command]
57+
}
58+
59+
case class SimpleTxPublisherFactory(nodeParams: NodeParams, watcher: ActorRef, bitcoinClient: ExtendedBitcoinClient) extends TxPublisherFactory {
60+
override def spawnTxPublisher(context: ActorContext, remoteNodeId: PublicKey): akka.actor.typed.ActorRef[TxPublisher.Command] = {
61+
context.spawn(Behaviors.supervise(TxPublisher(nodeParams, remoteNodeId, watcher, bitcoinClient)).onFailure(akka.actor.typed.SupervisorStrategy.restart), "tx-publisher")
62+
}
63+
}
64+
65+
def props(nodeParams: NodeParams, wallet: EclairWallet, remoteNodeId: PublicKey, blockchain: ActorRef, relayer: ActorRef, txPublisherFactory: TxPublisherFactory, origin_opt: Option[ActorRef]): Props =
66+
Props(new Channel(nodeParams, wallet, remoteNodeId, blockchain, relayer, txPublisherFactory, origin_opt))
5167

5268
// see https://github.com/lightningnetwork/lightning-rfc/blob/master/07-routing-gossip.md#requirements
5369
val ANNOUNCEMENTS_MINCONF = 6
@@ -100,7 +116,7 @@ object Channel {
100116

101117
}
102118

103-
class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId: PublicKey, blockchain: ActorRef, relayer: ActorRef, origin_opt: Option[ActorRef] = None)(implicit ec: ExecutionContext = ExecutionContext.Implicits.global) extends FSM[State, Data] with FSMDiagnosticActorLogging[State, Data] {
119+
class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId: PublicKey, blockchain: ActorRef, relayer: ActorRef, txPublisherFactory: Channel.TxPublisherFactory, origin_opt: Option[ActorRef] = None)(implicit ec: ExecutionContext = ExecutionContext.Implicits.global) extends FSM[State, Data] with FSMDiagnosticActorLogging[State, Data] {
104120

105121
import Channel._
106122

@@ -111,14 +127,16 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
111127

112128
// we assume that the peer is the channel's parent
113129
private val peer = context.parent
114-
//noinspection ActorMutableStateInspection
130+
// noinspection ActorMutableStateInspection
115131
// the last active connection we are aware of; note that the peer manages connections and asynchronously notifies
116132
// the channel, which means that if we get disconnected, the previous active connection will die and some messages will
117133
// be sent to dead letters, before the channel gets notified of the disconnection; knowing that this will happen, we
118134
// choose to not make this an Option (that would be None before the first connection), and instead embrace the fact
119135
// that the active connection may point to dead letters at all time
120136
private var activeConnection = context.system.deadLetters
121137

138+
private val txPublisher = txPublisherFactory.spawnTxPublisher(context, remoteNodeId)
139+
122140
// this will be used to detect htlc timeouts
123141
context.system.eventStream.subscribe(self, classOf[CurrentBlockCount])
124142
// this will be used to make sure the current commitment fee is up-to-date
@@ -165,6 +183,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
165183
case Event(initFunder@INPUT_INIT_FUNDER(temporaryChannelId, fundingSatoshis, pushMsat, initialFeeratePerKw, fundingTxFeeratePerKw, _, localParams, remote, _, channelFlags, channelVersion), Nothing) =>
166184
context.system.eventStream.publish(ChannelCreated(self, peer, remoteNodeId, isFunder = true, temporaryChannelId, initialFeeratePerKw, Some(fundingTxFeeratePerKw)))
167185
activeConnection = remote
186+
txPublisher ! SetChannelId(remoteNodeId, temporaryChannelId)
168187
val fundingPubKey = keyManager.fundingPublicKey(localParams.fundingKeyPath).publicKey
169188
val channelKeyPath = keyManager.keyPath(localParams, channelVersion)
170189
val open = OpenChannel(nodeParams.chainHash,
@@ -192,11 +211,13 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
192211

193212
case Event(inputFundee@INPUT_INIT_FUNDEE(_, localParams, remote, _, _), Nothing) if !localParams.isFunder =>
194213
activeConnection = remote
214+
txPublisher ! SetChannelId(remoteNodeId, inputFundee.temporaryChannelId)
195215
goto(WAIT_FOR_OPEN_CHANNEL) using DATA_WAIT_FOR_OPEN_CHANNEL(inputFundee)
196216

197217
case Event(INPUT_RESTORED(data), _) =>
198218
log.info("restoring channel")
199219
context.system.eventStream.publish(ChannelRestored(self, data.channelId, peer, remoteNodeId, data.commitments.localParams.isFunder, data.commitments))
220+
txPublisher ! SetChannelId(remoteNodeId, data.channelId)
200221
data match {
201222
// NB: order matters!
202223
case closing: DATA_CLOSING if Closing.nothingAtStake(closing) =>
@@ -413,6 +434,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
413434
)
414435
val channelId = toLongId(fundingTx.hash, fundingTxOutputIndex)
415436
peer ! ChannelIdAssigned(self, remoteNodeId, temporaryChannelId, channelId) // we notify the peer asap so it knows how to route messages
437+
txPublisher ! SetChannelId(remoteNodeId, channelId)
416438
context.system.eventStream.publish(ChannelIdAssigned(self, remoteNodeId, temporaryChannelId, channelId))
417439
// NB: we don't send a ChannelSignatureSent for the first commit
418440
goto(WAIT_FOR_FUNDING_SIGNED) using DATA_WAIT_FOR_FUNDING_SIGNED(channelId, localParams, remoteParams, fundingTx, fundingTxFee, initialRelayFees_opt, localSpec, localCommitTx, RemoteCommit(0, remoteSpec, remoteCommitTx.tx.txid, remoteFirstPerCommitmentPoint), open.channelFlags, channelVersion, fundingCreated) sending fundingCreated
@@ -469,6 +491,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
469491
remoteNextCommitInfo = Right(randomKey.publicKey), // we will receive their next per-commitment point in the next message, so we temporarily put a random byte array,
470492
commitInput, ShaChain.init, channelId = channelId)
471493
peer ! ChannelIdAssigned(self, remoteNodeId, temporaryChannelId, channelId) // we notify the peer asap so it knows how to route messages
494+
txPublisher ! SetChannelId(remoteNodeId, channelId)
472495
context.system.eventStream.publish(ChannelIdAssigned(self, remoteNodeId, temporaryChannelId, channelId))
473496
context.system.eventStream.publish(ChannelSignatureReceived(self, commitments))
474497
// NB: we don't send a ChannelSignatureSent for the first commit
@@ -1341,7 +1364,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
13411364
}
13421365
val revokedCommitPublished1 = d.revokedCommitPublished.map { rev =>
13431366
val (rev1, penaltyTxs) = Closing.claimRevokedHtlcTxOutputs(keyManager, d.commitments, rev, tx, nodeParams.onChainFeeConf.feeEstimator)
1344-
penaltyTxs.foreach(claimTx => blockchain ! PublishAsap(claimTx.tx, PublishStrategy.JustPublish))
1367+
penaltyTxs.foreach(claimTx => txPublisher ! PublishRawTx(claimTx))
13451368
penaltyTxs.foreach(claimTx => blockchain ! WatchSpent(self, tx.txid, claimTx.input.outPoint.index.toInt, BITCOIN_OUTPUT_SPENT, hints = Set(claimTx.tx.txid)))
13461369
rev1
13471370
}
@@ -1355,7 +1378,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
13551378
// If the tx is one of our HTLC txs, we now publish a 3rd-stage claim-htlc-tx that claims its output.
13561379
val (localCommitPublished1, claimHtlcTx_opt) = Closing.claimLocalCommitHtlcTxOutput(localCommitPublished, keyManager, d.commitments, tx, nodeParams.onChainFeeConf.feeEstimator, nodeParams.onChainFeeConf.feeTargets)
13571380
claimHtlcTx_opt.foreach(claimHtlcTx => {
1358-
blockchain ! PublishAsap(claimHtlcTx.tx, PublishStrategy.JustPublish)
1381+
txPublisher ! PublishRawTx(claimHtlcTx)
13591382
blockchain ! WatchConfirmed(self, claimHtlcTx.tx.txid, nodeParams.minDepthBlocks, BITCOIN_TX_CONFIRMED(claimHtlcTx.tx))
13601383
})
13611384
Closing.updateLocalCommitPublished(localCommitPublished1, tx)
@@ -1990,7 +2013,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
19902013
case Some(fundingTx) =>
19912014
// if we are funder, we never give up
19922015
log.info(s"republishing the funding tx...")
1993-
blockchain ! PublishAsap(fundingTx, PublishStrategy.JustPublish)
2016+
txPublisher ! PublishRawTx(fundingTx, "funding-tx")
19942017
// we also check if the funding tx has been double-spent
19952018
checkDoubleSpent(fundingTx)
19962019
context.system.scheduler.scheduleOnce(1 day, blockchain, GetTxWithMeta(txid))
@@ -2142,7 +2165,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
21422165
}
21432166

21442167
private def doPublish(closingTx: ClosingTx): Unit = {
2145-
blockchain ! PublishAsap(closingTx.tx, PublishStrategy.JustPublish)
2168+
txPublisher ! PublishRawTx(closingTx)
21462169
blockchain ! WatchConfirmed(self, closingTx.tx.txid, nodeParams.minDepthBlocks, BITCOIN_TX_CONFIRMED(closingTx.tx))
21472170
}
21482171

@@ -2171,11 +2194,11 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
21712194
/**
21722195
* This helper method will publish txs only if they haven't yet reached minDepth
21732196
*/
2174-
private def publishIfNeeded(txs: Iterable[PublishAsap], irrevocablySpent: Map[OutPoint, Transaction]): Unit = {
2197+
private def publishIfNeeded(txs: Iterable[PublishTx], irrevocablySpent: Map[OutPoint, Transaction]): Unit = {
21752198
val (skip, process) = txs.partition(publishTx => Closing.inputsAlreadySpent(publishTx.tx, irrevocablySpent))
21762199
process.foreach { publishTx =>
21772200
log.info(s"publishing txid=${publishTx.tx.txid}")
2178-
blockchain ! publishTx
2201+
txPublisher ! publishTx
21792202
}
21802203
skip.foreach(publishTx => log.info(s"no need to republish txid=${publishTx.tx.txid}, it has already been confirmed"))
21812204
}
@@ -2209,11 +2232,11 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
22092232

22102233
val publishQueue = commitments.commitmentFormat match {
22112234
case Transactions.DefaultCommitmentFormat =>
2212-
val txs = List(commitTx) ++ claimMainDelayedOutputTx.map(_.tx) ++ htlcTxs.values.flatten.map(_.tx) ++ claimHtlcDelayedTxs.map(_.tx)
2213-
txs.map(tx => PublishAsap(tx, PublishStrategy.JustPublish))
2235+
List(PublishRawTx(commitTx, "commit-tx")) ++ (claimMainDelayedOutputTx ++ htlcTxs.values.flatten ++ claimHtlcDelayedTxs).map(tx => PublishRawTx(tx))
22142236
case Transactions.AnchorOutputsCommitmentFormat =>
2215-
val (publishCommitTx, htlcTxs) = Helpers.Closing.createLocalCommitAnchorPublishStrategy(keyManager, commitments, nodeParams.onChainFeeConf.feeEstimator, nodeParams.onChainFeeConf.feeTargets)
2216-
List(publishCommitTx) ++ claimMainDelayedOutputTx.map(tx => PublishAsap(tx.tx, PublishStrategy.JustPublish)) ++ htlcTxs ++ claimHtlcDelayedTxs.map(tx => PublishAsap(tx.tx, PublishStrategy.JustPublish))
2237+
val claimLocalAnchor = claimAnchorTxs.collect { case tx: Transactions.ClaimLocalAnchorOutputTx => SignAndPublishTx(tx, commitments) }
2238+
val redeemableHtlcTxs = htlcTxs.values.collect { case Some(tx) => SignAndPublishTx(tx, commitments) }
2239+
List(PublishRawTx(commitTx, "commit-tx")) ++ claimLocalAnchor ++ claimMainDelayedOutputTx.map(tx => PublishRawTx(tx)) ++ redeemableHtlcTxs ++ claimHtlcDelayedTxs.map(tx => PublishRawTx(tx))
22172240
}
22182241
publishIfNeeded(publishQueue, irrevocablySpent)
22192242

@@ -2276,7 +2299,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
22762299
private def doPublish(remoteCommitPublished: RemoteCommitPublished): Unit = {
22772300
import remoteCommitPublished._
22782301

2279-
val publishQueue = (claimMainOutputTx ++ claimHtlcTxs.values.flatten).map(tx => PublishAsap(tx.tx, PublishStrategy.JustPublish))
2302+
val publishQueue = (claimMainOutputTx ++ claimHtlcTxs.values.flatten).map(tx => PublishRawTx(tx))
22802303
publishIfNeeded(publishQueue, irrevocablySpent)
22812304

22822305
// we watch:
@@ -2315,7 +2338,7 @@ class Channel(val nodeParams: NodeParams, val wallet: EclairWallet, remoteNodeId
23152338
private def doPublish(revokedCommitPublished: RevokedCommitPublished): Unit = {
23162339
import revokedCommitPublished._
23172340

2318-
val publishQueue = (claimMainOutputTx ++ mainPenaltyTx ++ htlcPenaltyTxs ++ claimHtlcDelayedPenaltyTxs).map(tx => PublishAsap(tx.tx, PublishStrategy.JustPublish))
2341+
val publishQueue = (claimMainOutputTx ++ mainPenaltyTx ++ htlcPenaltyTxs ++ claimHtlcDelayedPenaltyTxs).map(tx => PublishRawTx(tx))
23192342
publishIfNeeded(publishQueue, irrevocablySpent)
23202343

23212344
// we watch:

eclair-core/src/main/scala/fr/acinq/eclair/channel/ChannelTypes.scala

+3-3
Original file line numberDiff line numberDiff line change
@@ -19,8 +19,8 @@ package fr.acinq.eclair.channel
1919
import akka.actor.{ActorRef, PossiblyHarmful}
2020
import fr.acinq.bitcoin.Crypto.PublicKey
2121
import fr.acinq.bitcoin.{ByteVector32, DeterministicWallet, OutPoint, Satoshi, Transaction}
22-
import fr.acinq.eclair.blockchain.PublishAsap
2322
import fr.acinq.eclair.blockchain.fee.FeeratePerKw
23+
import fr.acinq.eclair.channel.TxPublisher.PublishTx
2424
import fr.acinq.eclair.payment.OutgoingPacket.Upstream
2525
import fr.acinq.eclair.router.Announcements
2626
import fr.acinq.eclair.transactions.CommitmentSpec
@@ -105,7 +105,7 @@ case object BITCOIN_FUNDING_SPENT extends BitcoinEvent
105105
case object BITCOIN_OUTPUT_SPENT extends BitcoinEvent
106106
case class BITCOIN_TX_CONFIRMED(tx: Transaction) extends BitcoinEvent
107107
case class BITCOIN_FUNDING_EXTERNAL_CHANNEL_SPENT(shortChannelId: ShortChannelId) extends BitcoinEvent
108-
case class BITCOIN_PARENT_TX_CONFIRMED(publishChildTx: PublishAsap) extends BitcoinEvent
108+
case class BITCOIN_PARENT_TX_CONFIRMED(childTx: PublishTx) extends BitcoinEvent
109109

110110
/*
111111
.d8888b. .d88888b. 888b d888 888b d888 d8888 888b 888 8888888b. .d8888b.
@@ -293,7 +293,7 @@ sealed trait CommitPublished {
293293
* @param claimHtlcDelayedTxs 3rd-stage txs (spending the output of HTLC txs).
294294
* @param claimAnchorTxs txs spending anchor outputs to bump the feerate of the commitment tx (if applicable).
295295
*/
296-
case class LocalCommitPublished(commitTx: Transaction, claimMainDelayedOutputTx: Option[ClaimLocalDelayedOutputTx], htlcTxs: Map[OutPoint, Option[HtlcTx]], claimHtlcDelayedTxs: List[ClaimLocalDelayedOutputTx], claimAnchorTxs: List[ClaimAnchorOutputTx], irrevocablySpent: Map[OutPoint, Transaction]) extends CommitPublished {
296+
case class LocalCommitPublished(commitTx: Transaction, claimMainDelayedOutputTx: Option[ClaimLocalDelayedOutputTx], htlcTxs: Map[OutPoint, Option[HtlcTx]], claimHtlcDelayedTxs: List[HtlcDelayedTx], claimAnchorTxs: List[ClaimAnchorOutputTx], irrevocablySpent: Map[OutPoint, Transaction]) extends CommitPublished {
297297
/**
298298
* A local commit is considered done when:
299299
* - all commitment tx outputs that we can spend have been spent and confirmed (even if the spending tx was not ours)

0 commit comments

Comments
 (0)