2017-06-28 15:10:17 +00:00
|
|
|
// Copyright 2017 Vector Creations Ltd
|
|
|
|
//
|
|
|
|
// 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
|
|
|
|
//
|
|
|
|
// http://www.apache.org/licenses/LICENSE-2.0
|
|
|
|
//
|
|
|
|
// 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.
|
|
|
|
|
|
|
|
package queue
|
|
|
|
|
|
|
|
import (
|
2017-09-13 10:03:41 +00:00
|
|
|
"context"
|
2020-03-27 16:28:22 +00:00
|
|
|
"encoding/json"
|
2017-06-28 15:10:17 +00:00
|
|
|
"fmt"
|
2020-07-01 10:46:38 +00:00
|
|
|
"sync"
|
2017-06-28 15:10:17 +00:00
|
|
|
"time"
|
|
|
|
|
2020-07-22 16:01:29 +00:00
|
|
|
"github.com/matrix-org/dendrite/federationsender/statistics"
|
2020-07-01 10:46:38 +00:00
|
|
|
"github.com/matrix-org/dendrite/federationsender/storage"
|
2020-07-20 15:55:20 +00:00
|
|
|
"github.com/matrix-org/dendrite/federationsender/storage/shared"
|
2020-06-10 15:54:43 +00:00
|
|
|
"github.com/matrix-org/dendrite/roomserver/api"
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
"github.com/matrix-org/gomatrix"
|
2017-06-28 15:10:17 +00:00
|
|
|
"github.com/matrix-org/gomatrixserverlib"
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
"github.com/sirupsen/logrus"
|
2017-11-16 10:12:02 +00:00
|
|
|
log "github.com/sirupsen/logrus"
|
2020-04-03 13:29:06 +00:00
|
|
|
"go.uber.org/atomic"
|
2017-06-28 15:10:17 +00:00
|
|
|
)
|
|
|
|
|
2020-07-20 15:55:20 +00:00
|
|
|
const (
|
|
|
|
maxPDUsPerTransaction = 50
|
|
|
|
maxEDUsPerTransaction = 50
|
|
|
|
queueIdleTimeout = time.Second * 30
|
|
|
|
)
|
2020-07-01 10:46:38 +00:00
|
|
|
|
2017-06-28 15:10:17 +00:00
|
|
|
// destinationQueue is a queue of events for a single destination.
|
|
|
|
// It is responsible for sending the events to the destination and
|
|
|
|
// ensures that only one request is in flight to a given destination
|
|
|
|
// at a time.
|
|
|
|
type destinationQueue struct {
|
2020-07-01 10:46:38 +00:00
|
|
|
db storage.Database
|
2020-06-10 15:54:43 +00:00
|
|
|
signing *SigningInfo
|
|
|
|
rsAPI api.RoomserverInternalAPI
|
2020-08-17 10:40:49 +00:00
|
|
|
client *gomatrixserverlib.FederationClient // federation client
|
|
|
|
origin gomatrixserverlib.ServerName // origin of requests
|
|
|
|
destination gomatrixserverlib.ServerName // destination of requests
|
|
|
|
running atomic.Bool // is the queue worker running?
|
|
|
|
backingOff atomic.Bool // true if we're backing off
|
|
|
|
statistics *statistics.ServerStatistics // statistics about this remote server
|
|
|
|
transactionIDMutex sync.Mutex // protects transactionID
|
|
|
|
transactionID gomatrixserverlib.TransactionID // last transaction ID
|
|
|
|
transactionCount atomic.Int32 // how many events in this transaction so far
|
|
|
|
notifyPDUs chan bool // interrupts idle wait for PDUs
|
|
|
|
notifyEDUs chan bool // interrupts idle wait for EDUs
|
|
|
|
interruptBackoff chan bool // interrupts backoff
|
2017-06-28 15:10:17 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// Send event adds the event to the pending queue for the destination.
|
|
|
|
// If the queue is empty then it starts a background goroutine to
|
|
|
|
// start sending events to that destination.
|
2020-07-20 15:55:20 +00:00
|
|
|
func (oq *destinationQueue) sendEvent(receipt *shared.Receipt) {
|
2020-07-01 10:46:38 +00:00
|
|
|
// Create a transaction ID. We'll either do this if we don't have
|
|
|
|
// one made up yet, or if we've exceeded the number of maximum
|
|
|
|
// events allowed in a single tranaction. We'll reset the counter
|
|
|
|
// when we do.
|
|
|
|
oq.transactionIDMutex.Lock()
|
|
|
|
if oq.transactionID == "" || oq.transactionCount.Load() >= maxPDUsPerTransaction {
|
|
|
|
now := gomatrixserverlib.AsTimestamp(time.Now())
|
|
|
|
oq.transactionID = gomatrixserverlib.TransactionID(fmt.Sprintf("%d-%d", now, oq.statistics.SuccessCount()))
|
|
|
|
oq.transactionCount.Store(0)
|
|
|
|
}
|
|
|
|
oq.transactionIDMutex.Unlock()
|
|
|
|
// Create a database entry that associates the given PDU NID with
|
|
|
|
// this destination queue. We'll then be able to retrieve the PDU
|
|
|
|
// later.
|
|
|
|
if err := oq.db.AssociatePDUWithDestination(
|
|
|
|
context.TODO(),
|
|
|
|
oq.transactionID, // the current transaction ID
|
|
|
|
oq.destination, // the destination server name
|
2020-07-20 15:55:20 +00:00
|
|
|
receipt, // NIDs from federationsender_queue_json table
|
2020-07-01 10:46:38 +00:00
|
|
|
); err != nil {
|
2020-07-20 15:55:20 +00:00
|
|
|
log.WithError(err).Errorf("failed to associate PDU receipt %q with destination %q", receipt.String(), oq.destination)
|
2020-07-01 10:46:38 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
// We've successfully added a PDU to the transaction so increase
|
|
|
|
// the counter.
|
|
|
|
oq.transactionCount.Add(1)
|
2020-07-22 16:01:29 +00:00
|
|
|
// Check if the destination is blacklisted. If it isn't then wake
|
|
|
|
// up the queue.
|
|
|
|
if !oq.statistics.Blacklisted() {
|
|
|
|
// Wake up the queue if it's asleep.
|
|
|
|
oq.wakeQueueIfNeeded()
|
|
|
|
// If we're blocking on waiting PDUs then tell the queue that we
|
|
|
|
// have work to do.
|
|
|
|
select {
|
|
|
|
case oq.notifyPDUs <- true:
|
|
|
|
default:
|
|
|
|
}
|
2020-07-01 10:46:38 +00:00
|
|
|
}
|
2017-06-28 15:10:17 +00:00
|
|
|
}
|
|
|
|
|
2018-08-10 15:26:57 +00:00
|
|
|
// sendEDU adds the EDU event to the pending queue for the destination.
|
|
|
|
// If the queue is empty then it starts a background goroutine to
|
2020-04-03 13:29:06 +00:00
|
|
|
// start sending events to that destination.
|
2020-07-20 15:55:20 +00:00
|
|
|
func (oq *destinationQueue) sendEDU(receipt *shared.Receipt) {
|
|
|
|
// Create a database entry that associates the given PDU NID with
|
|
|
|
// this destination queue. We'll then be able to retrieve the PDU
|
|
|
|
// later.
|
|
|
|
if err := oq.db.AssociateEDUWithDestination(
|
|
|
|
context.TODO(),
|
|
|
|
oq.destination, // the destination server name
|
|
|
|
receipt, // NIDs from federationsender_queue_json table
|
|
|
|
); err != nil {
|
|
|
|
log.WithError(err).Errorf("failed to associate EDU receipt %q with destination %q", receipt.String(), oq.destination)
|
|
|
|
return
|
|
|
|
}
|
|
|
|
// We've successfully added an EDU to the transaction so increase
|
|
|
|
// the counter.
|
|
|
|
oq.transactionCount.Add(1)
|
2020-07-22 16:01:29 +00:00
|
|
|
// Check if the destination is blacklisted. If it isn't then wake
|
|
|
|
// up the queue.
|
|
|
|
if !oq.statistics.Blacklisted() {
|
|
|
|
// Wake up the queue if it's asleep.
|
|
|
|
oq.wakeQueueIfNeeded()
|
|
|
|
// If we're blocking on waiting EDUs then tell the queue that we
|
|
|
|
// have work to do.
|
|
|
|
select {
|
|
|
|
case oq.notifyEDUs <- true:
|
|
|
|
default:
|
|
|
|
}
|
2020-07-20 15:55:20 +00:00
|
|
|
}
|
2020-04-03 13:29:06 +00:00
|
|
|
}
|
|
|
|
|
2020-07-03 15:31:56 +00:00
|
|
|
// wakeQueueIfNeeded will wake up the destination queue if it is
|
|
|
|
// not already running. If it is running but it is backing off
|
|
|
|
// then we will interrupt the backoff, causing any federation
|
|
|
|
// requests to retry.
|
2020-07-02 16:43:07 +00:00
|
|
|
func (oq *destinationQueue) wakeQueueIfNeeded() {
|
2020-07-03 15:31:56 +00:00
|
|
|
// If we are backing off then interrupt the backoff.
|
|
|
|
if oq.backingOff.CAS(true, false) {
|
|
|
|
oq.interruptBackoff <- true
|
|
|
|
}
|
|
|
|
// If we aren't running then wake up the queue.
|
2020-04-03 13:29:06 +00:00
|
|
|
if !oq.running.Load() {
|
2020-07-09 14:39:35 +00:00
|
|
|
// Start the queue.
|
2018-08-10 15:26:57 +00:00
|
|
|
go oq.backgroundSend()
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-07-09 14:39:35 +00:00
|
|
|
// waitForPDUs returns a channel for pending PDUs, which will be
|
|
|
|
// used in backgroundSend select. It returns a closed channel if
|
|
|
|
// there is something pending right now, or an open channel if
|
|
|
|
// we're waiting for something.
|
|
|
|
func (oq *destinationQueue) waitForPDUs() chan bool {
|
|
|
|
pendingPDUs, err := oq.db.GetPendingPDUCount(context.TODO(), oq.destination)
|
|
|
|
if err != nil {
|
|
|
|
log.WithError(err).Errorf("Failed to get pending PDU count on queue %q", oq.destination)
|
|
|
|
}
|
|
|
|
// If there are PDUs pending right now then we'll return a closed
|
|
|
|
// channel. This will mean that the backgroundSend will not block.
|
|
|
|
if pendingPDUs > 0 {
|
|
|
|
ch := make(chan bool, 1)
|
|
|
|
close(ch)
|
|
|
|
return ch
|
|
|
|
}
|
|
|
|
// If there are no PDUs pending right now then instead we'll return
|
|
|
|
// the notify channel, so that backgroundSend can pick up normal
|
|
|
|
// notifications from sendEvent.
|
|
|
|
return oq.notifyPDUs
|
|
|
|
}
|
|
|
|
|
2020-07-20 15:55:20 +00:00
|
|
|
// waitForEDUs returns a channel for pending EDUs, which will be
|
|
|
|
// used in backgroundSend select. It returns a closed channel if
|
|
|
|
// there is something pending right now, or an open channel if
|
|
|
|
// we're waiting for something.
|
|
|
|
func (oq *destinationQueue) waitForEDUs() chan bool {
|
|
|
|
pendingEDUs, err := oq.db.GetPendingEDUCount(context.TODO(), oq.destination)
|
|
|
|
if err != nil {
|
|
|
|
log.WithError(err).Errorf("Failed to get pending EDU count on queue %q", oq.destination)
|
|
|
|
}
|
|
|
|
// If there are EDUs pending right now then we'll return a closed
|
|
|
|
// channel. This will mean that the backgroundSend will not block.
|
|
|
|
if pendingEDUs > 0 {
|
|
|
|
ch := make(chan bool, 1)
|
|
|
|
close(ch)
|
|
|
|
return ch
|
|
|
|
}
|
|
|
|
// If there are no EDUs pending right now then instead we'll return
|
|
|
|
// the notify channel, so that backgroundSend can pick up normal
|
|
|
|
// notifications from sendEvent.
|
|
|
|
return oq.notifyEDUs
|
|
|
|
}
|
|
|
|
|
2020-04-03 13:29:06 +00:00
|
|
|
// backgroundSend is the worker goroutine for sending events.
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
// nolint:gocyclo
|
2017-06-28 15:10:17 +00:00
|
|
|
func (oq *destinationQueue) backgroundSend() {
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
// Check if a worker is already running, and if it isn't, then
|
|
|
|
// mark it as started.
|
|
|
|
if !oq.running.CAS(false, true) {
|
|
|
|
return
|
|
|
|
}
|
2020-04-03 13:29:06 +00:00
|
|
|
defer oq.running.Store(false)
|
|
|
|
|
2017-06-28 15:10:17 +00:00
|
|
|
for {
|
2020-07-20 15:55:20 +00:00
|
|
|
pendingPDUs, pendingEDUs := false, false
|
2020-07-09 14:39:35 +00:00
|
|
|
|
2020-07-01 10:46:38 +00:00
|
|
|
// If we have nothing to do then wait either for incoming events, or
|
|
|
|
// until we hit an idle timeout.
|
2020-07-02 16:43:07 +00:00
|
|
|
select {
|
2020-07-09 14:39:35 +00:00
|
|
|
case <-oq.waitForPDUs():
|
2020-07-02 16:43:07 +00:00
|
|
|
// We were woken up because there are new PDUs waiting in the
|
|
|
|
// database.
|
2020-07-09 14:39:35 +00:00
|
|
|
pendingPDUs = true
|
2020-07-20 15:55:20 +00:00
|
|
|
case <-oq.waitForEDUs():
|
|
|
|
// We were woken up because there are new PDUs waiting in the
|
|
|
|
// database.
|
|
|
|
pendingEDUs = true
|
2020-07-09 14:39:35 +00:00
|
|
|
case <-time.After(queueIdleTimeout):
|
2020-07-02 16:43:07 +00:00
|
|
|
// The worker is idle so stop the goroutine. It'll get
|
|
|
|
// restarted automatically the next time we have an event to
|
|
|
|
// send.
|
2020-09-29 16:08:47 +00:00
|
|
|
log.Tracef("Queue %q has been idle for %s, going to sleep", oq.destination, queueIdleTimeout)
|
2020-07-02 16:43:07 +00:00
|
|
|
return
|
2017-06-28 15:10:17 +00:00
|
|
|
}
|
|
|
|
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
// If we are backing off this server then wait for the
|
2020-06-01 17:34:08 +00:00
|
|
|
// backoff duration to complete first, or until explicitly
|
|
|
|
// told to retry.
|
2020-09-21 12:30:37 +00:00
|
|
|
until, blacklisted := oq.statistics.BackoffInfo()
|
|
|
|
if blacklisted {
|
2020-08-07 17:50:29 +00:00
|
|
|
// It's been suggested that we should give up because the backoff
|
|
|
|
// has exceeded a maximum allowable value. Clean up the in-memory
|
|
|
|
// buffers at this point. The PDU clean-up is already on a defer.
|
|
|
|
log.Warnf("Blacklisting %q due to exceeding backoff threshold", oq.destination)
|
|
|
|
return
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
}
|
2020-09-22 13:53:36 +00:00
|
|
|
if until != nil && until.After(time.Now()) {
|
2020-09-21 12:30:37 +00:00
|
|
|
// We haven't backed off yet, so wait for the suggested amount of
|
|
|
|
// time.
|
|
|
|
duration := time.Until(*until)
|
|
|
|
log.Warnf("Backing off %q for %s", oq.destination, duration)
|
|
|
|
select {
|
|
|
|
case <-time.After(duration):
|
|
|
|
case <-oq.interruptBackoff:
|
|
|
|
}
|
|
|
|
}
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
|
|
|
|
// If we have pending PDUs or EDUs then construct a transaction.
|
2020-07-20 15:55:20 +00:00
|
|
|
if pendingPDUs || pendingEDUs {
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
// Try sending the next transaction and see what happens.
|
2020-07-20 15:55:20 +00:00
|
|
|
transaction, terr := oq.nextTransaction()
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
if terr != nil {
|
2020-08-07 17:50:29 +00:00
|
|
|
// We failed to send the transaction. Mark it as a failure.
|
|
|
|
oq.statistics.Failure()
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
} else if transaction {
|
|
|
|
// If we successfully sent the transaction then clear out
|
2020-07-01 10:46:38 +00:00
|
|
|
// the pending events and EDUs, and wipe our transaction ID.
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
oq.statistics.Success()
|
|
|
|
}
|
|
|
|
}
|
2020-07-01 10:46:38 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2020-04-03 13:29:06 +00:00
|
|
|
// nextTransaction creates a new transaction from the pending event
|
|
|
|
// queue and sends it. Returns true if a transaction was sent or
|
|
|
|
// false otherwise.
|
2020-07-20 15:55:20 +00:00
|
|
|
// nolint:gocyclo
|
|
|
|
func (oq *destinationQueue) nextTransaction() (bool, error) {
|
2020-07-01 10:46:38 +00:00
|
|
|
// Before we do anything, we need to roll over the transaction
|
|
|
|
// ID that is being used to coalesce events into the next TX.
|
|
|
|
// Otherwise it's possible that we'll pick up an incomplete
|
|
|
|
// transaction and end up nuking the rest of the events at the
|
|
|
|
// cleanup stage.
|
|
|
|
oq.transactionIDMutex.Lock()
|
|
|
|
oq.transactionID = ""
|
|
|
|
oq.transactionIDMutex.Unlock()
|
|
|
|
oq.transactionCount.Store(0)
|
|
|
|
|
|
|
|
// Create the transaction.
|
2020-02-28 14:54:51 +00:00
|
|
|
t := gomatrixserverlib.Transaction{
|
2020-03-27 16:28:22 +00:00
|
|
|
PDUs: []json.RawMessage{},
|
2020-02-28 14:54:51 +00:00
|
|
|
EDUs: []gomatrixserverlib.EDU{},
|
|
|
|
}
|
2017-06-28 15:10:17 +00:00
|
|
|
t.Origin = oq.origin
|
|
|
|
t.Destination = oq.destination
|
2020-07-01 10:46:38 +00:00
|
|
|
t.OriginServerTS = gomatrixserverlib.AsTimestamp(time.Now())
|
|
|
|
|
|
|
|
// Ask the database for any pending PDUs from the next transaction.
|
|
|
|
// maxPDUsPerTransaction is an upper limit but we probably won't
|
|
|
|
// actually retrieve that many events.
|
2020-07-02 16:43:07 +00:00
|
|
|
ctx, cancel := context.WithTimeout(context.Background(), time.Second*5)
|
|
|
|
defer cancel()
|
2020-07-20 15:55:20 +00:00
|
|
|
txid, pdus, pduReceipt, err := oq.db.GetNextTransactionPDUs(
|
2020-07-02 16:43:07 +00:00
|
|
|
ctx, // context
|
2020-07-01 10:46:38 +00:00
|
|
|
oq.destination, // server name
|
|
|
|
maxPDUsPerTransaction, // max events to retrieve
|
|
|
|
)
|
|
|
|
if err != nil {
|
|
|
|
log.WithError(err).Errorf("failed to get next transaction PDUs for server %q", oq.destination)
|
|
|
|
return false, fmt.Errorf("oq.db.GetNextTransactionPDUs: %w", err)
|
2017-06-28 15:10:17 +00:00
|
|
|
}
|
2018-08-10 15:26:57 +00:00
|
|
|
|
2020-07-20 15:55:20 +00:00
|
|
|
edus, eduReceipt, err := oq.db.GetNextTransactionEDUs(
|
|
|
|
ctx, // context
|
|
|
|
oq.destination, // server name
|
|
|
|
maxEDUsPerTransaction, // max events to retrieve
|
|
|
|
)
|
|
|
|
if err != nil {
|
|
|
|
log.WithError(err).Errorf("failed to get next transaction EDUs for server %q", oq.destination)
|
|
|
|
return false, fmt.Errorf("oq.db.GetNextTransactionEDUs: %w", err)
|
|
|
|
}
|
|
|
|
|
2020-07-01 10:46:38 +00:00
|
|
|
// If we didn't get anything from the database and there are no
|
|
|
|
// pending EDUs then there's nothing to do - stop here.
|
2020-07-20 15:55:20 +00:00
|
|
|
if len(pdus) == 0 && len(edus) == 0 {
|
2020-07-01 10:46:38 +00:00
|
|
|
return false, nil
|
|
|
|
}
|
|
|
|
|
|
|
|
// Pick out the transaction ID from the database. If we didn't
|
|
|
|
// get a transaction ID (i.e. because there are no PDUs but only
|
|
|
|
// EDUs) then generate a transaction ID.
|
|
|
|
t.TransactionID = txid
|
|
|
|
if t.TransactionID == "" {
|
|
|
|
now := gomatrixserverlib.AsTimestamp(time.Now())
|
|
|
|
t.TransactionID = gomatrixserverlib.TransactionID(fmt.Sprintf("%d-%d", now, oq.statistics.SuccessCount()))
|
|
|
|
}
|
2018-08-10 15:26:57 +00:00
|
|
|
|
2020-07-01 10:46:38 +00:00
|
|
|
// Go through PDUs that we retrieved from the database, if any,
|
|
|
|
// and add them into the transaction.
|
|
|
|
for _, pdu := range pdus {
|
2020-03-27 16:28:22 +00:00
|
|
|
// Append the JSON of the event, since this is a json.RawMessage type in the
|
|
|
|
// gomatrixserverlib.Transaction struct
|
|
|
|
t.PDUs = append(t.PDUs, (*pdu).JSON())
|
2017-06-28 15:10:17 +00:00
|
|
|
}
|
2018-08-10 15:26:57 +00:00
|
|
|
|
2020-07-01 10:46:38 +00:00
|
|
|
// Do the same for pending EDUS in the queue.
|
2020-07-20 15:55:20 +00:00
|
|
|
for _, edu := range edus {
|
2018-08-10 15:26:57 +00:00
|
|
|
t.EDUs = append(t.EDUs, *edu)
|
|
|
|
}
|
|
|
|
|
2020-08-07 14:00:23 +00:00
|
|
|
logrus.WithField("server_name", oq.destination).Debugf("Sending transaction %q containing %d PDUs, %d EDUs", t.TransactionID, len(t.PDUs), len(t.EDUs))
|
2020-04-03 13:29:06 +00:00
|
|
|
|
2020-07-01 10:46:38 +00:00
|
|
|
// Try to send the transaction to the destination server.
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
// TODO: we should check for 500-ish fails vs 400-ish here,
|
|
|
|
// since we shouldn't queue things indefinitely in response
|
|
|
|
// to a 400-ish error
|
2020-07-10 15:28:18 +00:00
|
|
|
ctx, cancel = context.WithTimeout(context.Background(), time.Second*30)
|
2020-07-08 13:52:48 +00:00
|
|
|
defer cancel()
|
|
|
|
_, err = oq.client.SendTransaction(ctx, t)
|
2020-07-07 15:54:14 +00:00
|
|
|
switch err.(type) {
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
case nil:
|
2020-07-01 10:46:38 +00:00
|
|
|
// Clean up the transaction in the database.
|
2020-07-20 15:55:20 +00:00
|
|
|
if pduReceipt != nil {
|
|
|
|
//logrus.Infof("Cleaning PDUs %q", pduReceipt.String())
|
|
|
|
if err = oq.db.CleanPDUs(context.Background(), oq.destination, pduReceipt); err != nil {
|
|
|
|
log.WithError(err).Errorf("failed to clean PDUs %q for server %q", pduReceipt.String(), t.Destination)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
if eduReceipt != nil {
|
|
|
|
//logrus.Infof("Cleaning EDUs %q", eduReceipt.String())
|
|
|
|
if err = oq.db.CleanEDUs(context.Background(), oq.destination, eduReceipt); err != nil {
|
|
|
|
log.WithError(err).Errorf("failed to clean EDUs %q for server %q", eduReceipt.String(), t.Destination)
|
|
|
|
}
|
2020-07-01 10:46:38 +00:00
|
|
|
}
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
return true, nil
|
|
|
|
case gomatrix.HTTPError:
|
2020-07-07 15:53:10 +00:00
|
|
|
// Report that we failed to send the transaction and we
|
|
|
|
// will retry again, subject to backoff.
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
return false, err
|
|
|
|
default:
|
2020-04-03 13:29:06 +00:00
|
|
|
log.WithFields(log.Fields{
|
|
|
|
"destination": oq.destination,
|
|
|
|
log.ErrorKey: err,
|
|
|
|
}).Info("problem sending transaction")
|
Improve federation sender performance, implement backoff and blacklisting, fix up invites a bit (#1007)
* Improve federation sender performance and behaviour, add backoff
* Tweaks
* Tweaks
* Tweaks
* Take copies of events before passing to destination queues
* Don't accidentally drop queued messages
* Don't take copies again
* Tidy up a bit
* Break out statistics (tracked component-wide), report success and failures from Perform actions
* Fix comment, use atomic add
* Improve logic a bit, don't block on wakeup, move idle check
* Don't retry sucessful invites, don't dispatch sendEvent, sendInvite etc
* Dedupe destinations, fix other bug hopefully
* Dispatch sends again
* Federation sender to ignore invites that are destined locally
* Loopback invite events
* Remodel a bit with channels
* Linter
* Only loopback invite event if we know the room
* We should tell other resident servers about the invite if we know about the room
* Correct invite signing
* Fix invite loopback
* Check HTTP response codes, push new invites to front of queue
* Review comments
2020-05-07 11:42:06 +00:00
|
|
|
return false, err
|
2020-04-03 13:29:06 +00:00
|
|
|
}
|
|
|
|
}
|