score.go 25.2 KB
Newer Older
vyzo's avatar
vyzo committed
1 2 3
package pubsub

import (
vyzo's avatar
vyzo committed
4
	"context"
vyzo's avatar
vyzo committed
5
	"fmt"
6
	"net"
vyzo's avatar
vyzo committed
7
	"sync"
vyzo's avatar
vyzo committed
8 9
	"time"

vyzo's avatar
vyzo committed
10
	"github.com/libp2p/go-libp2p-core/host"
vyzo's avatar
vyzo committed
11 12
	"github.com/libp2p/go-libp2p-core/peer"
	"github.com/libp2p/go-libp2p-core/protocol"
vyzo's avatar
vyzo committed
13

14
	manet "github.com/multiformats/go-multiaddr-net"
vyzo's avatar
vyzo committed
15 16
)

vyzo's avatar
vyzo committed
17 18 19 20 21 22 23 24 25 26 27 28
type peerStats struct {
	// true if the peer is currently connected
	connected bool

	// expiration time of the score stats for disconnected peers
	expire time.Time

	// per topc stats
	topics map[string]*topicStats

	// IP tracking; store as string for easy processing
	ips []string
vyzo's avatar
vyzo committed
29 30 31

	// behavioural pattern penalties (applied by the router)
	behaviourPenalty float64
vyzo's avatar
vyzo committed
32 33 34 35 36 37 38 39 40 41 42 43 44 45 46 47 48 49 50 51 52 53
}

type topicStats struct {
	// true if the peer is in the mesh
	inMesh bool

	// time when the peer was (last) GRAFTed; valid only when in mesh
	graftTime time.Time

	// time in mesh (updated during refresh/decay to avoid calling gettimeofday on
	// every score invocation)
	meshTime time.Duration

	// first message deliveries
	firstMessageDeliveries float64

	// mesh message deliveries
	meshMessageDeliveries float64

	// true if the peer has been enough time in the mesh to activate mess message deliveries
	meshMessageDeliveriesActive bool

vyzo's avatar
vyzo committed
54 55 56
	// sticky mesh rate failure penalty counter
	meshFailurePenalty float64

vyzo's avatar
vyzo committed
57 58 59 60
	// invalid message counter
	invalidMessageDeliveries float64
}

vyzo's avatar
vyzo committed
61
type peerScore struct {
vyzo's avatar
vyzo committed
62 63 64 65 66 67 68 69
	sync.Mutex

	// the score parameters
	params *PeerScoreParams

	// per peer stats for score calculation
	peerStats map[peer.ID]*peerStats

Raúl Kripalani's avatar
Raúl Kripalani committed
70
	// IP colocation tracking; maps IP => set of peers.
vyzo's avatar
vyzo committed
71
	peerIPs map[string]map[peer.ID]struct{}
vyzo's avatar
vyzo committed
72 73 74 75 76

	// message delivery tracking
	deliveries *messageDeliveries

	msgID MsgIdFunction
vyzo's avatar
vyzo committed
77
	host  host.Host
78 79 80

	// debugging inspection
	inspect       PeerScoreInspectFn
vyzo's avatar
vyzo committed
81
	inspectEx     ExtendedPeerScoreInspectFn
82
	inspectPeriod time.Duration
vyzo's avatar
vyzo committed
83 84
}

85
var _ internalTracer = (*peerScore)(nil)
Raúl Kripalani's avatar
Raúl Kripalani committed
86

vyzo's avatar
vyzo committed
87
type messageDeliveries struct {
88 89 90 91 92
	records map[string]*deliveryRecord

	// queue for cleaning up old delivery records
	head *deliveryEntry
	tail *deliveryEntry
vyzo's avatar
vyzo committed
93 94 95 96
}

type deliveryRecord struct {
	status    int
97
	firstSeen time.Time
98
	validated time.Time
99
	peers     map[peer.ID]struct{}
vyzo's avatar
vyzo committed
100 101
}

102 103 104 105 106 107
type deliveryEntry struct {
	id     string
	expire time.Time
	next   *deliveryEntry
}

vyzo's avatar
vyzo committed
108 109
// delivery record status
const (
110 111 112
	deliveryUnknown   = iota // we don't know (yet) if the message is valid
	deliveryValid            // we know the message is valid
	deliveryInvalid          // we know the message is invalid
vyzo's avatar
vyzo committed
113
	deliveryIgnored          // we were intructed by the validator to ignore the message
114
	deliveryThrottled        // we can't tell if it is valid because validation throttled
vyzo's avatar
vyzo committed
115 116
)

117
type PeerScoreInspectFn func(map[peer.ID]float64)
vyzo's avatar
vyzo committed
118 119 120 121 122 123 124 125 126 127 128 129 130 131 132 133
type ExtendedPeerScoreInspectFn func(map[peer.ID]*PeerScoreSnapshot)

type PeerScoreSnapshot struct {
	Score              float64
	Topics             map[string]*TopicScoreSnapshot
	AppSpecificScore   float64
	IPColocationFactor int
	BehaviourPenalty   float64
}

type TopicScoreSnapshot struct {
	TimeInMesh               time.Duration
	FirstMessageDeliveries   float64
	MeshMessageDeliveries    float64
	InvalidMessageDeliveries float64
}
134

