Git Product home page Git Product logo

go-ycsb's Introduction

go-ycsb

go-ycsb is a Go port of YCSB. It fully supports all YCSB generators and the Core workload so we can do the basic CRUD benchmarks with Go.

Why another Go YCSB?

  • We want to build a standard benchmark tool in Go.
  • We are not familiar with Java.

Getting Started

Download

https://github.com/pingcap/go-ycsb/releases/latest

Linux

wget -c https://github.com/pingcap/go-ycsb/releases/latest/download/go-ycsb-linux-amd64.tar.gz -O - | tar -xz

# give it a try
./go-ycsb --help

OSX

wget -c https://github.com/pingcap/go-ycsb/releases/latest/download/go-ycsb-darwin-amd64.tar.gz -O - | tar -xz

# give it a try
./go-ycsb --help

Building from source

git clone https://github.com/pingcap/go-ycsb.git
cd go-ycsb
make

# give it a try
./bin/go-ycsb  --help

Notice:

  • Minimum supported go version is 1.16.
  • To use FoundationDB, you must install client library at first, now the supported version is 6.2.11.
  • To use RocksDB, you must follow INSTALL to install RocksDB at first.

Usage

Mostly, we can start from the official document Running-a-Workload.

Shell

./bin/go-ycsb shell basic
» help
YCSB shell command

Usage:
  shell [command]

Available Commands:
  delete      Delete a record
  help        Help about any command
  insert      Insert a record
  read        Read a record
  scan        Scan starting at key
  table       Get or [set] the name of the table
  update      Update a record

Load

./bin/go-ycsb load basic -P workloads/workloada

Run

./bin/go-ycsb run basic -P workloads/workloada

Supported Database

  • MySQL / TiDB
  • TiKV
  • FoundationDB
  • Aerospike
  • Badger
  • Cassandra / ScyllaDB
  • Pegasus
  • PostgreSQL / CockroachDB / AlloyDB / Yugabyte
  • RocksDB
  • Spanner
  • Sqlite
  • MongoDB
  • Redis and Redis Cluster
  • BoltDB
  • etcd
  • DynamoDB

Output configuration

field default value description
measurementtype "histogram" The mechanism for recording measurements, one of histogram, raw or csv
measurement.output_file "" File to write output to, default writes to stdout

Database Configuration

You can pass the database configurations through -p field=value in the command line directly.

Common configurations:

field default value description
dropdata false Whether to remove all data before test
verbose false Output the execution query
debug.pprof ":6060" Go debug profile address

MySQL & TiDB

field default value description
mysql.host "127.0.0.1" MySQL Host
mysql.port 3306 MySQL Port
mysql.user "root" MySQL User
mysql.password MySQL Password
mysql.db "test" MySQL Database
tidb.cluster_index true Whether to use cluster index, for TiDB only
tidb.instances "" Comma-seperated address list of tidb instances (eg: tidb-0:4000,tidb-1:4000)

TiKV

field default value description
tikv.pd "127.0.0.1:2379" PD endpoints, seperated by comma
tikv.type "raw" TiKV mode, "raw", "txn", or "coprocessor"
tikv.conncount 128 gRPC connection count
tikv.batchsize 128 Request batch size
tikv.async_commit true Enalbe async commit or not
tikv.one_pc true Enable one phase or not
tikv.apiversion "V1" api-version of tikv server, "V1" or "V2"

FoundationDB

field default value description
fdb.cluster "" The cluster file used for FoundationDB, if not set, will use the default
fdb.dbname "DB" The cluster database name
fdb.apiversion 510 API version, now only 5.1 is supported

PostgreSQL & CockroachDB & AlloyDB & Yugabyte

field default value description
pg.host "127.0.0.1" PostgreSQL Host
pg.port 5432 PostgreSQL Port
pg.user "root" PostgreSQL User
pg.password PostgreSQL Password
pg.db "test" PostgreSQL Database
pg.sslmode "disable PostgreSQL ssl mode

Aerospike

field default value description
aerospike.host "localhost" The port of the Aerospike service
aerospike.port 3000 The port of the Aerospike service
aerospike.ns "test" The namespace to use

Badger

field default value description
badger.dir "/tmp/badger" The directory to save data
badger.valuedir "/tmp/badger" The directory to save value, if not set, use badger.dir
badger.sync_writes false Sync all writes to disk
badger.num_versions_to_keep 1 How many versions to keep per key
badger.max_table_size 64MB Each table (or file) is at most this size
badger.level_size_multiplier 10 Equals SizeOf(Li+1)/SizeOf(Li)
badger.max_levels 7 Maximum number of levels of compaction
badger.value_threshold 32 If value size >= this threshold, only store value offsets in tree
badger.num_memtables 5 Maximum number of tables to keep in memory, before stalling
badger.num_level0_tables 5 Maximum number of Level 0 tables before we start compacting
badger.num_level0_tables_stall 10 If we hit this number of Level 0 tables, we will stall until L0 is compacted away
badger.level_one_size 256MB Maximum total size for L1
badger.value_log_file_size 1GB Size of single value log file
badger.value_log_max_entries 1000000 Max number of entries a value log file can hold (approximately). A value log file would be determined by the smaller of its file size and max entries
badger.num_compactors 3 Number of compaction workers to run concurrently
badger.do_not_compact false Stops LSM tree from compactions
badger.table_loading_mode options.LoadToRAM How should LSM tree be accessed
badger.value_log_loading_mode options.MemoryMap How should value log be accessed

RocksDB

