engine.go 9.93 KB
Newer Older
Jeromy's avatar
Jeromy committed
1
// package decision implements the decision engine for the bitswap service.
2
package decision
3 4

import (
Jeromy's avatar
Jeromy committed
5
	"context"
6
	"sync"
Jeromy's avatar
Jeromy committed
7
	"time"
8

Jeromy's avatar
Jeromy committed
9 10
	bsmsg "github.com/ipfs/go-bitswap/message"
	wl "github.com/ipfs/go-bitswap/wantlist"
11

Jeromy's avatar
Jeromy committed
12 13 14 15
	blocks "github.com/ipfs/go-block-format"
	bstore "github.com/ipfs/go-ipfs-blockstore"
	logging "github.com/ipfs/go-log"
	peer "github.com/libp2p/go-libp2p-peer"
16 17
)

18 19 20 21 22 23 24 25
// TODO consider taking responsibility for other types of requests. For
// example, there could be a |cancelQueue| for all of the cancellation
// messages that need to go out. There could also be a |wantlistQueue| for
// the local peer's wantlists. Alternatively, these could all be bundled
// into a single, intelligent global queue that efficiently
// batches/combines and takes all of these into consideration.
//
// Right now, messages go onto the network for four reasons:
26 27
// 1. an initial `sendwantlist` message to a provider of the first key in a
//    request
28 29 30 31 32 33 34 35 36 37 38 39
// 2. a periodic full sweep of `sendwantlist` messages to all providers
// 3. upon receipt of blocks, a `cancel` message to all peers
// 4. draining the priority queue of `blockrequests` from peers
//
// Presently, only `blockrequests` are handled by the decision engine.
// However, there is an opportunity to give it more responsibility! If the
// decision engine is given responsibility for all of the others, it can
// intelligently decide how to combine requests efficiently.
//
// Some examples of what would be possible:
//
// * when sending out the wantlists, include `cancel` requests
40 41
// * when handling `blockrequests`, include `sendwantlist` and `cancel` as
//   appropriate
42
// * when handling `cancel`, if we recently received a wanted block from a
43
//   peer, include a partial wantlist that contains a few other high priority
44 45 46 47 48 49
//   blocks
//
// In a sense, if we treat the decision engine as a black box, it could do
// whatever it sees fit to produce desired outcomes (get wanted keys
// quickly, maintain good relationships with peers, etc).

Jeromy's avatar
Jeromy committed
50
var log = logging.Logger("engine")
51

Brian Tiger Chow's avatar
Brian Tiger Chow committed
52
const (
53 54
	// outboxChanBuffer must be 0 to prevent stale messages from being sent
	outboxChanBuffer = 0
55 56
	// maxMessageSize is the maximum size of the batched payload
	maxMessageSize = 512 * 1024
Brian Tiger Chow's avatar
Brian Tiger Chow committed
57 58
)

59
// Envelope contains a message for a Peer
60
type Envelope struct {
61
	// Peer is the intended recipient
62
	Peer peer.ID
63

64 65
	// Message is the payload
	Message bsmsg.BitSwapMessage
Jeromy's avatar
Jeromy committed
66 67 68

	// A callback to notify the decision queue that the task is complete
	Sent func()
69 70
}

71
type Engine struct {
72 73 74
	// peerRequestQueue is a priority queue of requests received from peers.
	// Requests are popped from the queue, packaged up, and placed in the
	// outbox.
Jeromy's avatar
Jeromy committed
75
	peerRequestQueue *prq
76

77 78 79 80 81
	// FIXME it's a bit odd for the client and the worker to both share memory
	// (both modify the peerRequestQueue) and also to communicate over the
	// workSignal channel. consider sending requests over the channel and
	// allowing the worker to have exclusive access to the peerRequestQueue. In
	// that case, no lock would be required.
Jeromy's avatar
Jeromy committed
82
	workSignal chan struct{}
83

84 85
	// outbox contains outgoing messages to peers. This is owned by the
	// taskWorker goroutine
Brian Tiger Chow's avatar
Brian Tiger Chow committed
86
	outbox chan (<-chan *Envelope)
87 88 89

	bs bstore.Blockstore

90
	lock sync.Mutex // protects the fields immediatly below
91
	// ledgerMap lists Ledgers by their Partner key.
92
	ledgerMap map[peer.ID]*ledger
Jeromy's avatar
Jeromy committed
93 94

	ticker *time.Ticker
95 96
}