135
// WithPeerScoreInspect is a gossipsub router option that enables peer score debugging.
136
// When this option is enabled, the supplied function will be invoked periodically to allow
vyzo's avatar
vyzo committed
137 138 139 140 141 142
// the application to inspect or dump the scores for connected peers.
// The supplied function can have one of two signatures:
//  - PeerScoreInspectFn, which takes a map of peer IDs to score.
//  - ExtendedPeerScoreInspectFn, which takes a map of peer IDs to
//    PeerScoreSnapshots and allows inspection of individual score
//    components for debugging peer scoring.
143
// This option must be passed _after_ the WithPeerScore option.
vyzo's avatar
vyzo committed
144
func WithPeerScoreInspect(inspect interface{}, period time.Duration) Option {
145 146 147 148 149 150 151 152 153 154
	return func(ps *PubSub) error {
		gs, ok := ps.rt.(*GossipSubRouter)
		if !ok {
			return fmt.Errorf("pubsub router is not gossipsub")
		}

		if gs.score == nil {
			return fmt.Errorf("peer scoring is not enabled")
		}

vyzo's avatar
vyzo committed
155 156 157 158 159 160 161 162 163 164 165 166 167
		switch i := inspect.(type) {
		case PeerScoreInspectFn:
			gs.score.inspect = i
		case func(map[peer.ID]float64):
			gs.score.inspect = PeerScoreInspectFn(i)
		case ExtendedPeerScoreInspectFn:
			gs.score.inspectEx = i
		case func(map[peer.ID]*PeerScoreSnapshot):
			gs.score.inspectEx = ExtendedPeerScoreInspectFn(i)
		default:
			return fmt.Errorf("unknown peer score insector type: %v", inspect)
		}

168 169 170 171 172 173 174
		gs.score.inspectPeriod = period

		return nil
	}
}

// implementation
175
func newPeerScore(params *PeerScoreParams) *peerScore {
vyzo's avatar
vyzo committed
176 177 178 179 180 181 182
	return &peerScore{
		params:     params,
		peerStats:  make(map[peer.ID]*peerStats),
		peerIPs:    make(map[string]map[peer.ID]struct{}),
		deliveries: &messageDeliveries{records: make(map[string]*deliveryRecord)},
		msgID:      DefaultMsgIdFn,
	}
vyzo's avatar
vyzo committed
183 184 185
}

// router interface
186
func (ps *peerScore) Start(gs *GossipSubRouter) {
187 188 189 190
	if ps == nil {
		return
	}

vyzo's avatar
vyzo committed
191 192 193
	ps.msgID = gs.p.msgID
	ps.host = gs.p.host
	go ps.background(gs.p.ctx)
194 195
}

196
func (ps *peerScore) Score(p peer.ID) float64 {
vyzo's avatar
vyzo committed
197 198 199 200
	if ps == nil {
		return 0
	}

vyzo's avatar
vyzo committed
201 202 203
	ps.Lock()
	defer ps.Unlock()

204 205 206 207
	return ps.score(p)
}