field default value description
rocksdb.dir "/tmp/rocksdb" The directory to save data
rocksdb.allow_concurrent_memtable_writes true Sets whether to allow concurrent memtable writes
rocksdb.allow_mmap_reads false Enable/Disable mmap reads for reading sst tables
rocksdb.allow_mmap_writes false Enable/Disable mmap writes for writing sst tables
rocksdb.arena_block_size 0(write_buffer_size / 8) Sets the size of one block in arena memory allocation
rocksdb.db_write_buffer_size 0(disable) Sets the amount of data to build up in memtables across all column families before writing to disk
rocksdb.hard_pending_compaction_bytes_limit 256GB Sets the bytes threshold at which all writes are stopped if estimated bytes needed to be compaction exceed this threshold
rocksdb.level0_file_num_compaction_trigger 4 Sets the number of files to trigger level-0 compaction
rocksdb.level0_slowdown_writes_trigger 20 Sets the soft limit on number of level-0 files
rocksdb.level0_stop_writes_trigger 36 Sets the maximum number of level-0 files. We stop writes at this point
rocksdb.max_bytes_for_level_base 256MB Sets the maximum total data size for base level
rocksdb.max_bytes_for_level_multiplier 10 Sets the max Bytes for level multiplier
rocksdb.max_total_wal_size 0([sum of all write_buffer_size * max_write_buffer_number] * 4) Sets the maximum total wal size in bytes. Once write-ahead logs exceed this size, we will start forcing the flush of column families whose memtables are backed by the oldest live WAL file (i.e. the ones that are causing all the space amplification)
rocksdb.memtable_huge_page_size 0 Sets the page size for huge page for arena used by the memtable
rocksdb.num_levels 7 Sets the number of levels for this database
rocksdb.use_direct_reads false Enable/Disable direct I/O mode (O_DIRECT) for reads
rocksdb.use_fsync false Enable/Disable fsync
rocksdb.write_buffer_size 64MB Sets the amount of data to build up in memory (backed by an unsorted log on disk) before converting to a sorted on-disk file
rocksdb.max_write_buffer_number 2 Sets the maximum number of write buffers that are built up in memory
rocksdb.max_background_jobs 2 Sets maximum number of concurrent background jobs (compactions and flushes)
rocksdb.block_size 4KB Sets the approximate size of user data packed per block. Note that the block size specified here corresponds opts uncompressed data. The actual size of the unit read from disk may be smaller if compression is enabled
rocksdb.block_size_deviation 10 Sets the block size deviation. This is used opts close a block before it reaches the configured 'block_size'. If the percentage of free space in the current block is less than this specified number and adding a new record opts the block will exceed the configured block size, then this block will be closed and the new record will be written opts the next block
rocksdb.cache_index_and_filter_blocks false Indicating if we'd put index/filter blocks to the block cache. If not specified, each "table reader" object will pre-load index/filter block during table initialization
rocksdb.no_block_cache false Specify whether block cache should be used or not
rocksdb.pin_l0_filter_and_index_blocks_in_cache false Sets cache_index_and_filter_blocks. If is true and the below is true (hash_index_allow_collision), then filter and index blocks are stored in the cache, but a reference is held in the "table reader" object so the blocks are pinned and only evicted from cache when the table reader is freed
rocksdb.whole_key_filtering true Specify if whole keys in the filter (not just prefixes) should be placed. This must generally be true for gets opts be efficient
rocksdb.block_restart_interval 16 Sets the number of keys between restart points for delta encoding of keys. This parameter can be changed dynamically
rocksdb.filter_policy nil Sets the filter policy opts reduce disk reads. Many applications will benefit from passing the result of NewBloomFilterPolicy() here
rocksdb.index_type kBinarySearch Sets the index type used for this table. kBinarySearch: A space efficient index block that is optimized for binary-search-based index. kHashSearch: The hash index, if enabled, will do the hash lookup when Options.prefix_extractor is provided. kTwoLevelIndexSearch: A two-level index implementation. Both levels are binary search indexes
rocksdb.block_align false Enable/Disable align data blocks on lesser of page size and block size

Spanner

field default value description
spanner.db "" Spanner Database
spanner.credentials "~/.spanner/credentials.json" Google application credentials for Spanner

Sqlite

field default value description
sqlite.db "/tmp/sqlite.db" Database path
sqlite.mode "rwc" Open Mode: ro, rc, rwc, memory
sqlite.journalmode "DELETE" Journal mode: DELETE, TRUNCSTE, PERSIST, MEMORY, WAL, OFF
sqlite.cache "Shared" Cache: shared, private

Cassandra

field default value description
cassandra.cluster "127.0.0.1:9042" Cassandra cluster
cassandra.keyspace "test" Keyspace
cassandra.connections 2 Number of connections per host
cassandra.username cassandra Username
cassandra.password cassandra Password

MongoDB

field default value description
mongodb.url "mongodb://127.0.0.1:27017" MongoDB URI
mongodb.tls_skip_verify false Enable/disable server ca certificate verification
mongodb.tls_ca_file "" Path to mongodb server ca certificate file
mongodb.namespace "ycsb.ycsb" Namespace to use
mongodb.authdb "admin" Authentication database
mongodb.username N/A Username for authentication
mongodb.password N/A Password for authentication

Redis

field default value description
redis.datatype hash "hash", "string" or "json" ("json" requires RedisJSON available)
redis.mode single "single" or "cluster"
redis.network tcp "tcp" or "unix"
redis.addr Redis server address(es) in "host:port" form, can be semi-colon ; separated in cluster mode
redis.username Redis server username
redis.password Redis server password
redis.db 0 Redis server target db
redis.max_redirects 0 The maximum number of retries before giving up (only for cluster mode)
redis.read_only false Enables read-only commands on slave nodes (only for cluster mode)
redis.route_by_latency false Allows routing read-only commands to the closest master or slave node (only for cluster mode)
redis.route_randomly false Allows routing read-only commands to the random master or slave node (only for cluster mode)
redis.max_retries Max retries before giving up connection
redis.min_retry_backoff 8ms Minimum backoff between each retry
redis.max_retry_backoff 512ms Maximum backoff between each retry
redis.dial_timeout 5s Dial timeout for establishing new connection
redis.read_timeout 3s Timeout for socket reads
redis.write_timeout 3s Timeout for socket writes
redis.pool_size 10 Maximum number of socket connections
redis.min_idle_conns 0 Minimum number of idle connections
redis.max_idle_conns 0 Maximum number of idle connections. If <= 0, connections are not closed due to a connection's idle time.
redis.max_conn_age 0 Connection age at which client closes the connection
redis.pool_timeout 4s Amount of time client waits for connections are busy before returning an error
redis.idle_timeout 5m Amount of time after which client closes idle connections. Should be less than server timeout
redis.idle_check_frequency 1m Frequency of idle checks made by idle connections reaper. Deprecated in favour of redis.max_idle_conns
redis.tls_ca Path to CA file
redis.tls_cert Path to cert file
redis.tls_key Path to key file
redis.tls_insecure_skip_verify false Controls whether a client verifies the server's certificate chain and host name

