Jj/accounting updates (#820)

* tallies up data stored on each node in pointerdb

* adds comments for data type enums

* changes Open to BeginTx because Go convention

* removes online status check from identify active nodes

* changes identifyactivenodes to calculatestaticdata

* updates accounting dbx names
This commit is contained in:
Jennifer Li Johnson 2018-12-12 16:24:08 -05:00 committed by GitHub
parent 1a348fb356
commit 6642f97142
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
13 changed files with 775 additions and 669 deletions

12
pkg/accounting/common.go Normal file
View File

@ -0,0 +1,12 @@
// Copyright (C) 2018 Storj Labs, Inc.
// See LICENSE for copying information.
package accounting
// data_type enums for accounting_raw and accounting_rollup
const (
// AtRest is the data_type representing at-rest data calculated from pointerdb
AtRest = iota
// Bandwidth is the data_type representing bandwidth allocation.
Bandwith = iota
)

View File

@ -4,6 +4,8 @@
package accounting
import (
"context"
"github.com/zeebo/errs"
"storj.io/storj/internal/migrate"
@ -19,9 +21,15 @@ var (
LastBandwidthTally = dbx.Timestamps_Name("LastBandwidthTally")
)
// NewDb - constructor for DB
func NewDb(databaseURL string) (*dbx.DB, error) {
// Database contains access to accounting database
type Database struct {
db *dbx.DB
}
// NewDB - constructor for Database
func NewDB(databaseURL string) (*Database, error) {
driver, source, err := utils.SplitDBURL(databaseURL)
if err != nil {
return nil, Error.Wrap(err)
}
@ -32,8 +40,22 @@ func NewDb(databaseURL string) (*dbx.DB, error) {
}
err = migrate.Create("accounting", db)
if err != nil {
_ = db.Close()
return nil, Error.Wrap(err)
return nil, utils.CombineErrors(err, db.Close())
}
return db, nil
return &Database{db: db}, nil
}
// BeginTx is used to open db connection
func (db *Database) BeginTx(ctx context.Context) (*dbx.Tx, error) {
return db.db.Open(ctx)
}
// Close is used to close db connection
func (db *Database) Close() error {
return db.db.Close()
}
// FindLastBwTally returns the timestamp of the last bandwidth tally
func (db *Database) FindLastBwTally(ctx context.Context) (*dbx.Value_Row, error) {
return db.db.Find_Timestamps_Value_By_Name(ctx, LastBandwidthTally)
}

View File

@ -14,39 +14,51 @@ read scalar (
where timestamps.name = ?
)
model aggregate (
key node_id
model rollup (
key id
field id serial64
field node_id text
field start_time timestamp ( updatable )
field interval int64 ( updatable )
field start_time timestamp
field interval int64
field data_type int
field created_at timestamp ( autoinsert )
field updated_at timestamp ( autoinsert, autoupdate )
)
create aggregate ( )
update aggregate ( where aggregate.node_id = ? )
delete aggregate ( where aggregate.node_id = ? )
create rollup ( )
update rollup ( where rollup.id = ? )
delete rollup ( where rollup.id = ? )
read one (
select aggregate
where aggregate.node_id = ?
select rollup
where rollup.id = ?
)
read all (
select rollup
where rollup.node_id = ?
)
model granular (
key node_id
model raw (
key id
field id serial64
field node_id text
field start_time timestamp ( updatable )
field end_time timestamp ( updatable )
field data_total int64 ( updatable )
field interval_end_time timestamp
field data_total int64
field data_type int
field created_at timestamp ( autoinsert )
field updated_at timestamp ( autoinsert, autoupdate )
)
create granular ( )
update granular ( where granular.node_id = ? )
delete granular ( where granular.node_id = ? )
create raw ( )
update raw ( where raw.id = ? )
delete raw ( where raw.id = ? )
read one (
select granular
where granular.node_id = ?
select raw
where raw.id = ?
)
read all (
select raw
where raw.node_id = ?
)

File diff suppressed because it is too large Load Diff

View File

@ -1,21 +1,24 @@
-- AUTOGENERATED BY gopkg.in/spacemonkeygo/dbx.v1
-- DO NOT EDIT
CREATE TABLE aggregates (
CREATE TABLE raws (
id bigserial NOT NULL,
node_id text NOT NULL,
interval_end_time timestamp with time zone NOT NULL,
data_total bigint NOT NULL,
data_type integer NOT NULL,
created_at timestamp with time zone NOT NULL,
updated_at timestamp with time zone NOT NULL,
PRIMARY KEY ( id )
);
CREATE TABLE rollups (
id bigserial NOT NULL,
node_id text NOT NULL,
start_time timestamp with time zone NOT NULL,
interval bigint NOT NULL,
data_type integer NOT NULL,
created_at timestamp with time zone NOT NULL,
updated_at timestamp with time zone NOT NULL,
PRIMARY KEY ( node_id )
);
CREATE TABLE granulars (
node_id text NOT NULL,
start_time timestamp with time zone NOT NULL,
end_time timestamp with time zone NOT NULL,
data_total bigint NOT NULL,
created_at timestamp with time zone NOT NULL,
updated_at timestamp with time zone NOT NULL,
PRIMARY KEY ( node_id )
PRIMARY KEY ( id )
);
CREATE TABLE timestamps (
name text NOT NULL,

View File

@ -1,21 +1,24 @@
-- AUTOGENERATED BY gopkg.in/spacemonkeygo/dbx.v1
-- DO NOT EDIT
CREATE TABLE aggregates (
CREATE TABLE raws (
id INTEGER NOT NULL,
node_id TEXT NOT NULL,
interval_end_time TIMESTAMP NOT NULL,
data_total INTEGER NOT NULL,
data_type INTEGER NOT NULL,
created_at TIMESTAMP NOT NULL,
updated_at TIMESTAMP NOT NULL,
PRIMARY KEY ( id )
);
CREATE TABLE rollups (
id INTEGER NOT NULL,
node_id TEXT NOT NULL,
start_time TIMESTAMP NOT NULL,
interval INTEGER NOT NULL,
data_type INTEGER NOT NULL,
created_at TIMESTAMP NOT NULL,
updated_at TIMESTAMP NOT NULL,
PRIMARY KEY ( node_id )
);
CREATE TABLE granulars (
node_id TEXT NOT NULL,
start_time TIMESTAMP NOT NULL,
end_time TIMESTAMP NOT NULL,
data_total INTEGER NOT NULL,
created_at TIMESTAMP NOT NULL,
updated_at TIMESTAMP NOT NULL,
PRIMARY KEY ( node_id )
PRIMARY KEY ( id )
);
CREATE TABLE timestamps (
name TEXT NOT NULL,

View File

@ -20,7 +20,7 @@ type Config struct {
// Initialize a rollup struct
func (c Config) initialize(ctx context.Context) (Rollup, error) {
db, err := accounting.NewDb(c.DatabaseURL)
db, err := accounting.NewDB(c.DatabaseURL)
if err != nil {
return nil, Error.Wrap(err)
}

View File

@ -9,7 +9,7 @@ import (
"go.uber.org/zap"
dbx "storj.io/storj/pkg/accounting/dbx"
"storj.io/storj/pkg/accounting"
)
// Rollup is the service for totalling data on storage nodes for 1, 7, 30 day intervals
@ -20,10 +20,10 @@ type Rollup interface {
type rollup struct {
logger *zap.Logger
ticker *time.Ticker
db *dbx.DB
db *accounting.Database
}
func newRollup(logger *zap.Logger, db *dbx.DB, interval time.Duration) (*rollup, error) {
func newRollup(logger *zap.Logger, db *accounting.Database, interval time.Duration) (*rollup, error) {
return &rollup{
logger: logger,
ticker: time.NewTicker(interval),
@ -38,18 +38,18 @@ func (r *rollup) Run(ctx context.Context) (err error) {
for {
err = r.Query(ctx)
if err != nil {
zap.L().Error("Rollup Query failed", zap.Error(err))
r.logger.Error("Query failed", zap.Error(err))
}
select {
case <-r.ticker.C: // wait for the next interval to happen
case <-ctx.Done(): // or the rollup is canceled via context
_ = r.db.Close()
return ctx.Err()
}
}
}
func (r *rollup) Query(ctx context.Context) error {
//TODO
return nil
}

View File

@ -12,7 +12,6 @@ import (
"storj.io/storj/pkg/accounting"
"storj.io/storj/pkg/bwagreement"
"storj.io/storj/pkg/kademlia"
"storj.io/storj/pkg/overlay"
"storj.io/storj/pkg/pointerdb"
"storj.io/storj/pkg/provider"
@ -28,8 +27,7 @@ type Config struct {
func (c Config) initialize(ctx context.Context) (Tally, error) {
pointerdb := pointerdb.LoadFromContext(ctx)
overlay := overlay.LoadServerFromContext(ctx)
kademlia := kademlia.LoadFromContext(ctx)
db, err := accounting.NewDb(c.DatabaseURL)
db, err := accounting.NewDB(c.DatabaseURL)
if err != nil {
return nil, Error.Wrap(err)
}
@ -38,7 +36,7 @@ func (c Config) initialize(ctx context.Context) (Tally, error) {
if !ok {
return nil, errs.New("unable to get master db instance")
}
return newTally(zap.L(), db, masterDB.BandwidthAgreement(), pointerdb, overlay, kademlia, 0, c.Interval), nil
return newTally(zap.L(), db, masterDB.BandwidthAgreement(), pointerdb, overlay, 0, c.Interval), nil
}
// Run runs the tally with configured values

View File

@ -13,11 +13,8 @@ import (
"storj.io/storj/pkg/accounting"
dbx "storj.io/storj/pkg/accounting/dbx"
"storj.io/storj/pkg/bwagreement"
"storj.io/storj/pkg/kademlia"
"storj.io/storj/pkg/node"
"storj.io/storj/pkg/pb"
"storj.io/storj/pkg/pointerdb"
"storj.io/storj/pkg/provider"
"storj.io/storj/pkg/storj"
"storj.io/storj/storage"
)
@ -30,24 +27,22 @@ type Tally interface {
type tally struct {
pointerdb *pointerdb.Server
overlay pb.OverlayServer
kademlia *kademlia.Kademlia
limit int
logger *zap.Logger
ticker *time.Ticker
db *dbx.DB // accounting db
db *accounting.Database
bwAgreement bwagreement.DB // bwagreements database
}
func newTally(logger *zap.Logger, db *dbx.DB, bwAgreement bwagreement.DB, pointerdb *pointerdb.Server, overlay pb.OverlayServer, kademlia *kademlia.Kademlia, limit int, interval time.Duration) *tally {
func newTally(logger *zap.Logger, db *accounting.Database, bwAgreement bwagreement.DB, pointerdb *pointerdb.Server, overlay pb.OverlayServer, limit int, interval time.Duration) *tally {
return &tally{
pointerdb: pointerdb,
overlay: overlay,
kademlia: kademlia,
limit: limit,
logger: logger,
ticker: time.NewTicker(interval),
bwAgreement: bwAgreement,
db: db,
bwAgreement: bwAgreement,
}
}
@ -56,116 +51,69 @@ func (t *tally) Run(ctx context.Context) (err error) {
defer mon.Task()(&ctx)(&err)
for {
err = t.identifyActiveNodes(ctx)
err = t.calculateAtRestData(ctx)
if err != nil {
zap.L().Error("Tally failed", zap.Error(err))
t.logger.Error("calculateAtRestData failed", zap.Error(err))
}
err = t.Query(ctx)
if err != nil {
t.logger.Error("Query for bandwith failed", zap.Error(err))
}
select {
case <-t.ticker.C: // wait for the next interval to happen
case <-ctx.Done(): // or the tally is canceled via context
_ = t.db.Close()
return ctx.Err()
}
}
}
// identifyActiveNodes iterates through pointerdb and identifies nodes that have storage on them
func (t *tally) identifyActiveNodes(ctx context.Context) (err error) {
// calculateAtRestData iterates through the pieces on pointerdb and calculates
// the amount of at-rest data stored on each respective node
func (t *tally) calculateAtRestData(ctx context.Context) (err error) {
defer mon.Task()(&ctx)(&err)
rt, err := t.kademlia.GetRoutingTable(ctx)
if err != nil {
return Error.Wrap(err)
}
self := rt.Local()
identity, _ := provider.NewFullIdentity(ctx, 12, 4) //do i need anything in here?
client, err := node.NewNodeClient(identity, self, t.kademlia)
if err != nil {
return Error.Wrap(err)
}
var nodeData = make(map[storj.NodeID]int64)
err = t.pointerdb.Iterate(ctx, &pb.IterateRequest{Recurse: true},
func(it storage.Iterator) error {
var item storage.ListItem
lim := t.limit
if lim <= 0 || lim > storage.LookupLimit {
lim = storage.LookupLimit
}
for ; lim > 0 && it.Next(&item); lim-- {
for it.Next(&item) {
pointer := &pb.Pointer{}
err = proto.Unmarshal(item.Value, pointer)
if err != nil {
return Error.Wrap(err)
}
if pointer.Remote != nil {
pieces := pointer.Remote.RemotePieces
var nodeIDs storj.NodeIDList
for _, p := range pieces {
nodeIDs = append(nodeIDs, p.NodeId)
}
online, err := t.onlineNodes(ctx, nodeIDs)
if err != nil {
return Error.Wrap(err)
}
go t.tallyAtRestStorage(ctx, pointer, online, client)
remote := pointer.GetRemote()
if remote == nil {
continue
}
pieces := remote.GetRemotePieces()
if pieces == nil {
t.logger.Debug("no pieces on remote segment")
continue
}
segmentSize := pointer.GetSegmentSize()
redundancy := remote.GetRedundancy()
if redundancy == nil {
t.logger.Debug("no redundancy scheme present")
continue
}
minReq := redundancy.GetMinReq()
if minReq <= 0 {
t.logger.Debug("pointer minReq must be an int greater than 0")
continue
}
pieceSize := segmentSize / int64(minReq)
for _, piece := range pieces {
nodeData[piece.NodeId] += pieceSize
}
}
return nil
},
)
return err
return t.updateRawTable(ctx, nodeData)
}
func (t *tally) onlineNodes(ctx context.Context, nodeIDs storj.NodeIDList) (online []*pb.Node, err error) {
responses, err := t.overlay.BulkLookup(ctx, pb.NodeIDsToLookupRequests(nodeIDs))
if err != nil {
return []*pb.Node{}, err
}
nodes := pb.LookupResponsesToNodes(responses)
for _, n := range nodes {
if n != nil {
online = append(online, n)
}
}
return online, nil
}
func (t *tally) tallyAtRestStorage(ctx context.Context, pointer *pb.Pointer, nodes []*pb.Node, client node.Client) {
segmentSize := pointer.GetSegmentSize()
minReq := pointer.Remote.Redundancy.GetMinReq()
if minReq <= 0 {
zap.L().Error("minReq must be an int greater than 0")
return
}
pieceSize := segmentSize / int64(minReq)
for _, n := range nodes {
nodeAvail := true
var err error
ok := t.needToContact(n.Id)
if ok {
nodeAvail, err = client.Ping(ctx, *n)
if err != nil {
zap.L().Error("ping failed")
continue
}
}
if nodeAvail {
err := t.updateGranularTable(n.Id, pieceSize)
if err != nil {
zap.L().Error("update failed")
}
}
}
}
func (t *tally) needToContact(id storj.NodeID) bool {
//TODO
//check db if node was updated within the last time period
return true
}
func (t *tally) updateGranularTable(id storj.NodeID, pieceSize int64) error {
func (t *tally) updateRawTable(ctx context.Context, nodeData map[storj.NodeID]int64) error {
//TODO
return nil
}
@ -173,7 +121,7 @@ func (t *tally) updateGranularTable(id storj.NodeID, pieceSize int64) error {
// Query bandwidth allocation database, selecting all new contracts since the last collection run time.
// Grouping by storage node ID and adding total of bandwidth to granular data table.
func (t *tally) Query(ctx context.Context) error {
lastBwTally, err := t.db.Find_Timestamps_Value_By_Name(ctx, accounting.LastBandwidthTally)
lastBwTally, err := t.db.FindLastBwTally(ctx)
if err != nil {
return err
}
@ -212,7 +160,7 @@ func (t *tally) Query(ctx context.Context) error {
//insert all records in a transaction so if we fail, we don't have partial info stored
//todo: replace with a WithTx() method per DBX docs?
tx, err := t.db.Open(ctx)
tx, err := t.db.BeginTx(ctx)
if err != nil {
t.logger.DPanic("Failed to create DB txn in tally query")
return err
@ -228,11 +176,11 @@ func (t *tally) Query(ctx context.Context) error {
//todo: switch to bulk update SQL?
for k, v := range bwTotals {
nID := dbx.Granular_NodeId(k)
start := dbx.Granular_StartTime(lastBwTally.Value)
end := dbx.Granular_EndTime(latestBwa)
total := dbx.Granular_DataTotal(v)
_, err = tx.Create_Granular(ctx, nID, start, end, total)
nID := dbx.Raw_NodeId(k)
end := dbx.Raw_IntervalEndTime(latestBwa)
total := dbx.Raw_DataTotal(v)
dataType := dbx.Raw_DataType(accounting.Bandwith)
_, err = tx.Create_Raw(ctx, nID, end, total, dataType)
if err != nil {
t.logger.DPanic("Create granular SQL failed in tally query")
return err //todo: retry strategy?

View File

@ -5,97 +5,33 @@ package tally
import (
"crypto/ecdsa"
"math/rand"
"strconv"
"testing"
"time"
"github.com/stretchr/testify/assert"
"go.uber.org/zap"
"storj.io/storj/internal/identity"
testidentity "storj.io/storj/internal/identity"
"storj.io/storj/internal/testcontext"
"storj.io/storj/internal/teststorj"
"storj.io/storj/pkg/accounting"
"storj.io/storj/pkg/bwagreement"
"storj.io/storj/pkg/bwagreement/test"
"storj.io/storj/pkg/kademlia"
"storj.io/storj/pkg/overlay"
"storj.io/storj/pkg/overlay/mocks"
"storj.io/storj/pkg/pb"
"storj.io/storj/pkg/pointerdb"
"storj.io/storj/pkg/storj"
"storj.io/storj/satellite/satellitedb"
"storj.io/storj/storage/teststore"
)
func TestIdentifyActiveNodes(t *testing.T) {
}
func TestOnlineNodes(t *testing.T) {
ctx := testcontext.New(t)
defer ctx.Cleanup()
logger := zap.NewNop()
pointerdb := pointerdb.NewServer(teststore.New(), &overlay.Cache{}, logger, pointerdb.Config{}, nil)
const N = 50
nodes := []*pb.Node{}
nodeIDs := storj.NodeIDList{}
expectedOnline := []*pb.Node{}
for i := 0; i < N; i++ {
nodeID := teststorj.NodeIDFromString(strconv.Itoa(i))
n := &pb.Node{Id: nodeID, Type: pb.NodeType_STORAGE, Address: &pb.NodeAddress{Address: ""}}
nodes = append(nodes, n)
if i%(rand.Intn(5)+2) == 0 {
id := teststorj.NodeIDFromString("id" + nodeID.String())
nodeIDs = append(nodeIDs, id)
} else {
nodeIDs = append(nodeIDs, nodeID)
expectedOnline = append(expectedOnline, n)
}
}
overlayServer := mocks.NewOverlay(nodes)
kad := &kademlia.Kademlia{}
limit := 0
interval := time.Second
accountingDb, err := accounting.NewDb("sqlite3://file::memory:?mode=memory&cache=shared")
assert.NoError(t, err)
defer ctx.Check(accountingDb.Close)
masterDB, err := satellitedb.NewInMemory()
assert.NoError(t, err)
defer ctx.Check(masterDB.Close)
tally := newTally(logger, accountingDb, masterDB.BandwidthAgreement(), pointerdb, overlayServer, kad, limit, interval)
online, err := tally.onlineNodes(ctx, nodeIDs)
assert.NoError(t, err)
assert.Equal(t, expectedOnline, online)
}
func TestTallyAtRestStorage(t *testing.T) {
}
func TestNeedToContact(t *testing.T) {
}
func TestUpdateGranularTable(t *testing.T) {
}
func TestQueryNoAgreements(t *testing.T) {
ctx := testcontext.New(t)
defer ctx.Cleanup()
//get stuff we need
pointerdb := pointerdb.NewServer(teststore.New(), &overlay.Cache{}, zap.NewNop(), pointerdb.Config{}, nil)
overlayServer := mocks.NewOverlay([]*pb.Node{})
kad := &kademlia.Kademlia{}
accountingDb, err := accounting.NewDb("sqlite3://file::memory:?mode=memory&cache=shared")
accountingDb, err := accounting.NewDB("sqlite3://file::memory:?mode=memory&cache=shared")
assert.NoError(t, err)
defer ctx.Check(accountingDb.Close)
@ -103,9 +39,8 @@ func TestQueryNoAgreements(t *testing.T) {
assert.NoError(t, err)
defer ctx.Check(masterDB.Close)
tally := newTally(zap.NewNop(), accountingDb, masterDB.BandwidthAgreement(), pointerdb, overlayServer, kad, 0, time.Second)
tally := newTally(zap.NewNop(), accountingDb, masterDB.BandwidthAgreement(), pointerdb, overlayServer, 0, time.Second)
//check the db
err = tally.Query(ctx)
assert.NoError(t, err)
}
@ -114,22 +49,21 @@ func TestQueryWithBw(t *testing.T) {
ctx := testcontext.New(t)
defer ctx.Cleanup()
//get stuff we need
pointerdb := pointerdb.NewServer(teststore.New(), &overlay.Cache{}, zap.NewNop(), pointerdb.Config{}, nil)
overlayServer := mocks.NewOverlay([]*pb.Node{})
kad := &kademlia.Kademlia{}
accountingDb, err := accounting.NewDb("sqlite3://file::memory:?mode=memory&cache=shared")
accountingDb, err := accounting.NewDB("sqlite3://file::memory:?mode=memory&cache=shared")
assert.NoError(t, err)
defer ctx.Check(accountingDb.Close)
masterDB, err := satellitedb.NewInMemory()
assert.NoError(t, err)
defer ctx.Check(masterDB.Close)
err = masterDB.CreateTables()
assert.NoError(t, err)
defer ctx.Check(masterDB.Close)
bwDb := masterDB.BandwidthAgreement()
tally := newTally(zap.NewNop(), accountingDb, bwDb, pointerdb, overlayServer, kad, 0, time.Second)
tally := newTally(zap.NewNop(), accountingDb, bwDb, pointerdb, overlayServer, 0, time.Second)
//get a private key
fiC, err := testidentity.NewTestIdentity()

View File

@ -58,7 +58,7 @@ func (c *checker) Run(ctx context.Context) (err error) {
for {
err = c.identifyInjuredSegments(ctx)
if err != nil {
zap.L().Error("Checker failed", zap.Error(err))
c.logger.Error("Checker failed", zap.Error(err))
}
select {

View File

@ -5,7 +5,7 @@ package pb
import "storj.io/storj/pkg/storj"
// NodeIDsToLookupRequests ...
// NodeIDsToLookupRequests converts NodeIDs to LookupRequests
func NodeIDsToLookupRequests(nodeIDs storj.NodeIDList) *LookupRequests {
var rq []*LookupRequest
for _, v := range nodeIDs {
@ -15,11 +15,11 @@ func NodeIDsToLookupRequests(nodeIDs storj.NodeIDList) *LookupRequests {
return &LookupRequests{LookupRequest: rq}
}
// LookupResponsesToNodes ...
// LookupResponsesToNodes converts LookupResponses to Nodes
func LookupResponsesToNodes(responses *LookupResponses) []*Node {
var nodes []*Node
for _, v := range responses.LookupResponse {
n := v.Node
n := v.GetNode()
nodes = append(nodes, n)
}
return nodes