func (ps *peerScore) score(p peer.ID) float64 {
vyzo's avatar
vyzo committed
208 209 210 211 212 213 214 215 216 217 218 219 220 221 222 223
	pstats, ok := ps.peerStats[p]
	if !ok {
		return 0
	}

	var score float64

	// topic scores
	for topic, tstats := range pstats.topics {
		// the topic parameters
		topicParams, ok := ps.params.Topics[topic]
		if !ok {
			// we are not scoring this topic
			continue
		}

Raúl Kripalani's avatar
Raúl Kripalani committed
224 225 226
		// the topic score
		var topicScore float64

vyzo's avatar
vyzo committed
227 228 229
		// P1: time in Mesh
		if tstats.inMesh {
			p1 := float64(tstats.meshTime / topicParams.TimeInMeshQuantum)
vyzo's avatar
vyzo committed
230 231 232
			if p1 > topicParams.TimeInMeshCap {
				p1 = topicParams.TimeInMeshCap
			}
vyzo's avatar
vyzo committed
233 234 235 236 237 238 239 240 241 242 243 244 245 246 247 248
			topicScore += p1 * topicParams.TimeInMeshWeight
		}

		// P2: first message deliveries
		p2 := tstats.firstMessageDeliveries
		topicScore += p2 * topicParams.FirstMessageDeliveriesWeight

		// P3: mesh message deliveries
		if tstats.meshMessageDeliveriesActive {
			if tstats.meshMessageDeliveries < topicParams.MeshMessageDeliveriesThreshold {
				deficit := topicParams.MeshMessageDeliveriesThreshold - tstats.meshMessageDeliveries
				p3 := deficit * deficit
				topicScore += p3 * topicParams.MeshMessageDeliveriesWeight
			}
		}

vyzo's avatar
vyzo committed
249
		// P3b:
Raúl Kripalani's avatar
Raúl Kripalani committed
250
		// NOTE: the weight of P3b is negative (validated in TopicScoreParams.validate), so this detracts.
vyzo's avatar
vyzo committed
251 252 253
		p3b := tstats.meshFailurePenalty
		topicScore += p3b * topicParams.MeshFailurePenaltyWeight

vyzo's avatar
vyzo committed
254
		// P4: invalid messages
Raúl Kripalani's avatar
Raúl Kripalani committed
255
		// NOTE: the weight of P4 is negative (validated in TopicScoreParams.validate), so this detracts.
vyzo's avatar
vyzo committed
256
		p4 := (tstats.invalidMessageDeliveries * tstats.invalidMessageDeliveries)
vyzo's avatar
vyzo committed
257 258 259 260 261 262
		topicScore += p4 * topicParams.InvalidMessageDeliveriesWeight

		// update score, mixing with topic weight
		score += topicScore * topicParams.TopicWeight
	}

vyzo's avatar
vyzo committed
263 264 265 266 267
	// apply the topic score cap, if any
	if ps.params.TopicScoreCap > 0 && score > ps.params.TopicScoreCap {
		score = ps.params.TopicScoreCap
	}

vyzo's avatar
vyzo committed
268 269 270 271 272 273
	// P5: application-specific score
	p5 := ps.params.AppSpecificScore(p)
	score += p5 * ps.params.AppSpecificWeight

	// P6: IP collocation factor
	for _, ip := range pstats.ips {
274 275 276 277 278
		_, whitelisted := ps.params.IPColocationFactorWhitelist[ip]
		if whitelisted {
			continue
		}

Raúl Kripalani's avatar
Raúl Kripalani committed
279 280 281 282
		// P6 has a cliff (IPColocationFactorThreshold); it's only applied iff
		// at least that many peers are connected to us from that source IP
		// addr. It is quadratic, and the weight is negative (validated by
		// PeerScoreParams.validate).
vyzo's avatar
vyzo committed
283 284 285 286 287 288 289 290
		peersInIP := len(ps.peerIPs[ip])
		if peersInIP > ps.params.IPColocationFactorThreshold {
			surpluss := float64(peersInIP - ps.params.IPColocationFactorThreshold)
			p6 := surpluss * surpluss
			score += p6 * ps.params.IPColocationFactorWeight
		}
	}

vyzo's avatar
vyzo committed
291 292 293 294
	// P7: behavioural pattern penalty
	p7 := pstats.behaviourPenalty * pstats.behaviourPenalty
	score += p7 * ps.params.BehaviourPenaltyWeight

vyzo's avatar
vyzo committed
295
	return score
vyzo's avatar
vyzo committed
296 297
}

vyzo's avatar
vyzo committed
298 299 300 301 302 303 304 305 306 307 308 309 310 311 312 313 314
// behavioural pattern penalties
func (ps *peerScore) AddPenalty(p peer.ID, count int) {
	if ps == nil {
		return
	}

	ps.Lock()
	defer ps.Unlock()

	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

	pstats.behaviourPenalty += float64(count)
}

315
// periodic maintenance
vyzo's avatar
vyzo committed
316 317 318 319 320 321 322 323 324 325
func (ps *peerScore) background(ctx context.Context) {
	refreshScores := time.NewTicker(ps.params.DecayInterval)
	defer refreshScores.Stop()

	refreshIPs := time.NewTicker(time.Minute)
	defer refreshIPs.Stop()

	gcDeliveryRecords := time.NewTicker(time.Minute)
	defer gcDeliveryRecords.Stop()

326
	var inspectScores <-chan time.Time
vyzo's avatar
vyzo committed
327
	if ps.inspect != nil || ps.inspectEx != nil {
328 329
		ticker := time.NewTicker(ps.inspectPeriod)
		defer ticker.Stop()
330 331
		// also dump at exit for one final sample
		defer ps.inspectScores()
332 333 334
		inspectScores = ticker.C
	}

vyzo's avatar
vyzo committed
335 336 337 338 339 340 341 342 343 344 345
	for {
		select {
		case <-refreshScores.C:
			ps.refreshScores()

		case <-refreshIPs.C:
			ps.refreshIPs()

		case <-gcDeliveryRecords.C:
			ps.gcDeliveryRecords()

346 347 348
		case <-inspectScores:
			ps.inspectScores()

vyzo's avatar
vyzo committed
349 350 351 352 353 354
		case <-ctx.Done():
			return
		}
	}
}

Raúl Kripalani's avatar
Raúl Kripalani committed
355
// inspectScores dumps all tracked scores into the inspect function.
356
func (ps *peerScore) inspectScores() {
vyzo's avatar
vyzo committed
357 358 359 360 361 362 363 364
	if ps.inspect != nil {
		ps.inspectScoresSimple()
	} else if ps.inspectEx != nil {
		ps.inspectScoresExtended()
	}
}

func (ps *peerScore) inspectScoresSimple() {
365 366 367 368 369 370 371
	ps.Lock()
	scores := make(map[peer.ID]float64, len(ps.peerStats))
	for p := range ps.peerStats {
		scores[p] = ps.score(p)
	}
	ps.Unlock()

Raúl Kripalani's avatar
Raúl Kripalani committed
372 373 374 375 376
	// Since this is a user-injected function, it could be performing I/O, and
	// we don't want to block the scorer's background loop. Therefore, we launch
	// it in a separate goroutine. If the function needs to synchronise, it
	// should do so locally.
	go ps.inspect(scores)
377 378
}