BoltDB

field default value description
bolt.path "/tmp/boltdb" The database file path. If the file does not exists then it will be created automatically
bolt.timeout 0 The amount of time to wait to obtain a file lock. When set to zero it will wait indefinitely. This option is only available on Darwin and Linux
bolt.no_grow_sync false Sets DB.NoGrowSync flag before memory mapping the file
bolt.read_only false Open the database in read-only mode
bolt.mmap_flags 0 Set the DB.MmapFlags flag before memory mapping the file
bolt.initial_mmap_size 0 The initial mmap size of the database in bytes. If <= 0, the initial map size is 0. If the size is smaller than the previous database, it takes no effect

etcd

field default value description
etcd.endpoints "localhost:2379" The etcd endpoint(s), multiple endpoints can be passed separated by comma.
etcd.dial_timeout "2s" The dial timeout duration passed into the client config.
etcd.cert_file "" When using secure etcd, this should point to the crt file.
etcd.key_file "" When using secure etcd, this should point to the pem file.
etcd.cacert_file "" When using secure etcd, this should point to the ca file.
etcd.serializable_reads false Whether to use serializable reads.

DynamoDB

field default value description
dynamodb.tablename "ycsb" The database tablename
dynamodb.primarykey "_key" The table primary key fieldname
dynamodb.rc.units 10 Read request units throughput
dynamodb.wc.units 10 Write request units throughput
dynamodb.ensure.clean.table true On load mode ensure that the table is clean at the begining. In case of true and if the table previously exists it will be deleted and recreated
dynamodb.endpoint "" Used endpoint for connection. If empty will use the default loaded configs
dynamodb.region "" Used region for connection ( should match endpoint ). If empty will use the default loaded configs
dynamodb.consistent.reads false Reads on DynamoDB provide an eventually consistent read by default. If your benchmark/use-case requires a strongly consistent read, set this option to true
dynamodb.delete.after.run.stage false Detele the database table after the run stage

TODO

  • Support more measurement, like HdrHistogram
  • Add tests for generators

go-ycsb's People

Contributors

94rain avatar asmyasnikov avatar bb7133 avatar caosiyang avatar connor1996 avatar cwen0 avatar dbsid avatar dependabot[bot] avatar disksing avatar divanodestiny avatar dorianzheng avatar filipecosta90 avatar haojinming avatar hawkingrei avatar iosmanthus avatar jeffa5 avatar liukun4515 avatar losfair avatar ls4154 avatar mahjonp avatar ngaut avatar nsobjects avatar peng1999 avatar pierrez avatar siddontang avatar sillydong avatar tjungblu avatar yangkeao avatar zhouqiang-cl avatar zyguan avatar

Stargazers

 avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar

Watchers

 avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar  avatar

go-ycsb's Issues

tikv scan报出decode error

v, err := db.r.Decode(row, fields)

image
建议日志或者打印屏幕给出更多的详细信息,目前只有operation err: invalid encoded key flag 110,
加了一些打印信息,才定位到是scan这里报错

TiKV txn benchmark does not retry retriable error

Ran go-ycsb against tikv txn API, and there's significant portion of updates get update error. In particular, these are the command and param I'm using:

./bin/go-ycsb load tikv -P workloads/workloada -p tikv.pd="172.16.30.40:2379" -p tikv.type="txn" -p recordcount=10000000
./bin/go-ycsb run tikv -P workloads/workloada -p tikv.pd="172.16.30.40:2379" -p tikv.type="txn" -p recordcount=10000000 -p readproportion=0 -p updateproportion=1.0 -p operationcount=10000000 -p threadcount=1000

In general this is preload an empty tikv with 10M rows, and then try to do 10M updates from 1k threads. During the run phase, ycsb would report 15% update error:

UPDATE_ERROR - Takes(s): 242.8, Count: 1537013, OPS: 6329.8, Avg(us): 46655, Min(us): 1059, Max(us): 321298, 95th(us): 104000, 99th(us): 113000
UPDATE - Takes(s): 242.8, Count: 8462987, OPS: 34852.4, Avg(us): 19998, Min(us): 1381, Max(us): 499397, 95th(us): 36000, 99th(us): 47000

In fact those errors are all retriable errors of "get timestamp too slow". When I patch with yiwu-arbug@344243b the error rate drops significantly (but also QPS):

UPDATE - Takes(s): 739.7, Count: 9994297, OPS: 13511.0, Avg(us): 65915, Min(us): 1148, Max(us): 9848521, 95th(us): 15000, 99th(us): 0
UPDATE_ERROR - Takes(s): 731.1, Count: 5703, OPS: 7.8, Avg(us): 8625979, Min(us): 7012194, Max(us): 10176360, 95th(us): 0, 99th(us): 0

The remaining errors are still all of the same retriable "get timestamp too slow" error. I'm feeling that go-ycsb should retry these errors to simulate real scenario, where a normal client would retry before giving up.

start error: failed to get cluster id

Hi. I started 3 nodes tikv cluster. The commandline are:
pd:

bin/pd-server --name=pd1 --data-dir=pd1 --client-urls=http://10.16.148.72:2379 --peer-urls=http://10.16.148.72:2380 --initial-cluster=pd1=http://10.16.148.72:2380 --log-file=pd1.log -metrics-addr 10.110.27.6:9091