97 98
func NewEngine(ctx context.Context, bs bstore.Blockstore) *Engine {
	e := &Engine{
99
		ledgerMap:        make(map[peer.ID]*ledger),
Brian Tiger Chow's avatar
Brian Tiger Chow committed
100
		bs:               bs,
Brian Tiger Chow's avatar
Brian Tiger Chow committed
101
		peerRequestQueue: newPRQ(),
Brian Tiger Chow's avatar
Brian Tiger Chow committed
102
		outbox:           make(chan (<-chan *Envelope), outboxChanBuffer),
103
		workSignal:       make(chan struct{}, 1),
Jeromy's avatar
Jeromy committed
104
		ticker:           time.NewTicker(time.Millisecond * 100),
105
	}
106 107
	go e.taskWorker(ctx)
	return e
Jeromy's avatar
Jeromy committed
108 109
}

110
func (e *Engine) WantlistForPeer(p peer.ID) (out []*wl.Entry) {
111 112 113 114
	partner := e.findOrCreate(p)
	partner.lk.Lock()
	defer partner.lk.Unlock()
	return partner.wantList.SortedEntries()
115 116
}

117 118 119 120 121 122 123 124 125 126 127 128 129 130 131
func (e *Engine) LedgerForPeer(p peer.ID) *Receipt {
	ledger := e.findOrCreate(p)

	ledger.lk.Lock()
	defer ledger.lk.Unlock()

	return &Receipt{
		Peer:      ledger.Partner.String(),
		Value:     ledger.Accounting.Value(),
		Sent:      ledger.Accounting.BytesSent,
		Recv:      ledger.Accounting.BytesRecv,
		Exchanged: ledger.ExchangeCount(),
	}
}

132
func (e *Engine) taskWorker(ctx context.Context) {
133 134
	defer close(e.outbox) // because taskWorker uses the channel exclusively
	for {
Brian Tiger Chow's avatar
Brian Tiger Chow committed
135
		oneTimeUse := make(chan *Envelope, 1) // buffer to prevent blocking
136 137 138 139 140 141 142 143 144 145 146 147
		select {
		case <-ctx.Done():
			return
		case e.outbox <- oneTimeUse:
		}
		// receiver is ready for an outoing envelope. let's prepare one. first,
		// we must acquire a task from the PQ...
		envelope, err := e.nextEnvelope(ctx)
		if err != nil {
			close(oneTimeUse)
			return // ctx cancelled
		}
Brian Tiger Chow's avatar
Brian Tiger Chow committed
148
		oneTimeUse <- envelope // buffered. won't block
149 150 151 152 153 154 155
		close(oneTimeUse)
	}
}