vyzo's avatar
vyzo committed
379 380 381 382 383 384 385 386 387 388 389 390 391 392 393 394 395 396 397 398 399 400 401 402 403 404 405 406 407 408 409 410 411 412 413 414 415 416 417
func (ps *peerScore) inspectScoresExtended() {
	ps.Lock()
	scores := make(map[peer.ID]*PeerScoreSnapshot, len(ps.peerStats))
	for p, pstats := range ps.peerStats {
		pss := new(PeerScoreSnapshot)
		pss.Score = ps.score(p)
		if len(pstats.topics) > 0 {
			pss.Topics = make(map[string]*TopicScoreSnapshot, len(pstats.topics))
			for t, ts := range pstats.topics {
				pss.Topics[t] = &TopicScoreSnapshot{
					FirstMessageDeliveries:   ts.firstMessageDeliveries,
					MeshMessageDeliveries:    ts.meshMessageDeliveries,
					InvalidMessageDeliveries: ts.invalidMessageDeliveries,
				}
				if ts.inMesh {
					pss.Topics[t].TimeInMesh = ts.meshTime
				}
			}
		}
		pss.AppSpecificScore = ps.params.AppSpecificScore(p)
		for _, ip := range pstats.ips {
			_, whitelisted := ps.params.IPColocationFactorWhitelist[ip]
			if whitelisted {
				continue
			}
			peersInIP := len(ps.peerIPs[ip])
			if peersInIP > ps.params.IPColocationFactorThreshold {
				surpluss := peersInIP - ps.params.IPColocationFactorThreshold
				pss.IPColocationFactor += surpluss
			}
		}
		pss.BehaviourPenalty = pstats.behaviourPenalty
		scores[p] = pss
	}
	ps.Unlock()

	go ps.inspectEx(scores)
}

Raúl Kripalani's avatar
Raúl Kripalani committed
418 419
// refreshScores decays scores, and purges score records for disconnected peers,
// once their expiry has elapsed.
420 421 422 423 424 425 426 427 428
func (ps *peerScore) refreshScores() {
	ps.Lock()
	defer ps.Unlock()

	now := time.Now()
	for p, pstats := range ps.peerStats {
		if !pstats.connected {
			// has the retention period expired?
			if now.After(pstats.expire) {
vyzo's avatar
vyzo committed
429 430
				// yes, throw it away (but clean up the IP tracking first)
				ps.removeIPs(p, pstats.ips)
431 432 433 434 435 436 437 438 439 440 441 442 443 444 445 446 447 448 449 450
				delete(ps.peerStats, p)
			}

			// we don't decay retained scores, as the peer is not active.
			// this way the peer cannot reset a negative score by simply disconnecting and reconnecting,
			// unless the retention period has ellapsed.
			// similarly, a well behaved peer does not lose its score by getting disconnected.
			continue
		}

		for topic, tstats := range pstats.topics {
			// the topic parameters
			topicParams, ok := ps.params.Topics[topic]
			if !ok {
				// we are not scoring this topic
				continue
			}

			// decay counters
			tstats.firstMessageDeliveries *= topicParams.FirstMessageDeliveriesDecay
451 452 453
			if tstats.firstMessageDeliveries < ps.params.DecayToZero {
				tstats.firstMessageDeliveries = 0
			}
454
			tstats.meshMessageDeliveries *= topicParams.MeshMessageDeliveriesDecay
455 456 457
			if tstats.meshMessageDeliveries < ps.params.DecayToZero {
				tstats.meshMessageDeliveries = 0
			}
vyzo's avatar
vyzo committed
458 459 460 461
			tstats.meshFailurePenalty *= topicParams.MeshFailurePenaltyDecay
			if tstats.meshFailurePenalty < ps.params.DecayToZero {
				tstats.meshFailurePenalty = 0
			}
462
			tstats.invalidMessageDeliveries *= topicParams.InvalidMessageDeliveriesDecay
463 464 465
			if tstats.invalidMessageDeliveries < ps.params.DecayToZero {
				tstats.invalidMessageDeliveries = 0
			}
466 467 468 469 470 471 472 473
			// update mesh time and activate mesh message delivery parameter if need be
			if tstats.inMesh {
				tstats.meshTime = now.Sub(tstats.graftTime)
				if tstats.meshTime > topicParams.MeshMessageDeliveriesActivation {
					tstats.meshMessageDeliveriesActive = true
				}
			}
		}
vyzo's avatar
vyzo committed
474 475 476 477 478 479

		// decay P7 counter
		pstats.behaviourPenalty *= ps.params.BehaviourPenaltyDecay
		if pstats.behaviourPenalty < ps.params.DecayToZero {
			pstats.behaviourPenalty = 0
		}
480 481 482
	}
}

Raúl Kripalani's avatar
Raúl Kripalani committed
483
// refreshIPs refreshes IPs we know of peers we're tracking.
vyzo's avatar
vyzo committed
484 485 486 487
func (ps *peerScore) refreshIPs() {
	ps.Lock()
	defer ps.Unlock()

488
	// peer IPs may change, so we periodically refresh them
Raúl Kripalani's avatar
Raúl Kripalani committed
489 490 491 492 493
	//
	// TODO: it could be more efficient to collect connections for all peers
	// from the Network, populate a new map, and replace it in place. We are
	// incurring in those allocs anyway, and maybe even in more, in the form of
	// slices.
vyzo's avatar
vyzo committed
494 495 496 497 498 499 500
	for p, pstats := range ps.peerStats {
		if pstats.connected {
			ips := ps.getIPs(p)
			ps.setIPs(p, ips, pstats.ips)
			pstats.ips = ips
		}
	}
501 502
}