tikv:

bin/tikv-server --pd-endpoints=10.16.148.72:2379 --addr=10.16.148.73:20160 --data-dir=tikv1 --log-file=tikv1.log --metrics-addr 10.110.27.6:9091

I started go-ycsb against tikv 3.0.0-beta with following command:

./bin/go-ycsb load tikv -P workloads/workloada -p tikv.pd=10.16.148.72:2379 -p tikv.type=raw

and bin/pd-ctl store -d -u http://10.16.148.72:2379 shows all tikv servers are UP

But go-ycsb still showed such errors and I still couldn't figure out why.

image

shell command panic when field0 contain space

go-ycsb panics when I try the insert command with a field string that contains a space:

» insert hello.txt field0="hello world!"

The go-ycsb is build from master branch with code commit hash 00ca633.

The panic stack is following:

panic: runtime error: index out of range [1] with length 1

goroutine 1 [running]:
main.runShellInsertCommand(0xc0002fac80, 0xc0002ba0f0, 0x3, 0x3)
	/home/chenqinshui/Developer/tos-ycsb/cmd/go-ycsb/shell.go:173 +0x40f
github.com/spf13/cobra.(*Command).execute(0xc0002fac80, 0xc0002ba030, 0x3, 0x3, 0xc0002fac80, 0xc0002ba030)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x29d
github.com/spf13/cobra.(*Command).ExecuteC(0xc0002fa500, 0xc0003ffbd8, 0x6, 0x6)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x2ea
github.com/spf13/cobra.(*Command).Execute(...)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:800
main.runShellCommand(0xc0002788c0, 0x4, 0x4)
	/home/chenqinshui/Developer/tos-ycsb/cmd/go-ycsb/shell.go:112 +0x48b
main.shellLoop()
	/home/chenqinshui/Developer/tos-ycsb/cmd/go-ycsb/shell.go:245 +0x278
main.runShellCommandFunc(0xc00023e780, 0xc000203500, 0x1, 0x3)
	/home/chenqinshui/Developer/tos-ycsb/cmd/go-ycsb/shell.go:52 +0x113
github.com/spf13/cobra.(*Command).execute(0xc00023e780, 0xc0002034a0, 0x3, 0x3, 0xc00023e780, 0xc0002034a0)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x29d
github.com/spf13/cobra.(*Command).ExecuteC(0xc00023e500, 0xc00063ff20, 0x3, 0x3)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x2ea
github.com/spf13/cobra.(*Command).Execute(...)
	/home/chenqinshui/bin/go/packages/pkg/mod/github.com/spf13/[email protected]/command.go:800
main.main()
	/home/chenqinshui/Developer/tos-ycsb/cmd/go-ycsb/main.go:174 +0x276

threadcount vs. go-routines vs. threads vs. go-scheduler

YCSB allows to define the parameter "threadcount" which, in this (go) code, is used to create go-routines (client.go).

Go-routines don't translate to real threads and are managed by the go-scheduler, as far as I understand.
This poses some questions regrading reproducability and comparability of benchmark results:

If I define "-p threadcount=1000" do I really get 1000 concurrent entities that work in parallel or do I get whatever the go-scheduler makes of it?
Does "-p threadcount=1000" cause 1000 connections to the DB or potentially less?
How does the parameter 'threadcount' in go-ycsb compare to the same parameter in the original java-ycsb with regard to threads being created?
Could the go-scheduler actually interfer in any other way and limit the reproducability of the results?

nil pointer panic during mysql reconnect

[mysql] 2020/09/17 18:20:35 connection.go:157: invalid connection
panic: runtime error: invalid memory address or nil pointer dereference
[signal SIGSEGV: segmentation violation code=0x1 addr=0x28 pc=0x8506e6]

goroutine 144 [running]:
database/sql.(*Conn).grabConn(0x0, 0x1923200, 0xc0359824b0, 0x0, 0xe000, 0xc0031c9908, 0x50f80e)
        /usr/local/go/src/database/sql/sql.go:1854 +0x26
database/sql.(*Conn).PrepareContext(0x0, 0x1923200, 0xc0359824b0, 0xc03e1b9800, 0x35c1, 0xc0031c9900, 0x418f30, 0x7f9a7c893748)
        /usr/local/go/src/database/sql/sql.go:1910 +0x46
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).getAndCacheStmt(0xc00028d2f0, 0x1923200, 0xc0359824b0, 0xc03e1b9800, 0x35c1, 0x7f9a7c893748, 0x0, 0x149a7e0)
        /home/alei/go-ycsb/db/mysql/db.go:180 +0xfb
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).execQuery(0xc00028d2f0, 0x1923200, 0xc0359824b0, 0xc03e1b9800, 0x35c1, 0xc03eae0000, 0x1130, 0x9d08, 0xc0031c9b0d, 0x13)
        /home/alei/go-ycsb/db/mysql/db.go:293 +0x85
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).BatchInsert(0xc00028d2f0, 0x1923200, 0xc0359824b0, 0x16b73d6, 0x9, 0xc0200e4000, 0xc8, 0x100, 0xc043382000, 0xc8, ...)
        /home/alei/go-ycsb/db/mysql/db.go:410 +0x885
github.com/pingcap/go-ycsb/pkg/client.DbWrapper.BatchInsert(0x1931c60, 0xc00028d2f0, 0x1923200, 0xc0359824b0, 0x16b73d6, 0x9, 0xc0200e4000, 0xc8, 0x100, 0xc043382000, ...)
        /home/alei/go-ycsb/pkg/client/dbwrapper.go:131 +0x328
github.com/pingcap/go-ycsb/pkg/workload/corev2.(*corev2).DoBatchInsert(0xc0002a6c00, 0x1923200, 0xc0359824b0, 0xc8, 0x1932920, 0xc000404000, 0x0, 0x0)
        /home/alei/go-ycsb/pkg/workload/corev2/corev2.go:211 +0x5c3
