mirror of
https://github.com/matrix-org/dendrite.git
synced 2026-01-16 10:33:11 -06:00
Remove OutputReadUpdate stream
This commit is contained in:
parent
ec24000b35
commit
2d9767585f
|
|
@ -186,6 +186,7 @@ func setupNATS(process *process.ProcessContext, cfg *config.JetStream, nc *natsc
|
||||||
OutputTypingEvent: {"SyncAPIEDUServerTypingConsumer", "FederationAPIEDUServerConsumer"},
|
OutputTypingEvent: {"SyncAPIEDUServerTypingConsumer", "FederationAPIEDUServerConsumer"},
|
||||||
OutputRoomEvent: {"AppserviceRoomserverConsumer"},
|
OutputRoomEvent: {"AppserviceRoomserverConsumer"},
|
||||||
OutputStreamEvent: {"UserAPISyncAPIStreamEventConsumer"},
|
OutputStreamEvent: {"UserAPISyncAPIStreamEventConsumer"},
|
||||||
|
OutputReadUpdate: {"UserAPISyncAPIReadUpdateConsumer"},
|
||||||
} {
|
} {
|
||||||
streamName := cfg.Matrix.JetStream.Prefixed(stream)
|
streamName := cfg.Matrix.JetStream.Prefixed(stream)
|
||||||
for _, consumer := range consumers {
|
for _, consumer := range consumers {
|
||||||
|
|
|
||||||
|
|
@ -94,11 +94,6 @@ var streams = []*nats.StreamConfig{
|
||||||
Retention: nats.InterestPolicy,
|
Retention: nats.InterestPolicy,
|
||||||
Storage: nats.FileStorage,
|
Storage: nats.FileStorage,
|
||||||
},
|
},
|
||||||
{
|
|
||||||
Name: OutputReadUpdate,
|
|
||||||
Retention: nats.InterestPolicy,
|
|
||||||
Storage: nats.FileStorage,
|
|
||||||
},
|
|
||||||
{
|
{
|
||||||
Name: OutputPresenceEvent,
|
Name: OutputPresenceEvent,
|
||||||
Retention: nats.InterestPolicy,
|
Retention: nats.InterestPolicy,
|
||||||
|
|
|
||||||
|
|
@ -17,7 +17,6 @@ package consumers
|
||||||
import (
|
import (
|
||||||
"context"
|
"context"
|
||||||
"encoding/json"
|
"encoding/json"
|
||||||
"fmt"
|
|
||||||
|
|
||||||
"github.com/getsentry/sentry-go"
|
"github.com/getsentry/sentry-go"
|
||||||
"github.com/matrix-org/gomatrixserverlib"
|
"github.com/matrix-org/gomatrixserverlib"
|
||||||
|
|
@ -30,7 +29,6 @@ import (
|
||||||
"github.com/matrix-org/dendrite/setup/jetstream"
|
"github.com/matrix-org/dendrite/setup/jetstream"
|
||||||
"github.com/matrix-org/dendrite/setup/process"
|
"github.com/matrix-org/dendrite/setup/process"
|
||||||
"github.com/matrix-org/dendrite/syncapi/notifier"
|
"github.com/matrix-org/dendrite/syncapi/notifier"
|
||||||
"github.com/matrix-org/dendrite/syncapi/producers"
|
|
||||||
"github.com/matrix-org/dendrite/syncapi/storage"
|
"github.com/matrix-org/dendrite/syncapi/storage"
|
||||||
"github.com/matrix-org/dendrite/syncapi/types"
|
"github.com/matrix-org/dendrite/syncapi/types"
|
||||||
)
|
)
|
||||||
|
|
@ -45,7 +43,6 @@ type OutputClientDataConsumer struct {
|
||||||
stream types.StreamProvider
|
stream types.StreamProvider
|
||||||
notifier *notifier.Notifier
|
notifier *notifier.Notifier
|
||||||
serverName gomatrixserverlib.ServerName
|
serverName gomatrixserverlib.ServerName
|
||||||
producer *producers.UserAPIReadProducer
|
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewOutputClientDataConsumer creates a new OutputClientData consumer. Call Start() to begin consuming from room servers.
|
// NewOutputClientDataConsumer creates a new OutputClientData consumer. Call Start() to begin consuming from room servers.
|
||||||
|
|
@ -56,7 +53,6 @@ func NewOutputClientDataConsumer(
|
||||||
store storage.Database,
|
store storage.Database,
|
||||||
notifier *notifier.Notifier,
|
notifier *notifier.Notifier,
|
||||||
stream types.StreamProvider,
|
stream types.StreamProvider,
|
||||||
producer *producers.UserAPIReadProducer,
|
|
||||||
) *OutputClientDataConsumer {
|
) *OutputClientDataConsumer {
|
||||||
return &OutputClientDataConsumer{
|
return &OutputClientDataConsumer{
|
||||||
ctx: process.Context(),
|
ctx: process.Context(),
|
||||||
|
|
@ -67,7 +63,6 @@ func NewOutputClientDataConsumer(
|
||||||
notifier: notifier,
|
notifier: notifier,
|
||||||
stream: stream,
|
stream: stream,
|
||||||
serverName: cfg.Matrix.ServerName,
|
serverName: cfg.Matrix.ServerName,
|
||||||
producer: producer,
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
@ -112,15 +107,6 @@ func (s *OutputClientDataConsumer) onMessage(ctx context.Context, msgs []*nats.M
|
||||||
return false
|
return false
|
||||||
}
|
}
|
||||||
|
|
||||||
if err = s.sendReadUpdate(userID, output); err != nil {
|
|
||||||
log.WithError(err).WithFields(logrus.Fields{
|
|
||||||
"user_id": userID,
|
|
||||||
"room_id": output.RoomID,
|
|
||||||
}).Errorf("Failed to generate read update")
|
|
||||||
sentry.CaptureException(err)
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
|
|
||||||
if output.IgnoredUsers != nil {
|
if output.IgnoredUsers != nil {
|
||||||
if err := s.db.UpdateIgnoresForUser(ctx, userID, output.IgnoredUsers); err != nil {
|
if err := s.db.UpdateIgnoresForUser(ctx, userID, output.IgnoredUsers); err != nil {
|
||||||
log.WithError(err).WithFields(logrus.Fields{
|
log.WithError(err).WithFields(logrus.Fields{
|
||||||
|
|
@ -135,22 +121,3 @@ func (s *OutputClientDataConsumer) onMessage(ctx context.Context, msgs []*nats.M
|
||||||
|
|
||||||
return true
|
return true
|
||||||
}
|
}
|
||||||
|
|
||||||
func (s *OutputClientDataConsumer) sendReadUpdate(userID string, output eventutil.AccountData) error {
|
|
||||||
if output.Type != "m.fully_read" || output.ReadMarker == nil {
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
_, serverName, err := gomatrixserverlib.SplitID('@', userID)
|
|
||||||
if err != nil {
|
|
||||||
return fmt.Errorf("gomatrixserverlib.SplitID: %w", err)
|
|
||||||
}
|
|
||||||
if serverName != s.serverName {
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
if output.ReadMarker.Read != "" || output.ReadMarker.FullyRead != "" {
|
|
||||||
if err := s.producer.SendReadUpdate(userID, output.RoomID, output.ReadMarker.Read, output.ReadMarker.FullyRead); err != nil {
|
|
||||||
return fmt.Errorf("s.producer.SendReadUpdate: %w", err)
|
|
||||||
}
|
|
||||||
}
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
|
|
|
||||||
|
|
@ -16,20 +16,17 @@ package consumers
|
||||||
|
|
||||||
import (
|
import (
|
||||||
"context"
|
"context"
|
||||||
"fmt"
|
|
||||||
"strconv"
|
"strconv"
|
||||||
|
|
||||||
"github.com/getsentry/sentry-go"
|
"github.com/getsentry/sentry-go"
|
||||||
"github.com/matrix-org/gomatrixserverlib"
|
"github.com/matrix-org/gomatrixserverlib"
|
||||||
"github.com/nats-io/nats.go"
|
"github.com/nats-io/nats.go"
|
||||||
"github.com/sirupsen/logrus"
|
|
||||||
log "github.com/sirupsen/logrus"
|
log "github.com/sirupsen/logrus"
|
||||||
|
|
||||||
"github.com/matrix-org/dendrite/setup/config"
|
"github.com/matrix-org/dendrite/setup/config"
|
||||||
"github.com/matrix-org/dendrite/setup/jetstream"
|
"github.com/matrix-org/dendrite/setup/jetstream"
|
||||||
"github.com/matrix-org/dendrite/setup/process"
|
"github.com/matrix-org/dendrite/setup/process"
|
||||||
"github.com/matrix-org/dendrite/syncapi/notifier"
|
"github.com/matrix-org/dendrite/syncapi/notifier"
|
||||||
"github.com/matrix-org/dendrite/syncapi/producers"
|
|
||||||
"github.com/matrix-org/dendrite/syncapi/storage"
|
"github.com/matrix-org/dendrite/syncapi/storage"
|
||||||
"github.com/matrix-org/dendrite/syncapi/types"
|
"github.com/matrix-org/dendrite/syncapi/types"
|
||||||
)
|
)
|
||||||
|
|
@ -44,7 +41,6 @@ type OutputReceiptEventConsumer struct {
|
||||||
stream types.StreamProvider
|
stream types.StreamProvider
|
||||||
notifier *notifier.Notifier
|
notifier *notifier.Notifier
|
||||||
serverName gomatrixserverlib.ServerName
|
serverName gomatrixserverlib.ServerName
|
||||||
producer *producers.UserAPIReadProducer
|
|
||||||
}
|
}
|
||||||
|
|
||||||
// NewOutputReceiptEventConsumer creates a new OutputReceiptEventConsumer.
|
// NewOutputReceiptEventConsumer creates a new OutputReceiptEventConsumer.
|
||||||
|
|
@ -56,7 +52,6 @@ func NewOutputReceiptEventConsumer(
|
||||||
store storage.Database,
|
store storage.Database,
|
||||||
notifier *notifier.Notifier,
|
notifier *notifier.Notifier,
|
||||||
stream types.StreamProvider,
|
stream types.StreamProvider,
|
||||||
producer *producers.UserAPIReadProducer,
|
|
||||||
) *OutputReceiptEventConsumer {
|
) *OutputReceiptEventConsumer {
|
||||||
return &OutputReceiptEventConsumer{
|
return &OutputReceiptEventConsumer{
|
||||||
ctx: process.Context(),
|
ctx: process.Context(),
|
||||||
|
|
@ -67,7 +62,6 @@ func NewOutputReceiptEventConsumer(
|
||||||
notifier: notifier,
|
notifier: notifier,
|
||||||
stream: stream,
|
stream: stream,
|
||||||
serverName: cfg.Matrix.ServerName,
|
serverName: cfg.Matrix.ServerName,
|
||||||
producer: producer,
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
@ -111,36 +105,8 @@ func (s *OutputReceiptEventConsumer) onMessage(ctx context.Context, msgs []*nats
|
||||||
return true
|
return true
|
||||||
}
|
}
|
||||||
|
|
||||||
if err = s.sendReadUpdate(output); err != nil {
|
|
||||||
log.WithError(err).WithFields(logrus.Fields{
|
|
||||||
"user_id": output.UserID,
|
|
||||||
"room_id": output.RoomID,
|
|
||||||
}).Errorf("Failed to generate read update")
|
|
||||||
sentry.CaptureException(err)
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
|
|
||||||
s.stream.Advance(streamPos)
|
s.stream.Advance(streamPos)
|
||||||
s.notifier.OnNewReceipt(output.RoomID, types.StreamingToken{ReceiptPosition: streamPos})
|
s.notifier.OnNewReceipt(output.RoomID, types.StreamingToken{ReceiptPosition: streamPos})
|
||||||
|
|
||||||
return true
|
return true
|
||||||
}
|
}
|
||||||
|
|
||||||
func (s *OutputReceiptEventConsumer) sendReadUpdate(output types.OutputReceiptEvent) error {
|
|
||||||
if output.Type != "m.read" {
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
_, serverName, err := gomatrixserverlib.SplitID('@', output.UserID)
|
|
||||||
if err != nil {
|
|
||||||
return fmt.Errorf("gomatrixserverlib.SplitID: %w", err)
|
|
||||||
}
|
|
||||||
if serverName != s.serverName {
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
if output.EventID != "" {
|
|
||||||
if err := s.producer.SendReadUpdate(output.UserID, output.RoomID, output.EventID, ""); err != nil {
|
|
||||||
return fmt.Errorf("s.producer.SendReadUpdate: %w", err)
|
|
||||||
}
|
|
||||||
}
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
|
|
|
||||||
|
|
@ -1,47 +0,0 @@
|
||||||
// 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 producers
|
|
||||||
|
|
||||||
import (
|
|
||||||
"github.com/nats-io/nats.go"
|
|
||||||
log "github.com/sirupsen/logrus"
|
|
||||||
|
|
||||||
"github.com/matrix-org/dendrite/setup/jetstream"
|
|
||||||
)
|
|
||||||
|
|
||||||
// UserAPIProducer produces events for the user API server to consume
|
|
||||||
type UserAPIReadProducer struct {
|
|
||||||
Topic string
|
|
||||||
JetStream nats.JetStreamContext
|
|
||||||
}
|
|
||||||
|
|
||||||
// SendData sends account data to the user API server
|
|
||||||
func (p *UserAPIReadProducer) SendReadUpdate(userID, roomID string, readPos, fullyReadPos string) error {
|
|
||||||
m := nats.NewMsg(p.Topic)
|
|
||||||
m.Header.Set(jetstream.UserID, userID)
|
|
||||||
m.Header.Set(jetstream.RoomID, roomID)
|
|
||||||
m.Header.Set("read", readPos)
|
|
||||||
m.Header.Set("fully_read", fullyReadPos)
|
|
||||||
|
|
||||||
log.WithFields(log.Fields{
|
|
||||||
"user_id": userID,
|
|
||||||
"room_id": roomID,
|
|
||||||
"read_pos": readPos,
|
|
||||||
"fully_read_pos": fullyReadPos,
|
|
||||||
}).Tracef("Producing to topic '%s'", p.Topic)
|
|
||||||
|
|
||||||
_, err := p.JetStream.PublishMsg(m)
|
|
||||||
return err
|
|
||||||
}
|
|
||||||
|
|
@ -3,9 +3,10 @@ package streams
|
||||||
import (
|
import (
|
||||||
"context"
|
"context"
|
||||||
|
|
||||||
|
"github.com/matrix-org/gomatrixserverlib"
|
||||||
|
|
||||||
"github.com/matrix-org/dendrite/syncapi/types"
|
"github.com/matrix-org/dendrite/syncapi/types"
|
||||||
userapi "github.com/matrix-org/dendrite/userapi/api"
|
userapi "github.com/matrix-org/dendrite/userapi/api"
|
||||||
"github.com/matrix-org/gomatrixserverlib"
|
|
||||||
)
|
)
|
||||||
|
|
||||||
type AccountDataStreamProvider struct {
|
type AccountDataStreamProvider struct {
|
||||||
|
|
|
||||||
|
|
@ -77,11 +77,6 @@ func AddPublicRoutes(
|
||||||
logrus.WithError(err).Panicf("failed to start presence consumer")
|
logrus.WithError(err).Panicf("failed to start presence consumer")
|
||||||
}
|
}
|
||||||
|
|
||||||
userAPIReadUpdateProducer := &producers.UserAPIReadProducer{
|
|
||||||
JetStream: js,
|
|
||||||
Topic: cfg.Matrix.JetStream.Prefixed(jetstream.OutputReadUpdate),
|
|
||||||
}
|
|
||||||
|
|
||||||
keyChangeConsumer := consumers.NewOutputKeyChangeEventConsumer(
|
keyChangeConsumer := consumers.NewOutputKeyChangeEventConsumer(
|
||||||
base.ProcessContext, cfg, cfg.Matrix.JetStream.Prefixed(jetstream.OutputKeyChangeEvent),
|
base.ProcessContext, cfg, cfg.Matrix.JetStream.Prefixed(jetstream.OutputKeyChangeEvent),
|
||||||
js, rsAPI, syncDB, notifier,
|
js, rsAPI, syncDB, notifier,
|
||||||
|
|
@ -101,7 +96,6 @@ func AddPublicRoutes(
|
||||||
|
|
||||||
clientConsumer := consumers.NewOutputClientDataConsumer(
|
clientConsumer := consumers.NewOutputClientDataConsumer(
|
||||||
base.ProcessContext, cfg, js, syncDB, notifier, streams.AccountDataStreamProvider,
|
base.ProcessContext, cfg, js, syncDB, notifier, streams.AccountDataStreamProvider,
|
||||||
userAPIReadUpdateProducer,
|
|
||||||
)
|
)
|
||||||
if err = clientConsumer.Start(); err != nil {
|
if err = clientConsumer.Start(); err != nil {
|
||||||
logrus.WithError(err).Panicf("failed to start client data consumer")
|
logrus.WithError(err).Panicf("failed to start client data consumer")
|
||||||
|
|
@ -130,7 +124,6 @@ func AddPublicRoutes(
|
||||||
|
|
||||||
receiptConsumer := consumers.NewOutputReceiptEventConsumer(
|
receiptConsumer := consumers.NewOutputReceiptEventConsumer(
|
||||||
base.ProcessContext, cfg, js, syncDB, notifier, streams.ReceiptStreamProvider,
|
base.ProcessContext, cfg, js, syncDB, notifier, streams.ReceiptStreamProvider,
|
||||||
userAPIReadUpdateProducer,
|
|
||||||
)
|
)
|
||||||
if err = receiptConsumer.Start(); err != nil {
|
if err = receiptConsumer.Start(); err != nil {
|
||||||
logrus.WithError(err).Panicf("failed to start receipts consumer")
|
logrus.WithError(err).Panicf("failed to start receipts consumer")
|
||||||
|
|
|
||||||
236
userapi/consumers/clientapi.go
Normal file
236
userapi/consumers/clientapi.go
Normal file
|
|
@ -0,0 +1,236 @@
|
||||||
|
// Copyright 2022 The Matrix.org Foundation C.I.C.
|
||||||
|
//
|
||||||
|
// 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 consumers
|
||||||
|
|
||||||
|
import (
|
||||||
|
"context"
|
||||||
|
"encoding/json"
|
||||||
|
|
||||||
|
"github.com/getsentry/sentry-go"
|
||||||
|
"github.com/matrix-org/gomatrixserverlib"
|
||||||
|
"github.com/nats-io/nats.go"
|
||||||
|
log "github.com/sirupsen/logrus"
|
||||||
|
|
||||||
|
"github.com/matrix-org/dendrite/internal/eventutil"
|
||||||
|
"github.com/matrix-org/dendrite/internal/pushgateway"
|
||||||
|
"github.com/matrix-org/dendrite/userapi/storage"
|
||||||
|
|
||||||
|
"github.com/matrix-org/dendrite/setup/config"
|
||||||
|
"github.com/matrix-org/dendrite/setup/jetstream"
|
||||||
|
"github.com/matrix-org/dendrite/setup/process"
|
||||||
|
"github.com/matrix-org/dendrite/userapi/producers"
|
||||||
|
"github.com/matrix-org/dendrite/userapi/util"
|
||||||
|
)
|
||||||
|
|
||||||
|
// OutputReceiptEventConsumer consumes events that originated in the EDU server.
|
||||||
|
type OutputClientDataConsumer struct {
|
||||||
|
ctx context.Context
|
||||||
|
jetstream nats.JetStreamContext
|
||||||
|
durable string
|
||||||
|
topic string
|
||||||
|
db storage.Database
|
||||||
|
serverName gomatrixserverlib.ServerName
|
||||||
|
syncProducer *producers.SyncAPI
|
||||||
|
pgClient pushgateway.Client
|
||||||
|
}
|
||||||
|
|
||||||
|
// NewOutputReceiptEventConsumer creates a new OutputReceiptEventConsumer.
|
||||||
|
// Call Start() to begin consuming from the EDU server.
|
||||||
|
func NewOutputClientDataConsumer(
|
||||||
|
process *process.ProcessContext,
|
||||||
|
cfg *config.UserAPI,
|
||||||
|
js nats.JetStreamContext,
|
||||||
|
store storage.Database,
|
||||||
|
syncProducer *producers.SyncAPI,
|
||||||
|
pgClient pushgateway.Client,
|
||||||
|
) *OutputReceiptEventConsumer {
|
||||||
|
return &OutputReceiptEventConsumer{
|
||||||
|
ctx: process.Context(),
|
||||||
|
jetstream: js,
|
||||||
|
topic: cfg.Matrix.JetStream.Prefixed(jetstream.OutputClientData),
|
||||||
|
durable: cfg.Matrix.JetStream.Durable("UserAPIAccountDataConsumer"),
|
||||||
|
db: store,
|
||||||
|
serverName: cfg.Matrix.ServerName,
|
||||||
|
syncProducer: syncProducer,
|
||||||
|
pgClient: pgClient,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// Start consuming receipts events.
|
||||||
|
func (s *OutputClientDataConsumer) Start() error {
|
||||||
|
return jetstream.JetStreamConsumer(
|
||||||
|
s.ctx, s.jetstream, s.topic, s.durable, 1,
|
||||||
|
s.onMessage, nats.DeliverAll(), nats.ManualAck(),
|
||||||
|
)
|
||||||
|
}
|
||||||
|
|
||||||
|
func (s *OutputClientDataConsumer) onMessage(ctx context.Context, msgs []*nats.Msg) bool {
|
||||||
|
msg := msgs[0] // Guaranteed to exist if onMessage is called
|
||||||
|
|
||||||
|
userID := msg.Header.Get(jetstream.UserID)
|
||||||
|
var output eventutil.AccountData
|
||||||
|
if err := json.Unmarshal(msg.Data, &output); err != nil {
|
||||||
|
// If the message was invalid, log it and move on to the next message in the stream
|
||||||
|
log.WithError(err).Errorf("client API server output log: message parse failure")
|
||||||
|
sentry.CaptureException(err)
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
if output.Type != "m.fully_read" || output.ReadMarker == nil {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
localpart, domain, err := gomatrixserverlib.SplitID('@', userID)
|
||||||
|
if err != nil {
|
||||||
|
log.WithError(err).Error("userapi clientapi consumer: SplitID failure")
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
if domain != s.serverName {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
log := log.WithFields(log.Fields{
|
||||||
|
"room_id": output.RoomID,
|
||||||
|
"user_id": userID,
|
||||||
|
})
|
||||||
|
|
||||||
|
_, serverName, err := gomatrixserverlib.SplitID('@', userID)
|
||||||
|
if err != nil {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
if serverName != s.serverName {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
var notifyUsers bool
|
||||||
|
if output.ReadMarker.Read != "" {
|
||||||
|
_, err = s.db.SetNotificationsRead(ctx, localpart, output.RoomID, output.ReadMarker.Read, true)
|
||||||
|
if err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
notifyUsers = true
|
||||||
|
}
|
||||||
|
|
||||||
|
if output.ReadMarker.FullyRead != "" {
|
||||||
|
_, err := s.db.DeleteNotificationsUpTo(ctx, localpart, output.RoomID, output.ReadMarker.FullyRead)
|
||||||
|
if err != nil {
|
||||||
|
log.WithError(err).Errorf("userapi clientapi consumer: DeleteNotificationsUpTo failed")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
notifyUsers = true
|
||||||
|
}
|
||||||
|
|
||||||
|
if !notifyUsers {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
if err = s.syncProducer.GetAndSendNotificationData(ctx, userID, output.RoomID); err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer: GetAndSendNotificationData failed")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
if err = util.NotifyUserCountsAsync(ctx, s.pgClient, localpart, s.db); err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer: NotifyUserCounts failed")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
// OutputReceiptEventConsumer consumes events that originated in the EDU server.
|
||||||
|
type OutputReceiptEventConsumer struct {
|
||||||
|
ctx context.Context
|
||||||
|
jetstream nats.JetStreamContext
|
||||||
|
durable string
|
||||||
|
topic string
|
||||||
|
db storage.Database
|
||||||
|
serverName gomatrixserverlib.ServerName
|
||||||
|
syncProducer *producers.SyncAPI
|
||||||
|
pgClient pushgateway.Client
|
||||||
|
}
|
||||||
|
|
||||||
|
// NewOutputReceiptEventConsumer creates a new OutputReceiptEventConsumer.
|
||||||
|
// Call Start() to begin consuming from the EDU server.
|
||||||
|
func NewOutputReceiptEventConsumer(
|
||||||
|
process *process.ProcessContext,
|
||||||
|
cfg *config.UserAPI,
|
||||||
|
js nats.JetStreamContext,
|
||||||
|
store storage.Database,
|
||||||
|
syncProducer *producers.SyncAPI,
|
||||||
|
pgClient pushgateway.Client,
|
||||||
|
) *OutputReceiptEventConsumer {
|
||||||
|
return &OutputReceiptEventConsumer{
|
||||||
|
ctx: process.Context(),
|
||||||
|
jetstream: js,
|
||||||
|
topic: cfg.Matrix.JetStream.Prefixed(jetstream.OutputReceiptEvent),
|
||||||
|
durable: cfg.Matrix.JetStream.Durable("UserAPIReceiptConsumer"),
|
||||||
|
db: store,
|
||||||
|
serverName: cfg.Matrix.ServerName,
|
||||||
|
syncProducer: syncProducer,
|
||||||
|
pgClient: pgClient,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// Start consuming receipts events.
|
||||||
|
func (s *OutputReceiptEventConsumer) Start() error {
|
||||||
|
return jetstream.JetStreamConsumer(
|
||||||
|
s.ctx, s.jetstream, s.topic, s.durable, 1,
|
||||||
|
s.onMessage, nats.DeliverAll(), nats.ManualAck(),
|
||||||
|
)
|
||||||
|
}
|
||||||
|
|
||||||
|
func (s *OutputReceiptEventConsumer) onMessage(ctx context.Context, msgs []*nats.Msg) bool {
|
||||||
|
msg := msgs[0] // Guaranteed to exist if onMessage is called
|
||||||
|
|
||||||
|
userID := msg.Header.Get(jetstream.UserID)
|
||||||
|
roomID := msg.Header.Get(jetstream.RoomID)
|
||||||
|
readPos := msg.Header.Get(jetstream.EventID)
|
||||||
|
evType := msg.Header.Get("type")
|
||||||
|
|
||||||
|
if readPos == "" || evType != "m.read" {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
localpart, domain, err := gomatrixserverlib.SplitID('@', userID)
|
||||||
|
if err != nil {
|
||||||
|
log.WithError(err).Error("userapi clientapi consumer: SplitID failure")
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
if domain != s.serverName {
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
||||||
|
log := log.WithFields(log.Fields{
|
||||||
|
"room_id": roomID,
|
||||||
|
"user_id": userID,
|
||||||
|
})
|
||||||
|
|
||||||
|
_, err = s.db.SetNotificationsRead(ctx, localpart, roomID, readPos, true)
|
||||||
|
if err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
|
||||||
|
if err = s.syncProducer.GetAndSendNotificationData(ctx, userID, roomID); err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer: GetAndSendNotificationData failed")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
if err = util.NotifyUserCountsAsync(ctx, s.pgClient, localpart, s.db); err != nil {
|
||||||
|
log.WithError(err).Error("userapi EDU consumer: NotifyUserCounts failed")
|
||||||
|
return false
|
||||||
|
}
|
||||||
|
|
||||||
|
return true
|
||||||
|
}
|
||||||
|
|
@ -1,132 +0,0 @@
|
||||||
package consumers
|
|
||||||
|
|
||||||
import (
|
|
||||||
"context"
|
|
||||||
|
|
||||||
"github.com/matrix-org/gomatrixserverlib"
|
|
||||||
"github.com/nats-io/nats.go"
|
|
||||||
log "github.com/sirupsen/logrus"
|
|
||||||
|
|
||||||
"github.com/matrix-org/dendrite/internal/pushgateway"
|
|
||||||
"github.com/matrix-org/dendrite/setup/config"
|
|
||||||
"github.com/matrix-org/dendrite/setup/jetstream"
|
|
||||||
"github.com/matrix-org/dendrite/setup/process"
|
|
||||||
uapi "github.com/matrix-org/dendrite/userapi/api"
|
|
||||||
"github.com/matrix-org/dendrite/userapi/producers"
|
|
||||||
"github.com/matrix-org/dendrite/userapi/storage"
|
|
||||||
"github.com/matrix-org/dendrite/userapi/util"
|
|
||||||
)
|
|
||||||
|
|
||||||
type OutputReadUpdateConsumer struct {
|
|
||||||
ctx context.Context
|
|
||||||
cfg *config.UserAPI
|
|
||||||
jetstream nats.JetStreamContext
|
|
||||||
durable string
|
|
||||||
db storage.Database
|
|
||||||
pgClient pushgateway.Client
|
|
||||||
ServerName gomatrixserverlib.ServerName
|
|
||||||
topic string
|
|
||||||
userAPI uapi.UserInternalAPI
|
|
||||||
syncProducer *producers.SyncAPI
|
|
||||||
}
|
|
||||||
|
|
||||||
func NewOutputReadUpdateConsumer(
|
|
||||||
process *process.ProcessContext,
|
|
||||||
cfg *config.UserAPI,
|
|
||||||
js nats.JetStreamContext,
|
|
||||||
store storage.Database,
|
|
||||||
pgClient pushgateway.Client,
|
|
||||||
userAPI uapi.UserInternalAPI,
|
|
||||||
syncProducer *producers.SyncAPI,
|
|
||||||
) *OutputReadUpdateConsumer {
|
|
||||||
return &OutputReadUpdateConsumer{
|
|
||||||
ctx: process.Context(),
|
|
||||||
cfg: cfg,
|
|
||||||
jetstream: js,
|
|
||||||
db: store,
|
|
||||||
ServerName: cfg.Matrix.ServerName,
|
|
||||||
durable: cfg.Matrix.JetStream.Durable("UserAPISyncAPIReadUpdateConsumer"),
|
|
||||||
topic: cfg.Matrix.JetStream.Prefixed(jetstream.OutputReadUpdate),
|
|
||||||
pgClient: pgClient,
|
|
||||||
userAPI: userAPI,
|
|
||||||
syncProducer: syncProducer,
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
func (s *OutputReadUpdateConsumer) Start() error {
|
|
||||||
if err := jetstream.JetStreamConsumer(
|
|
||||||
s.ctx, s.jetstream, s.topic, s.durable, 1,
|
|
||||||
s.onMessage, nats.DeliverAll(), nats.ManualAck(),
|
|
||||||
); err != nil {
|
|
||||||
return err
|
|
||||||
}
|
|
||||||
return nil
|
|
||||||
}
|
|
||||||
|
|
||||||
func (s *OutputReadUpdateConsumer) onMessage(ctx context.Context, msgs []*nats.Msg) bool {
|
|
||||||
msg := msgs[0] // Guaranteed to exist if onMessage is called
|
|
||||||
userID := msg.Header.Get(jetstream.UserID)
|
|
||||||
roomID := msg.Header.Get(jetstream.RoomID)
|
|
||||||
readPos := msg.Header.Get("read")
|
|
||||||
fullyReadPos := msg.Header.Get("fully_read")
|
|
||||||
|
|
||||||
if readPos == "" && fullyReadPos == "" {
|
|
||||||
return true
|
|
||||||
}
|
|
||||||
|
|
||||||
localpart, domain, err := gomatrixserverlib.SplitID('@', userID)
|
|
||||||
if err != nil {
|
|
||||||
log.WithError(err).Error("userapi clientapi consumer: SplitID failure")
|
|
||||||
return true
|
|
||||||
}
|
|
||||||
if domain != s.ServerName {
|
|
||||||
log.Error("userapi clientapi consumer: not a local user")
|
|
||||||
return true
|
|
||||||
}
|
|
||||||
|
|
||||||
log := log.WithFields(log.Fields{
|
|
||||||
"room_id": roomID,
|
|
||||||
"user_id": userID,
|
|
||||||
})
|
|
||||||
|
|
||||||
if readPos != "" {
|
|
||||||
updated, err := s.db.SetNotificationsRead(ctx, localpart, roomID, readPos, true)
|
|
||||||
if err != nil {
|
|
||||||
log.WithError(err).Error("userapi EDU consumer")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
|
|
||||||
if updated {
|
|
||||||
if err = s.syncProducer.GetAndSendNotificationData(ctx, userID, roomID); err != nil {
|
|
||||||
log.WithError(err).Error("userapi EDU consumer: GetAndSendNotificationData failed")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
if err = util.NotifyUserCountsAsync(ctx, s.pgClient, localpart, s.db); err != nil {
|
|
||||||
log.WithError(err).Error("userapi EDU consumer: NotifyUserCounts failed")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
if fullyReadPos != "" {
|
|
||||||
deleted, err := s.db.DeleteNotificationsUpTo(ctx, localpart, roomID, fullyReadPos)
|
|
||||||
if err != nil {
|
|
||||||
log.WithError(err).Errorf("userapi clientapi consumer: DeleteNotificationsUpTo failed")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
|
|
||||||
if deleted {
|
|
||||||
if err := util.NotifyUserCountsAsync(ctx, s.pgClient, localpart, s.db); err != nil {
|
|
||||||
log.WithError(err).Error("userapi clientapi consumer: NotifyUserCounts failed")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
|
|
||||||
if err := s.syncProducer.GetAndSendNotificationData(ctx, userID, roomID); err != nil {
|
|
||||||
log.WithError(err).Errorf("userapi clientapi consumer: GetAndSendNotificationData failed")
|
|
||||||
return false
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
return true
|
|
||||||
}
|
|
||||||
|
|
@ -83,11 +83,18 @@ func NewInternalAPI(
|
||||||
DisableTLSValidation: cfg.PushGatewayDisableTLSValidation,
|
DisableTLSValidation: cfg.PushGatewayDisableTLSValidation,
|
||||||
}
|
}
|
||||||
|
|
||||||
readConsumer := consumers.NewOutputReadUpdateConsumer(
|
receiptConsumer := consumers.NewOutputReceiptEventConsumer(
|
||||||
base.ProcessContext, cfg, js, db, pgClient, userAPI, syncProducer,
|
base.ProcessContext, cfg, js, db, syncProducer, pgClient,
|
||||||
)
|
)
|
||||||
if err := readConsumer.Start(); err != nil {
|
if err := receiptConsumer.Start(); err != nil {
|
||||||
logrus.WithError(err).Panic("failed to start user API read update consumer")
|
logrus.WithError(err).Panic("failed to start user API receipt consumer")
|
||||||
|
}
|
||||||
|
|
||||||
|
readMarkerConsumer := consumers.NewOutputClientDataConsumer(
|
||||||
|
base.ProcessContext, cfg, js, db, syncProducer, pgClient,
|
||||||
|
)
|
||||||
|
if err := readMarkerConsumer.Start(); err != nil {
|
||||||
|
logrus.WithError(err).Panic("failed to start user API read marker consumer")
|
||||||
}
|
}
|
||||||
|
|
||||||
eventConsumer := consumers.NewOutputRoomEventConsumer(
|
eventConsumer := consumers.NewOutputRoomEventConsumer(
|
||||||
|
|
|
||||||
Loading…
Reference in a new issue