vyzo's avatar
vyzo committed
503 504 505 506 507 508 509
func (ps *peerScore) gcDeliveryRecords() {
	ps.Lock()
	defer ps.Unlock()

	ps.deliveries.gc()
}

vyzo's avatar
vyzo committed
510 511
// tracer interface
func (ps *peerScore) AddPeer(p peer.ID, proto protocol.ID) {
vyzo's avatar
vyzo committed
512 513 514 515 516 517 518 519 520 521 522
	ps.Lock()
	defer ps.Unlock()

	pstats, ok := ps.peerStats[p]
	if !ok {
		pstats = &peerStats{topics: make(map[string]*topicStats)}
		ps.peerStats[p] = pstats
	}

	pstats.connected = true
	ips := ps.getIPs(p)
vyzo's avatar
vyzo committed
523
	ps.setIPs(p, ips, pstats.ips)
vyzo's avatar
vyzo committed
524
	pstats.ips = ips
vyzo's avatar
vyzo committed
525 526 527
}

func (ps *peerScore) RemovePeer(p peer.ID) {
vyzo's avatar
vyzo committed
528 529 530 531 532 533 534 535
	ps.Lock()
	defer ps.Unlock()

	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

vyzo's avatar
vyzo committed
536 537 538 539 540 541 542 543 544 545 546 547 548 549 550 551 552 553 554 555 556 557
	// decide whether to retain the score; this currently only retains non-positive scores
	// to dissuade attacks on the score function.
	if ps.score(p) > 0 {
		ps.removeIPs(p, pstats.ips)
		delete(ps.peerStats, p)
		return
	}

	// furthermore, when we decide to retain the score, the firstMessageDelivery counters are
	// reset to 0 and mesh delivery penalties applied.
	for topic, tstats := range pstats.topics {
		tstats.firstMessageDeliveries = 0

		threshold := ps.params.Topics[topic].MeshMessageDeliveriesThreshold
		if tstats.inMesh && tstats.meshMessageDeliveriesActive && tstats.meshMessageDeliveries < threshold {
			deficit := threshold - tstats.meshMessageDeliveries
			tstats.meshFailurePenalty += deficit * deficit
		}

		tstats.inMesh = false
	}

vyzo's avatar
vyzo committed
558 559
	pstats.connected = false
	pstats.expire = time.Now().Add(ps.params.RetainScore)
vyzo's avatar
vyzo committed
560
}
vyzo's avatar
vyzo committed
561

vyzo's avatar
vyzo committed
562 563 564 565
func (ps *peerScore) Join(topic string)  {}
func (ps *peerScore) Leave(topic string) {}

func (ps *peerScore) Graft(p peer.ID, topic string) {
vyzo's avatar
vyzo committed
566 567 568 569 570 571 572 573 574 575 576 577 578 579 580 581
	ps.Lock()
	defer ps.Unlock()

	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

	tstats, ok := pstats.getTopicStats(topic, ps.params)
	if !ok {
		return
	}

	tstats.inMesh = true
	tstats.graftTime = time.Now()
	tstats.meshTime = 0
vyzo's avatar
vyzo committed
582
	tstats.meshMessageDeliveriesActive = false
vyzo's avatar
vyzo committed
583 584
}

vyzo's avatar
vyzo committed
585
func (ps *peerScore) Prune(p peer.ID, topic string) {
vyzo's avatar
vyzo committed
586 587 588 589 590 591 592 593 594 595 596 597 598
	ps.Lock()
	defer ps.Unlock()

	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

	tstats, ok := pstats.getTopicStats(topic, ps.params)
	if !ok {
		return
	}

599 600 601 602 603
	// sticky mesh delivery rate failure penalty
	threshold := ps.params.Topics[topic].MeshMessageDeliveriesThreshold
	if tstats.meshMessageDeliveriesActive && tstats.meshMessageDeliveries < threshold {
		deficit := threshold - tstats.meshMessageDeliveries
		tstats.meshFailurePenalty += deficit * deficit
vyzo's avatar
vyzo committed
604 605
	}

vyzo's avatar
vyzo committed
606
	tstats.inMesh = false
vyzo's avatar
vyzo committed
607
}
vyzo's avatar
vyzo committed
608

vyzo's avatar
vyzo committed
609 610 611 612 613 614 615 616 617
func (ps *peerScore) ValidateMessage(msg *Message) {
	ps.Lock()
	defer ps.Unlock()

	// the pubsub subsystem is beginning validation; create a record to track time in
	// the validation pipeline with an accurate firstSeen time.
	_ = ps.deliveries.getRecord(ps.msgID(msg.Message))
}