github.com/pingcap/go-ycsb/pkg/client.(*worker).run(0xc0001e5ec0, 0x1923200, 0xc0359824b0)
        /home/alei/go-ycsb/pkg/client/client.go:130 +0x34f
github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2(0xc0005616f0, 0xc0005140f0, 0x1f4, 0x1923140, 0xc000574400, 0x44)
        /home/alei/go-ycsb/pkg/client/client.go:212 +0x16a
created by github.com/pingcap/go-ycsb/pkg/client.(*Client).Run
        /home/alei/go-ycsb/pkg/client/client.go:206 +0x1af

Multi database nodes support for distributed system

go-ycsb currently support only one database node. For modern distributed database system, there are usually several nodes. Multiple nodes support will make go-ycsb more suitable for distributed database without a load balance. I want to work on this feature.

Panic on a given property file

I followed the original property file to write a one for MySQL:

mysql.properties

mysql.host = 127.0.0.1
mysql.port = 5000
mysql.user = root                

This file should be correct (?), it follows the style of Postgres property file here. However, the tool has a panic:

zhangys@xxx:~/go/src/github.com/pingcap/go-ycsb$ ./bin/go-ycsb run mysql -P workloads/workloada -p mysql.properties 
panic: runtime error: index out of range [1] with length 1

goroutine 1 [running]:
main.initialGlobal(0x7ffc348eb1dd, 0x5, 0xc00063fc20)
        /home/zhangys/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/main.go:94 +0x7e5
main.runClientCommandFunc(0xc0001ccf00, 0xc0002000a0, 0x1, 0x5, 0x1)
        /home/zhangys/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:30 +0xc5
main.runTransCommandFunc(0xc0001ccf00, 0xc0002000a0, 0x1, 0x5)
        /home/zhangys/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x4e
github.com/spf13/cobra.(*Command).execute(0xc0001ccf00, 0xc000200050, 0x5, 0x5, 0xc0001ccf00, 0xc000200050)
        /home/zhangys/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x2c2
github.com/spf13/cobra.(*Command).ExecuteC(0xc0001cc500, 0xc00063ff20, 0x3, 0x3)
        /home/zhangys/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x2fe
github.com/spf13/cobra.(*Command).Execute(...)
        /home/zhangys/go/pkg/mod/github.com/spf13/[email protected]/command.go:800
main.main()
        /home/zhangys/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/main.go:172 +0x27a

I know one workaround should be changing the source code, but it not very easy to do if I want to change the test database frequently.

How to use this tool with remote servers?

Tried to run this tool on my remote 64 core fdb cluster and got 64 queries per second. Am I missing something? Is this tool single threaded?

./bin/go-ycsb run fdb -P workloads/workloada | tail
Run finished, takes 8.200485453s
UPDATE - Takes(s): 7.7, Count: 493, QPS: 64.0, Avg(us): 435072, Min(us): 172548, Max(us): 5810164, 95th(us): 902000, 99th(us): 0
READ - Takes(s): 7.8, Count: 507, QPS: 64.9, Avg(us): 234089, Min(us): 108095, Max(us): 1216692, 95th(us): 600000, 99th(us): 699000

READ_ERROR错误

我使用go-ycsb测试我自己的kv数据库类似于levledb类型的,load完成是显示正常
image
在run完成之后出现了READ_ERROR 这个错误
image
请问READ_ERROR 这个错误什么时候会出现

support rocksdb.optionsfile in go-ycsb

YCSB git commit commit 4a990099a667c9226d5c33de7971f8b7ede9ffc0 supports a new parameter called "rocksdb.optionsfile". I have successfully using this parameter in our trocksdb project. Any plan for go-ycsb to support it as well?

Should use explicit transaction in TiDB to enable pessimistic transaction

pingcap/tidb#16068 suggests pessimistic transaction will be better in some workloads.

As mentioned in TiDB's documentation

The autocommit transactions do not support the pessimistic locking.

None of the autocommit statements acquire the pessimistic lock. These statements do not display any difference in the user side, because the nature of pessimistic transactions is to turn the retry of the whole transaction into a single DML retry. The autocommit transactions automatically retry even when TiDB closes the retry, which has the same effect as pessimistic transactions.

The autocommit SELECT FOR UPDATE statement does not wait for lock, either.

So we should do some modifications here.

record size differ

In the spec, 10 field with 100 bytes per field plus a 24 bytes key gives a 1024 byte record. But go-ycsb generates a 1481 byte value with 33 bytes key.
The generated key is something like:
usertable/user6284788457737146477
The usertable/ part contributes the extra bytes.
I'm not sure why the value is longer.

YCSB to test TiKV (without TiDB).

Can go-yCSB set the number of file bytes?if not.
When the Go-yCSB Load, what is the number of bytes written per time?
Is it a fixed number of bytes or random?
Thank you very much。

batch commands send error

./go-ycsb load tikv -p tikv.type=raw

When the test finished, it outputs too many batch commands send error logs, I remember we have fixed in tidb, seem forget to fix it client-go.

Avoid coordinated omission in the generator

It appears that go-ycsb might be affected by the coordinated omission. This could mask some serious performance issues.

In order to avoid CO, the load generator could launch a new request on a fixed schedule (e.g. by spawning a new goroutine) without waiting for the previous requests to complete.

Here is an example of a load benchmark executed against FoundationDB. First chart was produced by an experiment using a conventional load tester (client runs 1000 goroutines pushing requests sequentially). Second chart was produced by the load tester after switching to avoid coordinated omission and drive the load at predefined 20kHz - it reveals a bottleneck caused by the conflict contention and actually kills the client with OOM afterwards :)

ddlznfdxuaedfny jpg-large

summary

PS: This discussion has originated at the FoundationDB forum, Alex Miller suggested filing an issue here.

go-ycsb complaining foundationdb is not registered

