mirror of
https://github.com/lightningnetwork/lnd.git
synced 2025-09-19 20:15:18 +02:00
multi: address lingering TODO by no longer wiping out local HTLCs on remote close
In this commit, we fix a lingering TOOD statement in the channel arb. Before this commitment, we would simply wipe our our local HTLC set of the HTLC set that was on the remote commitment transaction on force close. This was incorrect as if our commitment transaction had an HTLC that the remote commitment didn't, then we would fail to cancel that back, and cause both channels to time out on chain. In order to remedy this, we introduce a new `HtlcSetKey` struct to track all 3 possible in-flight set of HTLCs: ours, theirs, and their pending. We also we start to tack on additional data to all the unilateral close messages we send to subscribers. This new data is the CommitSet, or the set of valid commitments at channel closure time. This new information will be used by the channel arb in an upcoming commit to ensure it will cancel back HTLCs in the case of split commitment state. Finally, we start to thread through an optional *CommitSet to the advanceState method. This additional information will give the channel arb addition information it needs to ensure it properly cancels back HTLCs that are about to time out or may time out depending on which commitment is played. Within the htlcswitch pakage, we modify the `SignNextCommitment` method to return the new set of pending HTLCs for the remote party's commitment transaction and `ReceiveRevocation` to return the latest set of commitment transactions on the remote party's commitment as well. This is a preparatory change which is part of a larger change to address a lingering TODO in the cnct. Additionally, rather than just send of the set of HTLCs after the we revoke, we'll also send of the set of HTLCs after the remote party revokes, and we create a pending commitment state for it.
This commit is contained in:
@@ -296,8 +296,25 @@ func newActiveChannelArbitrator(channel *channeldb.OpenChannel,
|
||||
return c.resolveContract(chanPoint, chanLog)
|
||||
}
|
||||
|
||||
// Finally, we'll need to construct a series of htlc Sets based on all
|
||||
// currently known valid commitments.
|
||||
htlcSets := make(map[HtlcSetKey]htlcSet)
|
||||
htlcSets[LocalHtlcSet] = newHtlcSet(channel.LocalCommitment.Htlcs)
|
||||
htlcSets[RemoteHtlcSet] = newHtlcSet(channel.RemoteCommitment.Htlcs)
|
||||
|
||||
pendingRemoteCommitment, err := channel.RemoteCommitChainTip()
|
||||
if err != nil && err != channeldb.ErrNoPendingCommit {
|
||||
blockEpoch.Cancel()
|
||||
return nil, err
|
||||
}
|
||||
if pendingRemoteCommitment != nil {
|
||||
htlcSets[RemotePendingHtlcSet] = newHtlcSet(
|
||||
pendingRemoteCommitment.Commitment.Htlcs,
|
||||
)
|
||||
}
|
||||
|
||||
return NewChannelArbitrator(
|
||||
arbCfg, channel.LocalCommitment.Htlcs, chanLog,
|
||||
arbCfg, htlcSets, chanLog,
|
||||
), nil
|
||||
}
|
||||
|
||||
@@ -557,15 +574,27 @@ func (c *ChainArbitrator) Stop() error {
|
||||
return nil
|
||||
}
|
||||
|
||||
// ContractUpdate is a message packages the latest set of active HTLCs on a
|
||||
// commitment, and also identifies which commitment received a new set of
|
||||
// HTLCs.
|
||||
type ContractUpdate struct {
|
||||
// HtlcKey identifies which commitment the HTLCs below are present on.
|
||||
HtlcKey HtlcSetKey
|
||||
|
||||
// Htlcs are the of active HTLCs on the commitment identified by the
|
||||
// above HtlcKey.
|
||||
Htlcs []channeldb.HTLC
|
||||
}
|
||||
|
||||
// ContractSignals wraps the two signals that affect the state of a channel
|
||||
// being watched by an arbitrator. The two signals we care about are: the
|
||||
// channel has a new set of HTLC's, and the remote party has just broadcast
|
||||
// their version of the commitment transaction.
|
||||
type ContractSignals struct {
|
||||
// HtlcUpdates is a channel that once we new commitment updates takes
|
||||
// place, the later set of HTLC's on the commitment transaction should
|
||||
// be sent over.
|
||||
HtlcUpdates chan []channeldb.HTLC
|
||||
// HtlcUpdates is a channel that the link will use to update the
|
||||
// designated channel arbitrator when the set of HTLCs on any valid
|
||||
// commitment changes.
|
||||
HtlcUpdates chan *ContractUpdate
|
||||
|
||||
// ShortChanID is the up to date short channel ID for a contract. This
|
||||
// can change either if when the contract was added it didn't yet have
|
||||
|
@@ -30,20 +30,77 @@ const (
|
||||
maxCommitPointPollTimeout = 10 * time.Minute
|
||||
)
|
||||
|
||||
// LocalUnilateralCloseInfo encapsulates all the informnation we need to act
|
||||
// on a local force close that gets confirmed.
|
||||
// LocalUnilateralCloseInfo encapsulates all the information we need to act on
|
||||
// a local force close that gets confirmed.
|
||||
type LocalUnilateralCloseInfo struct {
|
||||
*chainntnfs.SpendDetail
|
||||
*lnwallet.LocalForceCloseSummary
|
||||
*channeldb.ChannelCloseSummary
|
||||
|
||||
// CommitSet is the set of known valid commitments at the time the
|
||||
// remote party's commitment hit the chain.
|
||||
CommitSet CommitSet
|
||||
}
|
||||
|
||||
// CooperativeCloseInfo encapsulates all the informnation we need to act
|
||||
// on a cooperative close that gets confirmed.
|
||||
// CooperativeCloseInfo encapsulates all the information we need to act on a
|
||||
// cooperative close that gets confirmed.
|
||||
type CooperativeCloseInfo struct {
|
||||
*channeldb.ChannelCloseSummary
|
||||
}
|
||||
|
||||
// RemoteUnilateralCloseInfo wraps the normal UnilateralCloseSummary to couple
|
||||
// the CommitSet at the time of channel closure.
|
||||
type RemoteUnilateralCloseInfo struct {
|
||||
*lnwallet.UnilateralCloseSummary
|
||||
|
||||
// CommitSet is the set of known valid commitments at the time the
|
||||
// remote party's commitemnt hit the chain.
|
||||
CommitSet CommitSet
|
||||
}
|
||||
|
||||
// CommitSet is a collection of the set of known valid commitments at a given
|
||||
// instant. If ConfCommitKey is set, then the commitment identified by the
|
||||
// HtlcSetKey has hit the chain. This struct will be used to examine all live
|
||||
// HTLCs to determine if any additional actions need to be made based on the
|
||||
// remote party's commitments.
|
||||
type CommitSet struct {
|
||||
// ConfCommitKey if non-nil, identifies the commitment that was
|
||||
// confirmed in the chain.
|
||||
ConfCommitKey *HtlcSetKey
|
||||
|
||||
// HtlcSets stores the set of all known active HTLC for each active
|
||||
// commitment at the time of channel closure.
|
||||
HtlcSets map[HtlcSetKey][]channeldb.HTLC
|
||||
}
|
||||
|
||||
// IsEmpty returns true if there are no HTLCs at all within all commitments
|
||||
// that are a part of this commitment diff.
|
||||
func (c *CommitSet) IsEmpty() bool {
|
||||
if c == nil {
|
||||
return true
|
||||
}
|
||||
|
||||
for _, htlcs := range c.HtlcSets {
|
||||
if len(htlcs) != 0 {
|
||||
return false
|
||||
}
|
||||
}
|
||||
|
||||
return true
|
||||
}
|
||||
|
||||
// toActiveHTLCSets returns the set of all active HTLCs across all commitment
|
||||
// transactions.
|
||||
func (c *CommitSet) toActiveHTLCSets() map[HtlcSetKey]htlcSet {
|
||||
htlcSets := make(map[HtlcSetKey]htlcSet)
|
||||
|
||||
for htlcSetKey, htlcs := range c.HtlcSets {
|
||||
htlcSets[htlcSetKey] = newHtlcSet(htlcs)
|
||||
}
|
||||
|
||||
return htlcSets
|
||||
}
|
||||
|
||||
// ChainEventSubscription is a struct that houses a subscription to be notified
|
||||
// for any on-chain events related to a channel. There are three types of
|
||||
// possible on-chain events: a cooperative channel closure, a unilateral
|
||||
@@ -55,7 +112,7 @@ type ChainEventSubscription struct {
|
||||
|
||||
// RemoteUnilateralClosure is a channel that will be sent upon in the
|
||||
// event that the remote party's commitment transaction is confirmed.
|
||||
RemoteUnilateralClosure chan *lnwallet.UnilateralCloseSummary
|
||||
RemoteUnilateralClosure chan *RemoteUnilateralCloseInfo
|
||||
|
||||
// LocalUnilateralClosure is a channel that will be sent upon in the
|
||||
// event that our commitment transaction is confirmed.
|
||||
@@ -249,7 +306,7 @@ func (c *chainWatcher) SubscribeChannelEvents() *ChainEventSubscription {
|
||||
|
||||
sub := &ChainEventSubscription{
|
||||
ChanPoint: c.cfg.chanState.FundingOutpoint,
|
||||
RemoteUnilateralClosure: make(chan *lnwallet.UnilateralCloseSummary, 1),
|
||||
RemoteUnilateralClosure: make(chan *RemoteUnilateralCloseInfo, 1),
|
||||
LocalUnilateralClosure: make(chan *LocalUnilateralCloseInfo, 1),
|
||||
CooperativeClosure: make(chan *CooperativeCloseInfo, 1),
|
||||
ContractBreach: make(chan *lnwallet.BreachRetribution, 1),
|
||||
@@ -373,6 +430,30 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
return
|
||||
}
|
||||
|
||||
// Fetch the current known commit height for the remote party,
|
||||
// and their pending commitment chain tip if it exist.
|
||||
remoteStateNum := remoteCommit.CommitHeight
|
||||
remoteChainTip, err := c.cfg.chanState.RemoteCommitChainTip()
|
||||
if err != nil && err != channeldb.ErrNoPendingCommit {
|
||||
log.Errorf("unable to obtain chain tip for "+
|
||||
"ChannelPoint(%v): %v",
|
||||
c.cfg.chanState.FundingOutpoint, err)
|
||||
return
|
||||
}
|
||||
|
||||
// Now that we have all the possible valid commitments, we'll
|
||||
// make the CommitSet the ChannelArbitrator will need it in
|
||||
// order to carry out its duty.
|
||||
commitSet := CommitSet{
|
||||
HtlcSets: make(map[HtlcSetKey][]channeldb.HTLC),
|
||||
}
|
||||
commitSet.HtlcSets[LocalHtlcSet] = localCommit.Htlcs
|
||||
commitSet.HtlcSets[RemoteHtlcSet] = remoteCommit.Htlcs
|
||||
if remoteChainTip != nil {
|
||||
htlcs := remoteChainTip.Commitment.Htlcs
|
||||
commitSet.HtlcSets[RemotePendingHtlcSet] = htlcs
|
||||
}
|
||||
|
||||
// We'll not retrieve the latest sate of the revocation store
|
||||
// so we can populate the information within the channel state
|
||||
// object that we have.
|
||||
@@ -411,8 +492,10 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
// as we don't have any further processing we need to do (we
|
||||
// can't cheat ourselves :p).
|
||||
if isOurCommit {
|
||||
commitSet.ConfCommitKey = &LocalHtlcSet
|
||||
|
||||
if err := c.dispatchLocalForceClose(
|
||||
commitSpend, *localCommit,
|
||||
commitSpend, *localCommit, commitSet,
|
||||
); err != nil {
|
||||
log.Errorf("unable to handle local"+
|
||||
"close for chan_point=%v: %v",
|
||||
@@ -439,17 +522,6 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
log.Warnf("Unprompted commitment broadcast for "+
|
||||
"ChannelPoint(%v) ", c.cfg.chanState.FundingOutpoint)
|
||||
|
||||
// Fetch the current known commit height for the remote party,
|
||||
// and their pending commitment chain tip if it exist.
|
||||
remoteStateNum := remoteCommit.CommitHeight
|
||||
remoteChainTip, err := c.cfg.chanState.RemoteCommitChainTip()
|
||||
if err != nil && err != channeldb.ErrNoPendingCommit {
|
||||
log.Errorf("unable to obtain chain tip for "+
|
||||
"ChannelPoint(%v): %v",
|
||||
c.cfg.chanState.FundingOutpoint, err)
|
||||
return
|
||||
}
|
||||
|
||||
// If this channel has been recovered, then we'll modify our
|
||||
// behavior as it isn't possible for us to close out the
|
||||
// channel off-chain ourselves. It can only be the remote party
|
||||
@@ -465,9 +537,10 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
// we'll trigger the unilateral close signal so subscribers can
|
||||
// clean up the state as necessary.
|
||||
case broadcastStateNum == remoteStateNum && !isRecoveredChan:
|
||||
commitSet.ConfCommitKey = &RemoteHtlcSet
|
||||
|
||||
err := c.dispatchRemoteForceClose(
|
||||
commitSpend, *remoteCommit,
|
||||
commitSpend, *remoteCommit, commitSet,
|
||||
c.cfg.chanState.RemoteCurrentRevocation,
|
||||
)
|
||||
if err != nil {
|
||||
@@ -484,8 +557,10 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
case broadcastStateNum == remoteStateNum+1 &&
|
||||
remoteChainTip != nil && !isRecoveredChan:
|
||||
|
||||
commitSet.ConfCommitKey = &RemotePendingHtlcSet
|
||||
|
||||
err := c.dispatchRemoteForceClose(
|
||||
commitSpend, remoteChainTip.Commitment,
|
||||
commitSpend, *remoteCommit, commitSet,
|
||||
c.cfg.chanState.RemoteNextRevocation,
|
||||
)
|
||||
if err != nil {
|
||||
@@ -553,14 +628,15 @@ func (c *chainWatcher) closeObserver(spendNtfn *chainntnfs.SpendEvent) {
|
||||
c.cfg.chanState.FundingOutpoint)
|
||||
|
||||
// Since we don't have the commitment stored for this
|
||||
// state, we'll just pass an empty commitment. Note
|
||||
// that this means we won't be able to recover any HTLC
|
||||
// funds.
|
||||
// state, we'll just pass an empty commitment within
|
||||
// the commitment set. Note that this means we won't be
|
||||
// able to recover any HTLC funds.
|
||||
//
|
||||
// TODO(halseth): can we try to recover some HTLCs?
|
||||
commitSet.ConfCommitKey = &RemoteHtlcSet
|
||||
err = c.dispatchRemoteForceClose(
|
||||
commitSpend, channeldb.ChannelCommitment{},
|
||||
commitPoint,
|
||||
commitSet, commitPoint,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to handle remote "+
|
||||
@@ -691,7 +767,7 @@ func (c *chainWatcher) dispatchCooperativeClose(commitSpend *chainntnfs.SpendDet
|
||||
// dispatchLocalForceClose processes a unilateral close by us being confirmed.
|
||||
func (c *chainWatcher) dispatchLocalForceClose(
|
||||
commitSpend *chainntnfs.SpendDetail,
|
||||
localCommit channeldb.ChannelCommitment) error {
|
||||
localCommit channeldb.ChannelCommitment, commitSet CommitSet) error {
|
||||
|
||||
log.Infof("Local unilateral close of ChannelPoint(%v) "+
|
||||
"detected", c.cfg.chanState.FundingOutpoint)
|
||||
@@ -749,7 +825,10 @@ func (c *chainWatcher) dispatchLocalForceClose(
|
||||
// With the event processed, we'll now notify all subscribers of the
|
||||
// event.
|
||||
closeInfo := &LocalUnilateralCloseInfo{
|
||||
commitSpend, forceClose, closeSummary,
|
||||
SpendDetail: commitSpend,
|
||||
LocalForceCloseSummary: forceClose,
|
||||
ChannelCloseSummary: closeSummary,
|
||||
CommitSet: commitSet,
|
||||
}
|
||||
c.Lock()
|
||||
for _, sub := range c.clientSubscriptions {
|
||||
@@ -781,7 +860,7 @@ func (c *chainWatcher) dispatchLocalForceClose(
|
||||
func (c *chainWatcher) dispatchRemoteForceClose(
|
||||
commitSpend *chainntnfs.SpendDetail,
|
||||
remoteCommit channeldb.ChannelCommitment,
|
||||
commitPoint *btcec.PublicKey) error {
|
||||
commitSet CommitSet, commitPoint *btcec.PublicKey) error {
|
||||
|
||||
log.Infof("Unilateral close of ChannelPoint(%v) "+
|
||||
"detected", c.cfg.chanState.FundingOutpoint)
|
||||
@@ -802,7 +881,10 @@ func (c *chainWatcher) dispatchRemoteForceClose(
|
||||
c.Lock()
|
||||
for _, sub := range c.clientSubscriptions {
|
||||
select {
|
||||
case sub.RemoteUnilateralClosure <- uniClose:
|
||||
case sub.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
CommitSet: commitSet,
|
||||
}:
|
||||
case <-c.quit:
|
||||
c.Unlock()
|
||||
return fmt.Errorf("exiting")
|
||||
|
@@ -104,7 +104,7 @@ func TestChainWatcherRemoteUnilateralClose(t *testing.T) {
|
||||
|
||||
// We should get a new spend event over the remote unilateral close
|
||||
// event channel.
|
||||
var uniClose *lnwallet.UnilateralCloseSummary
|
||||
var uniClose *RemoteUnilateralCloseInfo
|
||||
select {
|
||||
case uniClose = <-chanEvents.RemoteUnilateralClosure:
|
||||
case <-time.After(time.Second * 15):
|
||||
@@ -186,7 +186,7 @@ func TestChainWatcherRemoteUnilateralClosePendingCommit(t *testing.T) {
|
||||
|
||||
// With the HTLC added, we'll now manually initiate a state transition
|
||||
// from Alice to Bob.
|
||||
_, _, err = aliceChannel.SignNextCommitment()
|
||||
_, _, _, err = aliceChannel.SignNextCommitment()
|
||||
if err != nil {
|
||||
t.Fatal(err)
|
||||
}
|
||||
@@ -211,7 +211,7 @@ func TestChainWatcherRemoteUnilateralClosePendingCommit(t *testing.T) {
|
||||
|
||||
// We should get a new spend event over the remote unilateral close
|
||||
// event channel.
|
||||
var uniClose *lnwallet.UnilateralCloseSummary
|
||||
var uniClose *RemoteUnilateralCloseInfo
|
||||
select {
|
||||
case uniClose = <-chanEvents.RemoteUnilateralClosure:
|
||||
case <-time.After(time.Second * 15):
|
||||
@@ -343,7 +343,7 @@ func TestChainWatcherDataLossProtect(t *testing.T) {
|
||||
|
||||
// We should get a new uni close resolution that indicates we
|
||||
// processed the DLP scenario.
|
||||
var uniClose *lnwallet.UnilateralCloseSummary
|
||||
var uniClose *RemoteUnilateralCloseInfo
|
||||
select {
|
||||
case uniClose = <-chanEvents.RemoteUnilateralClosure:
|
||||
// If we processed this as a DLP case, then the remote
|
||||
|
@@ -161,14 +161,14 @@ type ContractReport struct {
|
||||
// htlcSet represents the set of active HTLCs on a given commitment
|
||||
// transaction.
|
||||
type htlcSet struct {
|
||||
// incomingHTLCs is a map of all incoming HTLCs on our commitment
|
||||
// transaction. We may potentially go onchain to claim the funds sent
|
||||
// to us within this set.
|
||||
// incomingHTLCs is a map of all incoming HTLCs on the target
|
||||
// commitment transaction. We may potentially go onchain to claim the
|
||||
// funds sent to us within this set.
|
||||
incomingHTLCs map[uint64]channeldb.HTLC
|
||||
|
||||
// outgoingHTLCs is a map of all outgoing HTLCs on our commitment
|
||||
// transaction. We may potentially go onchain to reclaim the funds that
|
||||
// are currently in limbo.
|
||||
// outgoingHTLCs is a map of all outgoing HTLCs on the target
|
||||
// commitment transaction. We may potentially go onchain to reclaim the
|
||||
// funds that are currently in limbo.
|
||||
outgoingHTLCs map[uint64]channeldb.HTLC
|
||||
}
|
||||
|
||||
@@ -191,6 +191,30 @@ func newHtlcSet(htlcs []channeldb.HTLC) htlcSet {
|
||||
}
|
||||
}
|
||||
|
||||
// HtlcSetKey is a two-tuple that uniquely identifies a set of HTLCs on a
|
||||
// commitment transaction.
|
||||
type HtlcSetKey struct {
|
||||
// IsRemote denotes if the HTLCs are on the remote commitment
|
||||
// transaction.
|
||||
IsRemote bool
|
||||
|
||||
// IsPending denotes if the commitment transaction that HTLCS are on
|
||||
// are pending (the higher of two unrevoked commitments).
|
||||
IsPending bool
|
||||
}
|
||||
|
||||
var (
|
||||
// LocalHtlcSet is the HtlcSetKey used for local commitments.
|
||||
LocalHtlcSet = HtlcSetKey{IsRemote: false, IsPending: false}
|
||||
|
||||
// RemoteHtlcSet is the HtlcSetKey used for remote commitments.
|
||||
RemoteHtlcSet = HtlcSetKey{IsRemote: true, IsPending: false}
|
||||
|
||||
// RemotePendingHtlcSet is the HtlcSetKey used for dangling remote
|
||||
// commitment transactions.
|
||||
RemotePendingHtlcSet = HtlcSetKey{IsRemote: true, IsPending: true}
|
||||
)
|
||||
|
||||
// ChannelArbitrator is the on-chain arbitrator for a particular channel. The
|
||||
// struct will keep in sync with the current set of HTLCs on the commitment
|
||||
// transaction. The job of the attendant is to go on-chain to either settle or
|
||||
@@ -207,9 +231,9 @@ type ChannelArbitrator struct {
|
||||
// its next action, and the state of any unresolved contracts.
|
||||
log ArbitratorLog
|
||||
|
||||
// activeHTLCs is the set of active incoming/outgoing HTLC's on the
|
||||
// commitment transaction.
|
||||
activeHTLCs htlcSet
|
||||
// activeHTLCs is the set of active incoming/outgoing HTLC's on all
|
||||
// currently valid commitment transactions.
|
||||
activeHTLCs map[HtlcSetKey]htlcSet
|
||||
|
||||
// cfg contains all the functionality that the ChannelArbitrator requires
|
||||
// to do its duty.
|
||||
@@ -222,7 +246,7 @@ type ChannelArbitrator struct {
|
||||
// htlcUpdates is a channel that is sent upon with new updates from the
|
||||
// active channel. Each time a new commitment state is accepted, the
|
||||
// set of HTLC's on the new state should be sent across this channel.
|
||||
htlcUpdates <-chan []channeldb.HTLC
|
||||
htlcUpdates <-chan *ContractUpdate
|
||||
|
||||
// activeResolvers is a slice of any active resolvers. This is used to
|
||||
// be able to signal them for shutdown in the case that we shutdown.
|
||||
@@ -252,15 +276,15 @@ type ChannelArbitrator struct {
|
||||
// NewChannelArbitrator returns a new instance of a ChannelArbitrator backed by
|
||||
// the passed config struct.
|
||||
func NewChannelArbitrator(cfg ChannelArbitratorConfig,
|
||||
startingHTLCs []channeldb.HTLC, log ArbitratorLog) *ChannelArbitrator {
|
||||
htlcSets map[HtlcSetKey]htlcSet, log ArbitratorLog) *ChannelArbitrator {
|
||||
|
||||
return &ChannelArbitrator{
|
||||
log: log,
|
||||
signalUpdates: make(chan *signalUpdateMsg),
|
||||
htlcUpdates: make(<-chan []channeldb.HTLC),
|
||||
htlcUpdates: make(<-chan *ContractUpdate),
|
||||
resolutionSignal: make(chan struct{}),
|
||||
forceCloseReqs: make(chan *forceCloseReq),
|
||||
activeHTLCs: newHtlcSet(startingHTLCs),
|
||||
activeHTLCs: htlcSets,
|
||||
cfg: cfg,
|
||||
quit: make(chan struct{}),
|
||||
}
|
||||
@@ -335,7 +359,9 @@ func (c *ChannelArbitrator) Start() error {
|
||||
// We'll now attempt to advance our state forward based on the current
|
||||
// on-chain state, and our set of active contracts.
|
||||
startingState := c.state
|
||||
nextState, _, err := c.advanceState(triggerHeight, trigger)
|
||||
nextState, _, err := c.advanceState(
|
||||
triggerHeight, trigger, nil,
|
||||
)
|
||||
if err != nil {
|
||||
switch err {
|
||||
|
||||
@@ -600,8 +626,9 @@ func (t transitionTrigger) String() string {
|
||||
// the appropriate state transition if necessary. The next state we transition
|
||||
// to is returned, Additionally, if the next transition results in a commitment
|
||||
// broadcast, the commitment transaction itself is returned.
|
||||
func (c *ChannelArbitrator) stateStep(triggerHeight uint32,
|
||||
trigger transitionTrigger) (ArbitratorState, *wire.MsgTx, error) {
|
||||
func (c *ChannelArbitrator) stateStep(
|
||||
triggerHeight uint32, trigger transitionTrigger,
|
||||
confCommitSet *CommitSet) (ArbitratorState, *wire.MsgTx, error) {
|
||||
|
||||
var (
|
||||
nextState ArbitratorState
|
||||
@@ -932,8 +959,9 @@ func (c *ChannelArbitrator) launchResolvers(resolvers []ContractResolver) {
|
||||
// redundant transition, meaning that the state transition is a noop. The final
|
||||
// param is a callback that allows the caller to execute an arbitrary action
|
||||
// after each state transition.
|
||||
func (c *ChannelArbitrator) advanceState(triggerHeight uint32,
|
||||
trigger transitionTrigger) (ArbitratorState, *wire.MsgTx, error) {
|
||||
func (c *ChannelArbitrator) advanceState(
|
||||
triggerHeight uint32, trigger transitionTrigger,
|
||||
confCommitSet *CommitSet) (ArbitratorState, *wire.MsgTx, error) {
|
||||
|
||||
var (
|
||||
priorState ArbitratorState
|
||||
@@ -949,7 +977,7 @@ func (c *ChannelArbitrator) advanceState(triggerHeight uint32,
|
||||
priorState)
|
||||
|
||||
nextState, closeTx, err := c.stateStep(
|
||||
triggerHeight, trigger,
|
||||
triggerHeight, trigger, confCommitSet,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("ChannelArbitrator(%v): unable to advance "+
|
||||
@@ -1099,7 +1127,8 @@ func (c *ChannelArbitrator) checkChainActions(height uint32,
|
||||
// First, we'll make an initial pass over the set of incoming and
|
||||
// outgoing HTLC's to decide if we need to go on chain at all.
|
||||
haveChainActions := false
|
||||
for _, htlc := range c.activeHTLCs.outgoingHTLCs {
|
||||
localHTLCs := c.activeHTLCs[LocalHtlcSet]
|
||||
for _, htlc := range localHTLCs.outgoingHTLCs {
|
||||
// We'll need to go on-chain for an outgoing HTLC if it was
|
||||
// never resolved downstream, and it's "close" to timing out.
|
||||
toChain := c.shouldGoOnChain(
|
||||
@@ -1120,7 +1149,7 @@ func (c *ChannelArbitrator) checkChainActions(height uint32,
|
||||
haveChainActions = haveChainActions || toChain
|
||||
}
|
||||
|
||||
for _, htlc := range c.activeHTLCs.incomingHTLCs {
|
||||
for _, htlc := range localHTLCs.incomingHTLCs {
|
||||
// We'll need to go on-chain to pull an incoming HTLC iff we
|
||||
// know the pre-image and it's close to timing out. We need to
|
||||
// ensure that we claim the funds that our rightfully ours
|
||||
@@ -1166,7 +1195,7 @@ func (c *ChannelArbitrator) checkChainActions(height uint32,
|
||||
// active outgoing HTLC's to see if we either need to: sweep them after
|
||||
// a timeout (then cancel backwards), cancel them backwards
|
||||
// immediately, or watch them as they're still active contracts.
|
||||
for _, htlc := range c.activeHTLCs.outgoingHTLCs {
|
||||
for _, htlc := range localHTLCs.outgoingHTLCs {
|
||||
switch {
|
||||
// If the HTLC is dust, then we can cancel it backwards
|
||||
// immediately as there's no matching contract to arbitrate
|
||||
@@ -1221,7 +1250,7 @@ func (c *ChannelArbitrator) checkChainActions(height uint32,
|
||||
// observe the output on-chain if we don't In this last, case we'll
|
||||
// either learn of it eventually from the outgoing HTLC, or the sender
|
||||
// will timeout the HTLC.
|
||||
for _, htlc := range c.activeHTLCs.incomingHTLCs {
|
||||
for _, htlc := range localHTLCs.incomingHTLCs {
|
||||
log.Tracef("ChannelArbitrator(%v): watching chain to decide "+
|
||||
"action for incoming htlc=%x", c.cfg.ChanPoint,
|
||||
htlc.RHash[:])
|
||||
@@ -1671,7 +1700,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// Now that a new block has arrived, we'll attempt to
|
||||
// advance our state forward.
|
||||
nextState, _, err := c.advanceState(
|
||||
uint32(bestHeight), chainTrigger,
|
||||
uint32(bestHeight), chainTrigger, nil,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
@@ -1703,16 +1732,19 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// A new set of HTLC's has been added or removed from the
|
||||
// commitment transaction. So we'll update our activeHTLCs map
|
||||
// accordingly.
|
||||
case newStateHTLCs := <-c.htlcUpdates:
|
||||
// We'll wipe out our old set of HTLC's and instead
|
||||
// monitor only the HTLC's that are still active on the
|
||||
// current commitment state.
|
||||
c.activeHTLCs = newHtlcSet(newStateHTLCs)
|
||||
case htlcUpdate := <-c.htlcUpdates:
|
||||
// We'll wipe out our old set of HTLC's for each
|
||||
// htlcSetKey type included in this update in order to
|
||||
// only monitor the HTLCs that are still active on this
|
||||
// target commitment.
|
||||
c.activeHTLCs[htlcUpdate.HtlcKey] = newHtlcSet(
|
||||
htlcUpdate.Htlcs,
|
||||
)
|
||||
|
||||
log.Tracef("ChannelArbitrator(%v): fresh set of "+
|
||||
"htlcs=%v", c.cfg.ChanPoint,
|
||||
log.Tracef("ChannelArbitrator(%v): fresh set of htlcs=%v",
|
||||
c.cfg.ChanPoint,
|
||||
newLogClosure(func() string {
|
||||
return spew.Sdump(c.activeHTLCs)
|
||||
return spew.Sdump(htlcUpdate)
|
||||
}),
|
||||
)
|
||||
|
||||
@@ -1734,7 +1766,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// We'll now advance our state machine until it reaches
|
||||
// a terminal state, and the channel is marked resolved.
|
||||
_, _, err = c.advanceState(
|
||||
closeInfo.CloseHeight, coopCloseTrigger,
|
||||
closeInfo.CloseHeight, coopCloseTrigger, nil,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
@@ -1794,7 +1826,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// a terminal state.
|
||||
_, _, err = c.advanceState(
|
||||
uint32(closeInfo.SpendingHeight),
|
||||
localCloseTrigger,
|
||||
localCloseTrigger, &closeInfo.CommitSet,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
@@ -1804,7 +1836,6 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// We'll examine our state to determine if we need to act at
|
||||
// all.
|
||||
case uniClosure := <-c.cfg.ChainEvents.RemoteUnilateralClosure:
|
||||
|
||||
log.Infof("ChannelArbitrator(%v): remote party has "+
|
||||
"closed channel out on-chain", c.cfg.ChanPoint)
|
||||
|
||||
@@ -1817,12 +1848,6 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
HtlcResolutions: *uniClosure.HtlcResolutions,
|
||||
}
|
||||
|
||||
// As we're now acting upon an event triggered by the
|
||||
// broadcast of the remote commitment transaction,
|
||||
// we'll swap out our active HTLC set with the set
|
||||
// present on their commitment.
|
||||
c.activeHTLCs = newHtlcSet(uniClosure.RemoteCommit.Htlcs)
|
||||
|
||||
// When processing a unilateral close event, we'll
|
||||
// transition to the ContractClosed state. We'll log
|
||||
// out the set of resolutions such that they are
|
||||
@@ -1856,7 +1881,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
// a terminal state.
|
||||
_, _, err = c.advanceState(
|
||||
uint32(uniClosure.SpendingHeight),
|
||||
remoteCloseTrigger,
|
||||
remoteCloseTrigger, &uniClosure.CommitSet,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
@@ -1870,7 +1895,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
"fully resolved!", c.cfg.ChanPoint)
|
||||
|
||||
nextState, _, err := c.advanceState(
|
||||
uint32(bestHeight), chainTrigger,
|
||||
uint32(bestHeight), chainTrigger, nil,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
@@ -1904,7 +1929,7 @@ func (c *ChannelArbitrator) channelAttendant(bestHeight int32) {
|
||||
}
|
||||
|
||||
nextState, closeTx, err := c.advanceState(
|
||||
uint32(bestHeight), userTrigger,
|
||||
uint32(bestHeight), userTrigger, nil,
|
||||
)
|
||||
if err != nil {
|
||||
log.Errorf("unable to advance state: %v", err)
|
||||
|
@@ -152,7 +152,7 @@ func createTestChannelArbitrator(log ArbitratorLog) (*ChannelArbitrator,
|
||||
chanPoint := wire.OutPoint{}
|
||||
shortChanID := lnwire.ShortChannelID{}
|
||||
chanEvents := &ChainEventSubscription{
|
||||
RemoteUnilateralClosure: make(chan *lnwallet.UnilateralCloseSummary, 1),
|
||||
RemoteUnilateralClosure: make(chan *RemoteUnilateralCloseInfo, 1),
|
||||
LocalUnilateralClosure: make(chan *LocalUnilateralCloseInfo, 1),
|
||||
CooperativeClosure: make(chan *CooperativeCloseInfo, 1),
|
||||
ContractBreach: make(chan *lnwallet.BreachRetribution, 1),
|
||||
@@ -328,7 +328,13 @@ func TestChannelArbitratorRemoteForceClose(t *testing.T) {
|
||||
SpendDetail: commitSpend,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
}
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
CommitSet: CommitSet{
|
||||
ConfCommitKey: &RemoteHtlcSet,
|
||||
HtlcSets: make(map[HtlcSetKey][]channeldb.HTLC),
|
||||
},
|
||||
}
|
||||
|
||||
// It should transition StateDefault -> StateContractClosed ->
|
||||
// StateFullyResolved.
|
||||
@@ -430,12 +436,12 @@ func TestChannelArbitratorLocalForceClose(t *testing.T) {
|
||||
|
||||
// Now notify about the local force close getting confirmed.
|
||||
chanArb.cfg.ChainEvents.LocalUnilateralClosure <- &LocalUnilateralCloseInfo{
|
||||
&chainntnfs.SpendDetail{},
|
||||
&lnwallet.LocalForceCloseSummary{
|
||||
SpendDetail: &chainntnfs.SpendDetail{},
|
||||
LocalForceCloseSummary: &lnwallet.LocalForceCloseSummary{
|
||||
CloseTx: &wire.MsgTx{},
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
},
|
||||
&channeldb.ChannelCloseSummary{},
|
||||
ChannelCloseSummary: &channeldb.ChannelCloseSummary{},
|
||||
}
|
||||
|
||||
// It should transition StateContractClosed -> StateFullyResolved.
|
||||
@@ -483,7 +489,7 @@ func TestChannelArbitratorLocalForceClosePendingHtlc(t *testing.T) {
|
||||
defer chanArb.Stop()
|
||||
|
||||
// Create htlcUpdates channel.
|
||||
htlcUpdates := make(chan []channeldb.HTLC)
|
||||
htlcUpdates := make(chan *ContractUpdate)
|
||||
|
||||
signals := &ContractSignals{
|
||||
HtlcUpdates: htlcUpdates,
|
||||
@@ -492,14 +498,16 @@ func TestChannelArbitratorLocalForceClosePendingHtlc(t *testing.T) {
|
||||
chanArb.UpdateContractSignals(signals)
|
||||
|
||||
// Add HTLC to channel arbitrator.
|
||||
htlcIndex := uint64(99)
|
||||
htlc := channeldb.HTLC{
|
||||
Incoming: false,
|
||||
Amt: 10000,
|
||||
HtlcIndex: 0,
|
||||
HtlcIndex: htlcIndex,
|
||||
}
|
||||
|
||||
htlcUpdates <- []channeldb.HTLC{
|
||||
htlc,
|
||||
htlcUpdates <- &ContractUpdate{
|
||||
HtlcKey: LocalHtlcSet,
|
||||
Htlcs: []channeldb.HTLC{htlc},
|
||||
}
|
||||
|
||||
errChan := make(chan error, 1)
|
||||
@@ -572,8 +580,8 @@ func TestChannelArbitratorLocalForceClosePendingHtlc(t *testing.T) {
|
||||
}
|
||||
|
||||
chanArb.cfg.ChainEvents.LocalUnilateralClosure <- &LocalUnilateralCloseInfo{
|
||||
&chainntnfs.SpendDetail{},
|
||||
&lnwallet.LocalForceCloseSummary{
|
||||
SpendDetail: &chainntnfs.SpendDetail{},
|
||||
LocalForceCloseSummary: &lnwallet.LocalForceCloseSummary{
|
||||
CloseTx: closeTx,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{
|
||||
OutgoingHTLCs: []lnwallet.OutgoingHtlcResolution{
|
||||
@@ -581,7 +589,13 @@ func TestChannelArbitratorLocalForceClosePendingHtlc(t *testing.T) {
|
||||
},
|
||||
},
|
||||
},
|
||||
&channeldb.ChannelCloseSummary{},
|
||||
ChannelCloseSummary: &channeldb.ChannelCloseSummary{},
|
||||
CommitSet: CommitSet{
|
||||
ConfCommitKey: &LocalHtlcSet,
|
||||
HtlcSets: map[HtlcSetKey][]channeldb.HTLC{
|
||||
LocalHtlcSet: {htlc},
|
||||
},
|
||||
},
|
||||
}
|
||||
|
||||
assertStateTransitions(
|
||||
@@ -627,7 +641,6 @@ func TestChannelArbitratorLocalForceClosePendingHtlc(t *testing.T) {
|
||||
|
||||
// At this point channel should be marked as resolved.
|
||||
assertStateTransitions(t, arbLog.newStates, StateFullyResolved)
|
||||
|
||||
select {
|
||||
case <-resolved:
|
||||
case <-time.After(5 * time.Second):
|
||||
@@ -726,7 +739,9 @@ func TestChannelArbitratorLocalForceCloseRemoteConfirmed(t *testing.T) {
|
||||
SpendDetail: commitSpend,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
}
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
|
||||
// It should transition StateContractClosed -> StateFullyResolved.
|
||||
assertStateTransitions(t, log.newStates, StateContractClosed,
|
||||
@@ -832,7 +847,9 @@ func TestChannelArbitratorLocalForceDoubleSpend(t *testing.T) {
|
||||
SpendDetail: commitSpend,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
}
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
|
||||
// It should transition StateContractClosed -> StateFullyResolved.
|
||||
assertStateTransitions(t, log.newStates, StateContractClosed,
|
||||
@@ -878,7 +895,9 @@ func TestChannelArbitratorPersistence(t *testing.T) {
|
||||
SpendDetail: commitSpend,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
}
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
|
||||
// Since writing the resolutions fail, the arbitrator should not
|
||||
// advance to the next state.
|
||||
@@ -909,7 +928,9 @@ func TestChannelArbitratorPersistence(t *testing.T) {
|
||||
}
|
||||
|
||||
// Send a new remote force close event.
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
|
||||
// Since closing the channel failed, the arbitrator should stay in the
|
||||
// default state.
|
||||
@@ -934,7 +955,9 @@ func TestChannelArbitratorPersistence(t *testing.T) {
|
||||
|
||||
// Now make fetching the resolutions fail.
|
||||
log.failFetch = fmt.Errorf("intentional fetch failure")
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
|
||||
// Since logging the resolutions and closing the channel now succeeds,
|
||||
// it should advance to StateContractClosed.
|
||||
@@ -1015,7 +1038,9 @@ func TestChannelArbitratorCommitFailure(t *testing.T) {
|
||||
SpendDetail: commitSpend,
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
}
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- uniClose
|
||||
chanArb.cfg.ChainEvents.RemoteUnilateralClosure <- &RemoteUnilateralCloseInfo{
|
||||
UnilateralCloseSummary: uniClose,
|
||||
}
|
||||
},
|
||||
expectedStates: []ArbitratorState{StateContractClosed, StateFullyResolved},
|
||||
},
|
||||
@@ -1023,12 +1048,12 @@ func TestChannelArbitratorCommitFailure(t *testing.T) {
|
||||
closeType: channeldb.LocalForceClose,
|
||||
sendEvent: func(chanArb *ChannelArbitrator) {
|
||||
chanArb.cfg.ChainEvents.LocalUnilateralClosure <- &LocalUnilateralCloseInfo{
|
||||
&chainntnfs.SpendDetail{},
|
||||
&lnwallet.LocalForceCloseSummary{
|
||||
SpendDetail: &chainntnfs.SpendDetail{},
|
||||
LocalForceCloseSummary: &lnwallet.LocalForceCloseSummary{
|
||||
CloseTx: &wire.MsgTx{},
|
||||
HtlcResolutions: &lnwallet.HtlcResolutions{},
|
||||
},
|
||||
&channeldb.ChannelCloseSummary{},
|
||||
ChannelCloseSummary: &channeldb.ChannelCloseSummary{},
|
||||
}
|
||||
},
|
||||
expectedStates: []ArbitratorState{StateContractClosed, StateFullyResolved},
|
||||
@@ -1192,8 +1217,8 @@ func TestChannelArbitratorAlreadyForceClosed(t *testing.T) {
|
||||
case <-chanArb.quit:
|
||||
}
|
||||
|
||||
// Finally, we should ensure that we are not able to do so by seeing the
|
||||
// expected errAlreadyForceClosed error.
|
||||
// Finally, we should ensure that we are not able to do so by seeing
|
||||
// the expected errAlreadyForceClosed error.
|
||||
select {
|
||||
case err = <-errChan:
|
||||
if err != errAlreadyForceClosed {
|
||||
|
Reference in New Issue
Block a user