vyzo's avatar
vyzo committed
618 619 620 621 622 623 624 625
func (ps *peerScore) DeliverMessage(msg *Message) {
	ps.Lock()
	defer ps.Unlock()

	ps.markFirstMessageDelivery(msg.ReceivedFrom, msg)

	drec := ps.deliveries.getRecord(ps.msgID(msg.Message))

626 627
	// defensive check that this is the first delivery trace -- delivery status should be unknown
	if drec.status != deliveryUnknown {
vyzo's avatar
vyzo committed
628
		log.Warnf("unexpected delivery trace: message from %s was first seen %s ago and has delivery status %d", msg.ReceivedFrom, time.Now().Sub(drec.firstSeen), drec.status)
629 630 631
		return
	}

632
	// mark the message as valid and reward mesh peers that have already forwarded it to us
633
	drec.status = deliveryValid
634
	drec.validated = time.Now()
635
	for p := range drec.peers {
636 637 638 639 640
		// this check is to make sure a peer can't send us a message twice and get a double count
		// if it is a first delivery.
		if p != msg.ReceivedFrom {
			ps.markDuplicateMessageDelivery(p, msg, time.Time{})
		}
vyzo's avatar
vyzo committed
641
	}
vyzo's avatar
vyzo committed
642 643
}

vyzo's avatar
vyzo committed
644 645 646 647
func (ps *peerScore) RejectMessage(msg *Message, reason string) {
	ps.Lock()
	defer ps.Unlock()

648 649
	switch reason {
	// we don't track those messages, but we penalize the peer as they are clearly invalid
650
	case rejectMissingSignature:
651
		fallthrough
652
	case rejectInvalidSignature:
653
		fallthrough
654 655 656 657
	case rejectUnexpectedSignature:
		fallthrough
	case rejectUnexpectedAuthInfo:
		fallthrough
658
	case rejectSelfOrigin:
vyzo's avatar
vyzo committed
659
		ps.markInvalidMessageDelivery(msg.ReceivedFrom, msg)
660 661 662
		return

		// we ignore those messages, so do nothing.
663
	case rejectBlacklstedPeer:
664
		fallthrough
665
	case rejectBlacklistedSource:
vyzo's avatar
vyzo committed
666
		return
667

668
	case rejectValidationQueueFull:
669 670 671 672
		// the message was rejected before it entered the validation pipeline;
		// we don't know if this message has a valid signature, and thus we also don't know if
		// it has a valid message ID; all we can do is ignore it.
		return
vyzo's avatar
vyzo committed
673 674 675
	}

	drec := ps.deliveries.getRecord(ps.msgID(msg.Message))
vyzo's avatar
vyzo committed
676

677 678
	// defensive check that this is the first rejection trace -- delivery status should be unknown
	if drec.status != deliveryUnknown {
vyzo's avatar
vyzo committed
679
		log.Warnf("unexpected rejection trace: message from %s was first seen %s ago and has delivery status %d", msg.ReceivedFrom, time.Now().Sub(drec.firstSeen), drec.status)
680 681 682
		return
	}

vyzo's avatar
vyzo committed
683 684
	switch reason {
	case rejectValidationThrottled:
vyzo's avatar
vyzo committed
685 686
		// if we reject with "validation throttled" we don't penalize the peer(s) that forward it
		// because we don't know if it was valid.
687
		drec.status = deliveryThrottled
vyzo's avatar
vyzo committed
688 689 690
		// release the delivery time tracking map to free some memory early
		drec.peers = nil
		return
vyzo's avatar
vyzo committed
691 692 693 694 695 696
	case rejectValidationIgnored:
		// we were explicitly instructed by the validator to ignore the message but not penalize
		// the peer
		drec.status = deliveryIgnored
		drec.peers = nil
		return
vyzo's avatar
vyzo committed
697 698 699
	}

	// mark the message as invalid and penalize peers that have already forwarded it.
700
	drec.status = deliveryInvalid
701 702

	ps.markInvalidMessageDelivery(msg.ReceivedFrom, msg)
vyzo's avatar
vyzo committed
703 704 705 706 707 708
	for p := range drec.peers {
		ps.markInvalidMessageDelivery(p, msg)
	}

	// release the delivery time tracking map to free some memory early
	drec.peers = nil
vyzo's avatar
vyzo committed
709 710
}

vyzo's avatar
vyzo committed
711 712 713 714 715 716 717 718 719 720 721
func (ps *peerScore) DuplicateMessage(msg *Message) {
	ps.Lock()
	defer ps.Unlock()

	drec := ps.deliveries.getRecord(ps.msgID(msg.Message))

	_, ok := drec.peers[msg.ReceivedFrom]
	if ok {
		// we have already seen this duplicate!
		return
	}
vyzo's avatar
vyzo committed
722

vyzo's avatar
vyzo committed
723
	switch drec.status {
724
	case deliveryUnknown:
725
		// the message is being validated; track the peer delivery and wait for
vyzo's avatar
vyzo committed
726
		// the Deliver/Reject notification.
727
		drec.peers[msg.ReceivedFrom] = struct{}{}
vyzo's avatar
vyzo committed
728

729
	case deliveryValid:
vyzo's avatar
vyzo committed
730
		// mark the peer delivery time to only count a duplicate delivery once.
731
		drec.peers[msg.ReceivedFrom] = struct{}{}
732
		ps.markDuplicateMessageDelivery(msg.ReceivedFrom, msg, drec.validated)
vyzo's avatar
vyzo committed
733

734
	case deliveryInvalid:
vyzo's avatar
vyzo committed
735 736 737
		// we no longer track delivery time
		ps.markInvalidMessageDelivery(msg.ReceivedFrom, msg)

738
	case deliveryThrottled:
739
		// the message was throttled; do nothing (we don't know if it was valid)
vyzo's avatar
vyzo committed
740 741
	case deliveryIgnored:
		// the message was ignored; do nothing
vyzo's avatar
vyzo committed
742
	}
vyzo's avatar
vyzo committed
743 744
}