Is foundationdb actively supported?

$ ./bin/go-ycsb run foundationdb -P workloads/workloada
foundationdb is not registered

$ fdbcli --version
FoundationDB CLI 5.1 (v5.1.7)
source version 9ad8d02386d4a6a5efecf898df80f2747695c627
protocol fdb00a551040001

$ go version
go version go1.10.3 linux/amd64

Update docker image `pingcap/go-ycsb`

The image is not up to date and some benchmarks failed to run during my test.

For example I got an error:

scylla is not registered

After checking the code, it only pops when the cassandra package is not imported.

MySQL Primary Key is truncated by field length

For example, when field length is set to 8, this will cause only user0000 ~ user9999 inserted into the database, even if recordcount >> 10000.

s := fmt.Sprintf("CREATE TABLE IF NOT EXISTS %s (YCSB_KEY VARCHAR(%d) PRIMARY KEY", tableName, fieldLength)

Maybe a warning should be printed when such scenario happens.

make go-ycsb

when make go-ycsb ,some packages can't be found because of timeout.
these packages are from golang.org/x and cloud.google.com/go
who knows how to solve this problem ?

How to test the generator work correctly?

I met an Insert Panic before because that I forgot to reduce 1 in the Counter Next, this error is hidden deep, so I think we need to write some tests to ensure they can generate well.

cassandra.cluster can not support multiple IPs

what I do

I run go-ycsb to test cassandra by multiple cassandra endpoints

./bin/go-ycsb load cassandra -p recordcount=1000000 -p cassandra.cluster=127.0.0.1:9042,127.0.0.2:9042 -p mysql.port=4000

what happened

panic: runtime error: index out of range

goroutine 1 [running]:
main.initialGlobal(0x7ffeefbff953, 0x5, 0xc000373c18)
	/Users/xiaojingchen/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/main.go:91 +0x93e
main.runClientCommandFunc(0xc0003de500, 0xc0003f8070, 0x1, 0x7, 0x0)
	/Users/xiaojingchen/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:30 +0xbb
main.runLoadCommandFunc(0xc0003de500, 0xc0003f8070, 0x1, 0x7)
	/Users/xiaojingchen/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:62 +0x4e
github.com/spf13/cobra.(*Command).execute(0xc0003de500, 0xc0003f8000, 0x7, 0x7, 0xc0003de500, 0xc0003f8000)
	/Users/xiaojingchen/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x2ae
github.com/spf13/cobra.(*Command).ExecuteC(0xc0003de000, 0xc000373f30, 0x3, 0x3)
	/Users/xiaojingchen/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x2ec
github.com/spf13/cobra.(*Command).Execute(...)
	/Users/xiaojingchen/go/pkg/mod/github.com/spf13/[email protected]/command.go:800
main.main()
	/Users/xiaojingchen/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/main.go:170 +0x292

The reason

the function StringSliceVarP can not parse the values which contains ,correctly.
e.g: -p a=b,c was parsed as a=b and c

// StringSliceVar defines a string flag with specified name, default value, and usage string.
// The argument p points to a []string variable in which to store the value of the flag.
// Compared to StringArray flags, StringSlice flags take comma-separated value as arguments and split them accordingly.
// For example:
//   --ss="v1,v2" -ss="v3"
// will result in
//   []string{"v1", "v2", "v3"}
func (f *FlagSet) StringSliceVar(p *[]string, name string, value []string, usage string) {
	f.VarP(newStringSliceValue(value, p), name, "", usage)
}

// StringSliceVarP is like StringSliceVar, but accepts a shorthand letter that can be used after a single dash.
func (f *FlagSet) StringSliceVarP(p *[]string, name, shorthand string, value []string, usage string) {
	f.VarP(newStringSliceValue(value, p), name, shorthand, usage)
}

data race