// nextEnvelope runs in the taskWorker goroutine. Returns an error if the
// context is cancelled before the next Envelope can be created.
func (e *Engine) nextEnvelope(ctx context.Context) (*Envelope, error) {
Jeromy's avatar
Jeromy committed
156
	for {
Brian Tiger Chow's avatar
Brian Tiger Chow committed
157
		nextTask := e.peerRequestQueue.Pop()
158
		for nextTask == nil {
Jeromy's avatar
Jeromy committed
159
			select {
160
			case <-ctx.Done():
161
				return nil, ctx.Err()
162
			case <-e.workSignal:
163
				nextTask = e.peerRequestQueue.Pop()
Jeromy's avatar
Jeromy committed
164 165 166
			case <-e.ticker.C:
				e.peerRequestQueue.thawRound()
				nextTask = e.peerRequestQueue.Pop()
Jeromy's avatar
Jeromy committed
167 168
			}
		}
169 170

		// with a task in hand, we're ready to prepare the envelope...
171 172 173 174 175 176 177 178 179
		msg := bsmsg.New(true)
		for _, entry := range nextTask.Entries {
			block, err := e.bs.Get(entry.Cid)
			if err != nil {
				log.Errorf("tried to execute a task and errored fetching block: %s", err)
				continue
			}
			msg.AddBlock(block)
		}
180

181
		if msg.Empty() {
Jeromy's avatar
Jeromy committed
182 183
			// If we don't have the block, don't hold that against the peer
			// make sure to update that the task has been 'completed'
184
			nextTask.Done(nextTask.Entries)
Brian Tiger Chow's avatar
Brian Tiger Chow committed
185
			continue
186
		}
187

188
		return &Envelope{
189 190
			Peer:    nextTask.Target,
			Message: msg,
191
			Sent: func() {
192
				nextTask.Done(nextTask.Entries)
193 194 195 196 197 198 199
				select {
				case e.workSignal <- struct{}{}:
					// work completing may mean that our queue will provide new
					// work to be done.
				default:
				}
			},
200
		}, nil
Jeromy's avatar
Jeromy committed
201 202 203
	}
}

204
// Outbox returns a channel of one-time use Envelope channels.
Brian Tiger Chow's avatar
Brian Tiger Chow committed
205
func (e *Engine) Outbox() <-chan (<-chan *Envelope) {
206
	return e.outbox
207 208 209
}

// Returns a slice of Peers with whom the local node has active sessions
210
func (e *Engine) Peers() []peer.ID {
211 212
	e.lock.Lock()
	defer e.lock.Unlock()
213

214 215
	response := make([]peer.ID, 0, len(e.ledgerMap))

216
	for _, ledger := range e.ledgerMap {
217 218 219 220 221 222 223
		response = append(response, ledger.Partner)
	}
	return response
}

// MessageReceived performs book-keeping. Returns error if passed invalid
// arguments.
224
func (e *Engine) MessageReceived(p peer.ID, m bsmsg.BitSwapMessage) error {
225
	if m.Empty() {
226
		log.Debugf("received empty message from %s", p)
227 228
	}

229 230 231
	newWorkExists := false
	defer func() {
		if newWorkExists {
232
			e.signalNewWork()
233 234
		}
	}()
235

236
	l := e.findOrCreate(p)
Jeromy's avatar
Jeromy committed
237 238
	l.lk.Lock()
	defer l.lk.Unlock()
239 240 241
	if m.Full() {
		l.wantList = wl.New()
	}
242

243 244
	var msgSize int
	var activeEntries []*wl.Entry
245 246
	for _, entry := range m.Wantlist() {
		if entry.Cancel {
247 248 249
			log.Debugf("%s cancel %s", p, entry.Cid)
			l.CancelWant(entry.Cid)
			e.peerRequestQueue.Remove(entry.Cid, p)
250
		} else {
251 252
			log.Debugf("wants %s - %d", entry.Cid, entry.Priority)
			l.Wants(entry.Cid, entry.Priority)
253 254 255 256 257 258 259
			blockSize, err := e.bs.GetSize(entry.Cid)
			if err != nil {
				if err == bstore.ErrNotFound {
					continue
				}
				log.Error(err)
			} else {
260
				// we have the block
Brian Tiger Chow's avatar
Brian Tiger Chow committed
261
				newWorkExists = true
262
				if msgSize+blockSize > maxMessageSize {
263 264 265 266 267 268
					e.peerRequestQueue.Push(p, activeEntries...)
					activeEntries = []*wl.Entry{}
					msgSize = 0
				}
				activeEntries = append(activeEntries, entry.Entry)
				msgSize += blockSize
269
			}
270 271
		}
	}
272 273 274
	if len(activeEntries) > 0 {
		e.peerRequestQueue.Push(p, activeEntries...)
	}
275
	for _, block := range m.Blocks() {
Jeromy's avatar
Jeromy committed
276 277
		log.Debugf("got block %s %d bytes", block, len(block.RawData()))
		l.ReceivedBytes(len(block.RawData()))
278 279 280 281
	}
	return nil
}