vyzo's avatar
vyzo committed
745 746
// message delivery records
func (d *messageDeliveries) getRecord(id string) *deliveryRecord {
747 748 749 750 751
	rec, ok := d.records[id]
	if ok {
		return rec
	}

752 753 754
	now := time.Now()

	rec = &deliveryRecord{peers: make(map[peer.ID]struct{}), firstSeen: now}
755 756
	d.records[id] = rec

757
	entry := &deliveryEntry{id: id, expire: now.Add(TimeCacheDuration)}
758 759 760 761 762 763 764 765 766 767 768 769 770 771 772 773 774 775 776 777 778 779 780 781 782
	if d.tail != nil {
		d.tail.next = entry
		d.tail = entry
	} else {
		d.head = entry
		d.tail = entry
	}

	return rec
}

func (d *messageDeliveries) gc() {
	if d.head == nil {
		return
	}

	now := time.Now()
	for d.head != nil && now.After(d.head.expire) {
		delete(d.records, d.head.id)
		d.head = d.head.next
	}

	if d.head == nil {
		d.tail = nil
	}
vyzo's avatar
vyzo committed
783 784
}

Raúl Kripalani's avatar
Raúl Kripalani committed
785 786 787
// getTopicStats returns existing topic stats for a given a given (peer, topic)
// tuple, or initialises a new topicStats object and inserts it in the
// peerStats, iff the topic is scored.
vyzo's avatar
vyzo committed
788 789 790 791 792 793 794 795 796 797
func (pstats *peerStats) getTopicStats(topic string, params *PeerScoreParams) (*topicStats, bool) {
	tstats, ok := pstats.topics[topic]
	if ok {
		return tstats, true
	}

	_, scoredTopic := params.Topics[topic]
	if !scoredTopic {
		return nil, false
	}
vyzo's avatar
vyzo committed
798

vyzo's avatar
vyzo committed
799 800 801 802
	tstats = &topicStats{}
	pstats.topics[topic] = tstats

	return tstats, true
vyzo's avatar
vyzo committed
803 804
}

Raúl Kripalani's avatar
Raúl Kripalani committed
805 806
// markInvalidMessageDelivery increments the "invalid message deliveries"
// counter for all scored topics the message is published in.
vyzo's avatar
vyzo committed
807 808 809 810 811 812 813 814 815 816 817
func (ps *peerScore) markInvalidMessageDelivery(p peer.ID, msg *Message) {
	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

	for _, topic := range msg.GetTopicIDs() {
		tstats, ok := pstats.getTopicStats(topic, ps.params)
		if !ok {
			continue
		}
vyzo's avatar
vyzo committed
818

vyzo's avatar
vyzo committed
819 820
		tstats.invalidMessageDeliveries += 1
	}
vyzo's avatar
vyzo committed
821 822
}

Raúl Kripalani's avatar
Raúl Kripalani committed
823 824 825
// markFirstMessageDelivery increments the "first message deliveries" counter
// for all scored topics the message is published in, as well as the "mesh
// message deliveries" counter, if the peer is in the mesh for the topic.
vyzo's avatar
vyzo committed
826 827 828 829 830 831 832 833 834 835 836 837
func (ps *peerScore) markFirstMessageDelivery(p peer.ID, msg *Message) {
	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

	for _, topic := range msg.GetTopicIDs() {
		tstats, ok := pstats.getTopicStats(topic, ps.params)
		if !ok {
			continue
		}

vyzo's avatar
vyzo committed
838
		cap := ps.params.Topics[topic].FirstMessageDeliveriesCap
vyzo's avatar
vyzo committed
839
		tstats.firstMessageDeliveries += 1
vyzo's avatar
vyzo committed
840 841
		if tstats.firstMessageDeliveries > cap {
			tstats.firstMessageDeliveries = cap
vyzo's avatar
vyzo committed
842 843 844 845 846
		}

		if !tstats.inMesh {
			continue
		}
vyzo's avatar
vyzo committed
847

vyzo's avatar
vyzo committed
848
		cap = ps.params.Topics[topic].MeshMessageDeliveriesCap
vyzo's avatar
vyzo committed
849
		tstats.meshMessageDeliveries += 1
vyzo's avatar
vyzo committed
850 851
		if tstats.meshMessageDeliveries > cap {
			tstats.meshMessageDeliveries = cap
vyzo's avatar
vyzo committed
852 853
		}
	}
vyzo's avatar
vyzo committed
854 855
}