reproduce:

  1. go build -race cmd/go-ycsb/*
  2. echo 'threadcount=50' >> workloads/workloadb
  3. ./client load basic -P workloads/workloadb
  4. ./client run basic -P workloads/workloadb

this is the ouput:

 ./client  run   basic -P workloads/workloadb
***************** properties *****************
"updateproportion"="0.05"
"recordcount"="1000"
"workload"="core"
"insertproportion"="0"
"requestdistribution"="uniform"
"dotransactions"="true"
"operationcount"="1000"
"readallfields"="true"
"readproportion"="0.95"
"threadcount"="50"
"scanproportion"="0"
**********************************************
==================
WARNING: DATA RACE
Write at 0x00c00026ff60 by goroutine 64:
  github.com/pingcap/go-ycsb/pkg/generator.(*Discrete).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x1c3
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:358 +0xc1
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Previous write at 0x00c00026ff60 by goroutine 63:
  github.com/pingcap/go-ycsb/pkg/generator.(*Discrete).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x1c3
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:358 +0xc1
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Goroutine 64 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f

Goroutine 63 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f
==================
==================
WARNING: DATA RACE
Write at 0x00c0001fc180 by goroutine 64:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).nextKeyNum()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:408 +0x157
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:513 +0x68
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Previous write at 0x00c0001fc180 by goroutine 63:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).nextKeyNum()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:408 +0x157
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:513 +0x68
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Goroutine 64 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f

Goroutine 63 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f
==================
==================
WARNING: DATA RACE
Write at 0x00c0001fc180 by goroutine 65:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).nextKeyNum()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:408 +0x157
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:513 +0x68
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Previous write at 0x00c0001fc180 by goroutine 63:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).nextKeyNum()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:408 +0x157
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionRead()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:416 +0x7b
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:361 +0x230
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Goroutine 65 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f

Goroutine 63 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f
==================
==================
WARNING: DATA RACE
Write at 0x00c0001fc1a0 by goroutine 64:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).buildSingleValue()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:171 +0xae
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:520 +0x24d
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Previous write at 0x00c0001fc1a0 by goroutine 63:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).buildSingleValue()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:171 +0xae
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:520 +0x24d
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Goroutine 64 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f

Goroutine 63 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f
==================
==================
WARNING: DATA RACE
Write at 0x00c0001fc1a0 by goroutine 65:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).buildSingleValue()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:171 +0xae
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:520 +0x24d
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Previous write at 0x00c0001fc1a0 by goroutine 63:
  github.com/pingcap/go-ycsb/pkg/generator.(*Uniform).Next()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/generator/number.go:40 +0x97
  github.com/pingcap/go-ycsb/pkg/workload.(*core).buildSingleValue()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:171 +0xae
  github.com/pingcap/go-ycsb/pkg/workload.(*core).doTransactionUpdate()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:520 +0x24d
  github.com/pingcap/go-ycsb/pkg/workload.(*core).DoTransaction()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:363 +0x1ad
  github.com/pingcap/go-ycsb/pkg/client.(*worker).run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:126 +0x4d1
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x253

Goroutine 65 (running) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f

Goroutine 63 (finished) created at:
  github.com/pingcap/go-ycsb/pkg/client.(*Client).Run()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1fb
  main.runClientCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:55 +0x511
  main.runTransCommandFunc()
      /root/workspace/go/src/github.com/pingcap/go-ycsb/cmd/go-ycsb/client.go:66 +0x5b
  github.com/spf13/cobra.(*Command).execute()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:766 +0x8e0
  github.com/spf13/cobra.(*Command).ExecuteC()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:852 +0x416
  main.main()
      /root/workspace/go/pkg/mod/github.com/spf13/[email protected]/command.go:800 +0x37f
==================
Run finished, takes 24.978378ms
READ   - Takes(s): 0.0, Count: 952, OPS: 35603.8, Avg(us): 0, Min(us): 0, Max(us): 40, 99th(us): 1000, 99.9th(us): 1000, 99.99th(us): 1000
UPDATE - Takes(s): 0.0, Count: 48, OPS: 1633.8, Avg(us): 1, Min(us): 0, Max(us): 3, 99th(us): 1000, 99.9th(us): 1000, 99.99th(us): 1000
Found 5 data race(s)

batch commands send error:EOF

got error when I ran(total 1million records)

~/docker-compose run --rm  ycsb run tikv -P workloads/fa1 -p tikv.pd=pd0:2379 -p measurementtype=timeseries -p timeseries.granularity=500 -p threadcount=10

ycsb1

batch commands send error: EOF

Hi. I'm testing tikv with go-ycsb raw kv. And error message batch commands send error: EOF shows up the second time I enter shell commandline and type exit. I think this is a tiny bug. I'll try to fix it in spare time.

image

one of tikv server log:

[2019/03/18 15:03:14.073 +08:00] [ERROR] [kv.rs:862] ["batch commands error"] [err="RpcFinished(Some(RpcStatus { status: Ok, details: None }))"]

cloudspanner is not registered

/bin/go-ycsb run cloudspanner -P cloudspanner.properties -P workloads/workloadb -p recordcount=10000000 -p operationcount=1000000 --threads 1

cloudspanner is not registered

go version

go version go1.14.4 darwin/amd64

Build fail

fatal: unable to access 'https://git.apache.org/thrift.git/': LibreSSL SSL_connect: SSL_ERROR_SYSCALL in connection to git.apache.org:443 

Suggest to apply a patch similar to kubernetes/minikube#5245

There is a panic

after many log after "packets.go:36: unexpected EOF", ther is a panic

panic: runtime error: invalid memory address or nil pointer dereference
[signal SIGSEGV: segmentation violation code=0x1 addr=0x28 pc=0x7bf2a6]

goroutine 180 [running]:
database/sql.(*Conn).grabConn(0x0, 0x175bac0, 0xc0016aede0, 0x1e, 0x7, 0xc002f1db00, 0x4ef75e)
/usr/lib/golang/src/database/sql/sql.go:1745 +0x26
database/sql.(*Conn).PrepareContext(0x0, 0x175bac0, 0xc0016aede0, 0xc0023dc9c0, 0x191, 0x7f282833cb00, 0x0, 0x1e)
/usr/lib/golang/src/database/sql/sql.go:1801 +0x46
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).getAndCacheStmt(0xc00043aff0, 0x175bac0, 0xc0016aede0, 0xc0023dc9c0, 0x191, 0xc0023dc9c0, 0x0, 0xc000470380)
/home/tidb/go/src/github.com/pingcap/go-ycsb/db/mysql/db.go:170 +0xfa
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).execQuery(0xc00043aff0, 0x175bac0, 0xc0016aede0, 0xc0023dc9c0, 0x191, 0xc000e09000, 0x1f, 0x1f, 0x4f0dab, 0xc000214570)
/home/tidb/go/src/github.com/pingcap/go-ycsb/db/mysql/db.go:279 +0x83
github.com/pingcap/go-ycsb/db/mysql.(*mysqlDB).Insert(0xc00043aff0, 0x175bac0, 0xc0016aede0, 0xc0002aa550, 0xa, 0xc0021d7080, 0x17, 0xc0037fc690, 0x0, 0x0)
/home/tidb/go/src/github.com/pingcap/go-ycsb/db/mysql/db.go:342 +0x523
github.com/pingcap/go-ycsb/pkg/client.DbWrapper.Insert(0x1764160, 0xc00043aff0, 0x175bac0, 0xc0016aede0, 0xc0002aa550, 0xa, 0xc0021d7080, 0x17, 0xc0037fc690, 0x0, ...)
/home/tidb/go/src/github.com/pingcap/go-ycsb/pkg/client/dbwrapper.go:121 +0xe4
github.com/pingcap/go-ycsb/pkg/workload.(*core).DoInsert(0xc0004ae000, 0x175bac0, 0xc0016aede0, 0x1764d00, 0xc00026a000, 0x0, 0x0)
/home/tidb/go/src/github.com/pingcap/go-ycsb/pkg/workload/core.go:269 +0x226
github.com/pingcap/go-ycsb/pkg/client.(*worker).run(0xc0000e0480, 0x175bac0, 0xc0016aede0)
/home/tidb/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:133 +0x370
github.com/pingcap/go-ycsb/pkg/client.(*Client).Run.func2(0xc0002a8880, 0xc000102810, 0x3e8, 0x175ba00, 0xc000446100, 0x34)
/home/tidb/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:212 +0x157
created by github.com/pingcap/go-ycsb/pkg/client.(*Client).Run
/home/tidb/go/src/github.com/pingcap/go-ycsb/pkg/client/client.go:206 +0x1af

Faster Zipfian generator

Hi there,

for a research project I borrowed the Zipfian generator from go-ycsb (https://github.com/pingcap/go-ycsb/blob/master/pkg/generator/zipfian.go) and applied three patches to make it faster. Maybe you want to apply some of them to the project. You can see the differences here (among clutter from changing Go package paths): master...leoholz:master

  1. The check in line 158 recomputes the threshold for the first item on every call of Next(). Unfortunately, the Go compiler is not clever enough to recognize that this value never changes after initialization. You can about half the calculation time of Next() by replacing 1.0+math.Pow(0.5, z.theta) with a precalculated value.

  2. The performance of math.Pow() depends on the number of digits that the exponent has. Here the floating point math on line 110 is not helpful, e.g. for the default Zipfian constant 0.99 the computed alpha value is 99.99999999999991 (https://play.golang.org/p/P76sdDmLaZj). Therefore, one can speed up calculations by sensibly rounding the divisor: z.alpha = 1.0 / (math.Round((1.0-theta)*math.Pow10(10)) / math.Pow10(10)) (https://play.golang.org/p/JqWTjos6zru).

  3. The most time-consuming part of the initialization phase is the calculation of the harmonic number in zetaStatic(), this takes about 2 minutes for my test set of 1024*1024*1024 items. Luckily, the parts of the sum are independent of each other, so it is possible to use concurrency to speed things up. You can find an implementation here: https://github.com/leoholz/go-ycsb/blob/master/pkg/generator/zipfian.go#L135

about custom prefix of YCSB_KEY

YCSB_KEY is the format of "userXXXXXXXX" and the prefix is "user".
In my benchmark, a key with 40 bytes or longer is needed.
So I suggest a property of custom prefix.

Duration display none bug

I found sometimes the duration display 0, you can see the following picture. Maybe the duration exceed some bound?

duration-display-no

failed to load data on mongodb with authentication disabled

My mongod is authentication disabled, when loading data, many errrors occured as follows:

auth error: sasl conversation error: unable to authenticate using mechanism "SCRAM-SHA-1": (AuthenticationFailed) Authentication failed.
auth error: sasl conversation error: unable to authenticate using mechanism "SCRAM-SHA-1": (AuthenticationFailed) Authentication failed.
auth error: sasl conversation error: unable to authenticate using mechanism "SCRAM-SHA-1": (AuthenticationFailed) Authentication failed.

Failed to Build

While building with commit a001232, I got this error:

make
go build -o bin/go-ycsb cmd/go-ycsb/*
go: downloading github.com/pingcap/pd v2.1.0-rc.4+incompatible
go: downloading google.golang.org/grpc v1.16.0
go: downloading github.com/grpc-ecosystem/go-grpc-middleware v1.0.0
go: downloading github.com/google/btree v0.0.0-20180813153112-4030bb1f1f0c
go: downloading github.com/spaolacci/murmur3 v0.0.0-20180118202830-f09979ecbc72
go: downloading github.com/coreos/etcd v3.3.10+incompatible
go: downloading github.com/pingcap/tidb-tools v2.1.0-rc.5+incompatible
go: verifying github.com/grpc-ecosystem/[email protected]: checksum mismatch
	downloaded: h1:BWIsLfhgKhV5g/oF34aRjniBHLTZe5DNekSjbAjIS6c=
	go.sum:     h1:Iju5GlWwrvL6UBg4zJJt3btmonfrMlCDdsejg4CZE7c=
make: *** [build] Error 1

I need to delete go.sum first then it works.

strang output `INFO[0014] con:0 2PC clean up done, tid: 409393655270080780`

./go-ycsb run tikv -p tikv.type=txn
INFO[0014] con:0 2PC clean up done, tid: 409393655270080780
INFO[0014] con:0 2PC clean up done, tid: 409393655283974485
INFO[0014] con:0 2PC clean up done, tid: 409393655258284619
INFO[0014] con:0 2PC clean up done, tid: 409393655270080910

I think we shouldn't output these logs.

Recommend Projects

  • React photo React

    A declarative, efficient, and flexible JavaScript library for building user interfaces.

  • Vue.js photo Vue.js

    🖖 Vue.js is a progressive, incrementally-adoptable JavaScript framework for building UI on the web.

  • Typescript photo Typescript

    TypeScript is a superset of JavaScript that compiles to clean JavaScript output.

  • TensorFlow photo TensorFlow

    An Open Source Machine Learning Framework for Everyone

  • Django photo Django

    The Web framework for perfectionists with deadlines.

  • D3 photo D3

    Bring data to life with SVG, Canvas and HTML. 📊📈🎉

Recommend Topics

  • javascript

    JavaScript (JS) is a lightweight interpreted programming language with first-class functions.

  • web

    Some thing interesting about web. New door for the world.

  • server

    A server is a program made to process requests and deliver data to clients.

  • Machine learning

    Machine learning is a way of modeling and interpreting data that allows a piece of software to respond intelligently.

  • Game

    Some thing interesting about game, make everyone happy.

Recommend Org

  • Facebook photo Facebook

    We are working to build community through open source technology. NB: members must have two-factor auth.

  • Microsoft photo Microsoft

    Open source projects and samples from Microsoft.

  • Google photo Google

    Google ❤️ Open Source for everyone.

  • D3 photo D3

    Data-Driven Documents codes.