282
func (e *Engine) addBlock(block blocks.Block) {
283 284 285
	work := false

	for _, l := range e.ledgerMap {
Jeromy's avatar
Jeromy committed
286
		l.lk.Lock()
287
		if entry, ok := l.WantListContains(block.Cid()); ok {
288
			e.peerRequestQueue.Push(l.Partner, entry)
289 290
			work = true
		}
Jeromy's avatar
Jeromy committed
291
		l.lk.Unlock()
292 293 294 295 296 297 298
	}

	if work {
		e.signalNewWork()
	}
}

299
func (e *Engine) AddBlock(block blocks.Block) {
300 301 302 303 304 305
	e.lock.Lock()
	defer e.lock.Unlock()

	e.addBlock(block)
}

306 307 308 309 310 311
// TODO add contents of m.WantList() to my local wantlist? NB: could introduce
// race conditions where I send a message, but MessageSent gets handled after
// MessageReceived. The information in the local wantlist could become
// inconsistent. Would need to ensure that Sends and acknowledgement of the
// send happen atomically

312
func (e *Engine) MessageSent(p peer.ID, m bsmsg.BitSwapMessage) error {
313
	l := e.findOrCreate(p)
314 315 316
	l.lk.Lock()
	defer l.lk.Unlock()

317
	for _, block := range m.Blocks() {
Jeromy's avatar
Jeromy committed
318
		l.SentBytes(len(block.RawData()))
319 320
		l.wantList.Remove(block.Cid())
		e.peerRequestQueue.Remove(block.Cid(), p)
321 322 323 324 325
	}

	return nil
}

326 327
func (e *Engine) PeerConnected(p peer.ID) {
	e.lock.Lock()
328
	defer e.lock.Unlock()
329 330 331 332 333 334
	l, ok := e.ledgerMap[p]
	if !ok {
		l = newLedger(p)
		e.ledgerMap[p] = l
	}
	l.lk.Lock()
335
	defer l.lk.Unlock()
336 337 338
	l.ref++
}

339
func (e *Engine) PeerDisconnected(p peer.ID) {
340 341 342 343 344 345 346
	e.lock.Lock()
	defer e.lock.Unlock()
	l, ok := e.ledgerMap[p]
	if !ok {
		return
	}
	l.lk.Lock()
347
	defer l.lk.Unlock()
348 349 350 351
	l.ref--
	if l.ref <= 0 {
		delete(e.ledgerMap, p)
	}
352 353
}

354
func (e *Engine) numBytesSentTo(p peer.ID) uint64 {
Brian Tiger Chow's avatar
Brian Tiger Chow committed
355
	// NB not threadsafe
356
	return e.findOrCreate(p).Accounting.BytesSent
357 358
}

359
func (e *Engine) numBytesReceivedFrom(p peer.ID) uint64 {
Brian Tiger Chow's avatar
Brian Tiger Chow committed
360
	// NB not threadsafe
361
	return e.findOrCreate(p).Accounting.BytesRecv
362 363 364
}

// ledger lazily instantiates a ledger
365
func (e *Engine) findOrCreate(p peer.ID) *ledger {
Jeromy's avatar
Jeromy committed
366
	e.lock.Lock()
367
	defer e.lock.Unlock()
368
	l, ok := e.ledgerMap[p]
369 370
	if !ok {
		l = newLedger(p)
371
		e.ledgerMap[p] = l
372 373 374
	}
	return l
}
375 376 377 378 379 380 381 382

func (e *Engine) signalNewWork() {
	// Signal task generation to restart (if stopped!)
	select {
	case e.workSignal <- struct{}{}:
	default:
	}
}