Raúl Kripalani's avatar
Raúl Kripalani committed
856 857 858
// markDuplicateMessageDelivery increments the "mesh message deliveries" counter
// for messages we've seen before, as long the message was received within the
// P3 window.
859
func (ps *peerScore) markDuplicateMessageDelivery(p peer.ID, msg *Message, validated time.Time) {
860 861
	var now time.Time

vyzo's avatar
vyzo committed
862 863 864 865 866
	pstats, ok := ps.peerStats[p]
	if !ok {
		return
	}

867
	if !validated.IsZero() {
868 869 870
		now = time.Now()
	}

vyzo's avatar
vyzo committed
871 872 873 874 875 876 877 878 879
	for _, topic := range msg.GetTopicIDs() {
		tstats, ok := pstats.getTopicStats(topic, ps.params)
		if !ok {
			continue
		}

		if !tstats.inMesh {
			continue
		}
vyzo's avatar
vyzo committed
880

Raúl Kripalani's avatar
Raúl Kripalani committed
881 882
		tparams := ps.params.Topics[topic]

883 884 885
		// check against the mesh delivery window -- if the validated time is passed as 0, then
		// the message was received before we finished validation and thus falls within the mesh
		// delivery window.
Raúl Kripalani's avatar
Raúl Kripalani committed
886
		if !validated.IsZero() && now.After(validated.Add(tparams.MeshMessageDeliveriesWindow)) {
vyzo's avatar
vyzo committed
887 888 889
			continue
		}

Raúl Kripalani's avatar
Raúl Kripalani committed
890
		cap := tparams.MeshMessageDeliveriesCap
vyzo's avatar
vyzo committed
891
		tstats.meshMessageDeliveries += 1
vyzo's avatar
vyzo committed
892 893
		if tstats.meshMessageDeliveries > cap {
			tstats.meshMessageDeliveries = cap
vyzo's avatar
vyzo committed
894 895
		}
	}
vyzo's avatar
vyzo committed
896
}
vyzo's avatar
vyzo committed
897

Raúl Kripalani's avatar
Raúl Kripalani committed
898
// getIPs gets the current IPs for a peer.
vyzo's avatar
vyzo committed
899
func (ps *peerScore) getIPs(p peer.ID) []string {
vyzo's avatar
vyzo committed
900 901 902 903 904 905
	// in unit tests this can be nil
	if ps.host == nil {
		return nil
	}

	conns := ps.host.Network().ConnsToPeer(p)
906
	res := make([]string, 0, 1)
vyzo's avatar
vyzo committed
907 908
	for _, c := range conns {
		remote := c.RemoteMultiaddr()
909 910 911 912
		ip, err := manet.ToIP(remote)
		if err != nil {
			continue
		}
vyzo's avatar
vyzo committed
913

914 915
		// ignore those; loopback is used for unit testing
		if ip.IsLoopback() {
916 917 918
			continue
		}

919
		if len(ip.To4()) == 4 {
920 921
			// IPv4 address
			ip4 := ip.String()
vyzo's avatar
vyzo committed
922
			res = append(res, ip4)
923
		} else {
924 925
			// IPv6 address -- we add both the actual address and the /64 subnet
			ip6 := ip.String()
vyzo's avatar
vyzo committed
926
			res = append(res, ip6)
927

928 929
			ip6mask := ip.Mask(net.CIDRMask(64, 128)).String()
			res = append(res, ip6mask)
vyzo's avatar
vyzo committed
930 931 932 933
		}
	}

	return res
vyzo's avatar
vyzo committed
934 935
}

Raúl Kripalani's avatar
Raúl Kripalani committed
936 937
// setIPs adds tracking for the new IPs in the list, and removes tracking from
// the obsolete IPs.
vyzo's avatar
vyzo committed
938
func (ps *peerScore) setIPs(p peer.ID, newips, oldips []string) {
vyzo's avatar
vyzo committed
939 940 941 942 943 944 945 946 947 948 949 950 951 952 953 954 955 956 957 958 959 960 961 962 963 964
addNewIPs:
	// add the new IPs to the tracking
	for _, ip := range newips {
		// check if it is in the old ips list
		for _, xip := range oldips {
			if ip == xip {
				continue addNewIPs
			}
		}
		// no, it's a new one -- add it to the tracker
		peers, ok := ps.peerIPs[ip]
		if !ok {
			peers = make(map[peer.ID]struct{})
			ps.peerIPs[ip] = peers
		}
		peers[p] = struct{}{}
	}

removeOldIPs:
	// remove the obsolete old IPs from the tracking
	for _, ip := range oldips {
		// check if it is in the new ips list
		for _, xip := range newips {
			if ip == xip {
				continue removeOldIPs
			}
vyzo's avatar
vyzo committed
965 966 967 968 969 970 971 972 973
		}
		// no, it's obsolete -- remove it from the tracker
		peers, ok := ps.peerIPs[ip]
		if !ok {
			continue
		}
		delete(peers, p)
		if len(peers) == 0 {
			delete(ps.peerIPs, ip)
vyzo's avatar
vyzo committed
974 975 976 977
		}
	}
}

Raúl Kripalani's avatar
Raúl Kripalani committed
978
// removeIPs removes an IP list from the tracking list for a peer.
vyzo's avatar
vyzo committed
979 980 981 982 983 984 985 986 987 988 989 990 991
func (ps *peerScore) removeIPs(p peer.ID, ips []string) {
	for _, ip := range ips {
		peers, ok := ps.peerIPs[ip]
		if !ok {
			continue
		}

		delete(peers, p)
		if len(peers) == 0 {
			delete(ps.peerIPs, ip)
		}
	}
}